25 #ifndef KUDU_CLIENT_CLIENT_H
26 #define KUDU_CLIENT_CLIENT_H
38 #include "kudu/client/row_result.h"
39 #include "kudu/client/scan_predicate.h"
40 #include "kudu/client/schema.h"
42 #ifdef KUDU_HEADERS_NO_STUBS
43 #include <gtest/gtest_prod.h>
45 #include "kudu/gutil/macros.h"
46 #include "kudu/gutil/port.h"
48 #include "kudu/client/stubs.h"
50 #include "kudu/util/kudu_export.h"
51 #include "kudu/util/slice.h"
57 class ClientStressTest_TestUniqueClientIds_Test;
61 class PartitionSchema;
62 class SecurityUnknownTskTest;
69 namespace transactions {
71 class TxnSystemClient;
75 class LeaderMasterProxy;
76 class RemoteKsckCluster;
81 class KuduColumnarScanBatch;
84 class KuduInsertIgnore;
85 class KuduLoggingCallback;
86 class KuduPartitioner;
89 class KuduStatusCallback;
90 class KuduTableAlterer;
91 class KuduTableCreator;
92 class KuduTableStatistics;
94 class KuduTabletServer;
98 class KuduWriteOperation;
99 class ResourceMetrics;
103 class ErrorCollector;
104 class GetTableSchemaRpc;
108 class RemoteTabletServer;
109 class ReplicaController;
110 class RetrieveAuthzTokenRpc;
111 class ScanBatchDataInterface;
113 template <
class ReqClass,
class RespClass>
137 void KUDU_EXPORT UninstallLoggingCallback();
154 void KUDU_EXPORT SetVerboseLogLevel(
int level);
165 Status KUDU_EXPORT SetInternalSignalNumber(
int signum) WARN_UNUSED_RESULT;
178 Status KUDU_EXPORT DisableSaslInitialization() WARN_UNUSED_RESULT;
203 Status KUDU_EXPORT DisableOpenSSLInitialization() WARN_UNUSED_RESULT;
207 std::
string KUDU_EXPORT GetShortVersionString();
211 std::
string KUDU_EXPORT GetAllVersionInfo();
296 class KUDU_NO_EXPORT Data;
298 friend class internal::ReplicaController;
332 class KUDU_EXPORT
KuduClient :
public sp::enable_shared_from_this<KuduClient> {
351 bool* create_in_progress);
372 Status DeleteTableInCatalogs(
const std::string& table_name,
373 bool modify_external_catalogs) KUDU_NO_EXPORT;
393 bool* alter_in_progress);
420 const std::string& filter =
"");
448 sp::shared_ptr<KuduTable>* table);
471 Status GetTablet(
const std::string& tablet_id,
481 Status GetTableStatistics(
const std::string& table_name,
491 std::string GetMasterAddresses() const KUDU_NO_EXPORT;
570 std::string GetHiveMetastoreUris() const KUDU_NO_EXPORT;
577 bool GetHiveMetastoreSaslEnabled() const KUDU_NO_EXPORT;
588 std::
string GetHiveMetastoreUuid() const KUDU_NO_EXPORT;
596 std::
string location() const KUDU_NO_EXPORT;
600 class KUDU_NO_EXPORT Data;
602 template <class ReqClass, class RespClass>
603 friend class internal::AsyncLeaderMasterRpc;
605 friend class ClientTest;
606 friend class ConnectToClusterBaseTest;
616 friend class internal::Batcher;
617 friend class internal::GetTableSchemaRpc;
618 friend class internal::LookupRpc;
619 friend class internal::MetaCache;
620 friend class internal::RemoteTablet;
621 friend class internal::RemoteTabletServer;
622 friend class internal::RetrieveAuthzTokenRpc;
623 friend class internal::WriteRpc;
624 friend class kudu::AuthzTokenTest;
625 friend class kudu::SecurityUnknownTskTest;
626 friend class transactions::CoordinatorRpc;
627 friend class transactions::TxnSystemClient;
628 friend class tools::LeaderMasterProxy;
629 friend class tools::RemoteKsckCluster;
631 FRIEND_TEST(kudu::ClientStressTest, TestUniqueClientIds);
632 FRIEND_TEST(ClientTest, TestCacheAuthzTokens);
633 FRIEND_TEST(ClientTest, TestGetSecurityInfoFromMaster);
634 FRIEND_TEST(ClientTest, TestGetTabletServerBlacklist);
635 FRIEND_TEST(ClientTest, TestMasterDown);
636 FRIEND_TEST(ClientTest, TestMasterLookupPermits);
637 FRIEND_TEST(ClientTest, TestMetaCacheExpiry);
638 FRIEND_TEST(ClientTest, TestNonCoveringRangePartitions);
639 FRIEND_TEST(ClientTest, TestRetrieveAuthzTokenInParallel);
640 FRIEND_TEST(ClientTest, TestReplicatedTabletWritesWithLeaderElection);
641 FRIEND_TEST(ClientTest, TestScanFaultTolerance);
642 FRIEND_TEST(ClientTest, TestScanTimeout);
643 FRIEND_TEST(ClientTest, TestWriteWithDeadMaster);
644 FRIEND_TEST(MasterFailoverTest, TestPauseAfterCreateTableIssued);
661 const std::string&
uuid()
const;
677 const std::string& location() const KUDU_NO_EXPORT;
681 class KUDU_NO_EXPORT Data;
712 friend class internal::ReplicaController;
714 class KUDU_NO_EXPORT Data;
731 const std::string&
id()
const;
738 const std::vector<const KuduReplica*>&
replicas()
const;
744 class KUDU_NO_EXPORT Data;
801 int32_t num_buckets);
819 int32_t num_buckets, int32_t seed);
899 ATTRIBUTE_DEPRECATED(
"use add_range_partition_split() instead");
970 class KUDU_NO_EXPORT Data;
973 friend class transactions::TxnSystemClient;
1007 class KUDU_NO_EXPORT Data;
1037 class KUDU_EXPORT
KuduTable :
public sp::enable_shared_from_this<KuduTable> {
1051 const std::string&
id()
const;
1146 std::vector<KuduBloomFilter*>* bloom_filters);
1189 const std::vector<Slice>& bloom_filters);
1216 std::vector<KuduValue*>* values);
1263 Status ListPartitions(std::vector<Partition>* partitions) KUDU_NO_EXPORT;
1268 class KUDU_NO_EXPORT Data;
1274 KuduTable(
const sp::shared_ptr<KuduClient>& client,
1275 const std::string& name,
1276 const std::string&
id,
1278 const std::string& owner,
1280 const PartitionSchema& partition_schema,
1281 const std::map<std::string, std::string>& extra_configs);
1422 const std::string& dimension_label,
1504 KuduTableAlterer* modify_external_catalogs(
bool modify_external_catalogs) KUDU_NO_EXPORT;
1515 class KUDU_NO_EXPORT Data;
1520 const std::string& name);
1565 class KUDU_NO_EXPORT Data;
1567 friend class internal::Batcher;
1568 friend class internal::ErrorCollector;
1638 class KUDU_EXPORT
KuduSession :
public sp::enable_shared_from_this<KuduSession> {
1950 bool HasPendingOperations() const;
1973 int CountBufferedOperations() const
1974 ATTRIBUTE_DEPRECATED("this method is experimental and will disappear "
1975 "in a future release");
2001 Status SetErrorBufferSpace(
size_t size_bytes);
2012 int CountPendingErrors() const;
2024 void GetPendingErrors(std::vector<
KuduError*>* errors,
bool* overflowed);
2030 class KUDU_NO_EXPORT Data;
2033 friend class internal::Batcher;
2034 friend class ClientTest;
2035 FRIEND_TEST(ClientTest, TestAutoFlushBackgroundApplyBlocks);
2036 FRIEND_TEST(ClientTest, TestAutoFlushBackgroundAndErrorCollector);
2119 enum { kScanTimeoutMillis = 30000 };
2160 ATTRIBUTE_DEPRECATED(
"use SetProjectedColumnNames() instead");
2190 ATTRIBUTE_DEPRECATED(
"use AddLowerBound() instead");
2210 ATTRIBUTE_DEPRECATED(
"use AddExclusiveUpperBound() instead");
2295 ATTRIBUTE_DEPRECATED(
"use NextBatch(KuduScanBatch*) instead");
2367 ATTRIBUTE_DEPRECATED(
"use SetFaultTolerant() instead");
2388 Status SetSnapshotMicros(uint64_t snapshot_timestamp_micros) WARN_UNUSED_RESULT;
2399 Status SetSnapshotRaw(uint64_t snapshot_timestamp) WARN_UNUSED_RESULT;
2417 Status SetDiffScan(uint64_t start_timestamp, uint64_t end_timestamp)
2418 WARN_UNUSED_RESULT KUDU_NO_EXPORT;
2444 static const uint64_t NO_FLAGS = 0;
2451 static const uint64_t PAD_UNIXTIME_MICROS_TO_16_BYTES = 1 << 0;
2460 static const uint64_t COLUMNAR_LAYOUT = 1 << 1;
2497 Status SetLimit(int64_t limit) WARN_UNUSED_RESULT;
2504 std::
string ToString() const;
2507 class KUDU_NO_EXPORT Data;
2509 Status NextBatch(internal::ScanBatchDataInterface* batch);
2512 FRIEND_TEST(ClientTest, TestBlockScannerHijackingAttempts);
2513 FRIEND_TEST(ClientTest, TestScanCloseProxy);
2514 FRIEND_TEST(ClientTest, TestScanFaultTolerance);
2515 FRIEND_TEST(ClientTest, TestScanNoBlockCaching);
2516 FRIEND_TEST(ClientTest, TestScanTimeout);
2517 FRIEND_TEST(ClientTest, TestReadAtSnapshotNoTimestampSet);
2518 FRIEND_TEST(ConsistencyITest, TestSnapshotScanTimestampReuse);
2519 FRIEND_TEST(ScanTokenTest, TestScanTokens);
2588 const std::string& serialized_token,
2592 class KUDU_NO_EXPORT Data;
2678 Status SetSnapshotMicros(uint64_t snapshot_timestamp_micros)
2682 Status SetSnapshotRaw(uint64_t snapshot_timestamp) WARN_UNUSED_RESULT;
2687 Status SetDiffScan(uint64_t start_timestamp, uint64_t end_timestamp)
2688 WARN_UNUSED_RESULT KUDU_NO_EXPORT;
2692 Status SetTimeoutMillis(
int millis) WARN_UNUSED_RESULT;
2701 Status IncludeTableMetadata(
bool include_metadata) WARN_UNUSED_RESULT;
2710 Status IncludeTabletMetadata(
bool include_metadata) WARN_UNUSED_RESULT;
2723 class KUDU_NO_EXPORT Data;
2768 class KUDU_NO_EXPORT Data;
2808 class KUDU_NO_EXPORT Data;
KuduWriteOperation * release_failed_op()
KuduTableCreator & extra_configs(const std::map< std::string, std::string > &extra_configs)
KuduTableAlterer * wait(bool wait)
This class represents an error which occurred in a write operation.
Definition: client.h:1533
Representation of a Kudu client session.
Definition: client.h:1638
KuduTableAlterer * NewTableAlterer(const std::string &table_name)
const std::string & id() const
const std::string & hostname() const
bool IsMultiMaster() const
KuduTableAlterer * DropRangePartition(KuduPartialRow *lower_bound, KuduPartialRow *upper_bound, KuduTableCreator::RangePartitionBound lower_bound_type=KuduTableCreator::INCLUSIVE_BOUND, KuduTableCreator::RangePartitionBound upper_bound_type=KuduTableCreator::EXCLUSIVE_BOUND)
In-memory representation of a remote tablet.
Definition: client.h:725
Status AddExclusiveUpperBoundPartitionKeyRaw(const Slice &partition_key)
KuduTableCreator & set_range_partition_columns(const std::vector< std::string > &columns)
Status IsCreateTableInProgress(const std::string &table_name, bool *create_in_progress)
KuduClientBuilder & master_server_addrs(const std::vector< std::string > &addrs)
KuduTableAlterer * SetOwner(const std::string &new_owner)
In-memory statistics of table.
Definition: client.h:984
const std::string & id() const
KuduTableCreator & dimension_label(const std::string &dimension_label)
KuduPartitionerBuilder * SetBuildTimeout(MonoDelta timeout)
KuduInsertIgnore * NewInsertIgnore()
KuduTableAlterer * timeout(const MonoDelta &timeout)
RangePartitionBound
Range partition bound type.
Definition: client.h:836
KuduTableCreator & split_rows(const std::vector< const KuduPartialRow * > &split_rows) ATTRIBUTE_DEPRECATED("use add_range_partition_split() instead")
Status ExportAuthenticationCredentials(std::string *authn_creds) const
Status TableExists(const std::string &table_name, bool *exists)
Status Build(KuduPartitioner **partitioner)
Smart pointer typedefs for externally-faced code.
KuduTableAlterer * AlterExtraConfig(const std::map< std::string, std::string > &extra_configs)
const KuduWriteOperation & failed_op() const
@ CLIENT_PROPAGATED
Definition: client.h:1723
bool was_possibly_successful() const
Status SetCacheBlocks(bool cache_blocks) WARN_UNUSED_RESULT
Status IsAlterTableInProgress(const std::string &table_name, bool *alter_in_progress)
KuduTableCreator & add_range_partition(KuduPartialRow *lower_bound, KuduPartialRow *upper_bound, RangePartitionBound lower_bound_type=INCLUSIVE_BOUND, RangePartitionBound upper_bound_type=EXCLUSIVE_BOUND)
A generic catalog of simple metrics.
Definition: resource_metrics.h:39
Status SetMutationBufferSpace(size_t size_bytes) WARN_UNUSED_RESULT
Status SetSelection(KuduClient::ReplicaSelection selection) WARN_UNUSED_RESULT
Definition: client.h:2786
Status Build(sp::shared_ptr< KuduClient > *client)
This class is a representation of a single scan.
Definition: client.h:2051
A constant cell value with a specific type.
Definition: value.h:35
const Status & status() const
const KuduTabletServer & ts() const
Status SetSelection(KuduClient::ReplicaSelection selection) WARN_UNUSED_RESULT
sp::shared_ptr< KuduSession > NewSession()
In-memory representation of a remote tablet server.
Definition: client.h:655
ReadMode
The read modes for scanners.
Definition: client.h:2054
KuduTableAlterer * AddRangePartition(KuduPartialRow *lower_bound, KuduPartialRow *upper_bound, KuduTableCreator::RangePartitionBound lower_bound_type=KuduTableCreator::INCLUSIVE_BOUND, KuduTableCreator::RangePartitionBound upper_bound_type=KuduTableCreator::EXCLUSIVE_BOUND)
Status SetMutationBufferMaxNum(unsigned int max_num) WARN_UNUSED_RESULT
KuduPredicate * NewIsNotNullPredicate(const Slice &col_name)
@ READ_AT_SNAPSHOT
Definition: client.h:2083
KuduClientBuilder & import_authentication_credentials(std::string authn_creds)
A row which may only contain values for a subset of the columns.
Definition: partial_row.h:72
Status SetMutationBufferFlushInterval(unsigned int millis) WARN_UNUSED_RESULT
KuduTableCreator & table_name(const std::string &name)
std::string ToString() const
Status GetCurrentServer(KuduTabletServer **server)
const MonoDelta & default_admin_operation_timeout() const
KuduTableCreator & add_range_partition_split(KuduPartialRow *split_row)
A batch of zero or more rows returned by a scan operation.
Definition: scan_batch.h:84
ExternalConsistencyMode
The possible external consistency modes on which Kudu operates.
Definition: client.h:1704
Status SetCacheBlocks(bool cache_blocks)
Status SetMutationBufferFlushWatermark(double watermark_pct) WARN_UNUSED_RESULT
KuduTableAlterer * RenameTo(const std::string &new_name)
KuduTableCreator & add_hash_partitions(const std::vector< std::string > &columns, int32_t num_buckets, int32_t seed)
A single row update to be sent to the cluster.
Definition: write_op.h:217
A representation of comparison predicate for Kudu queries.
Definition: scan_predicate.h:43
Status AddLowerBound(const KuduPartialRow &key) WARN_UNUSED_RESULT
Status SetProjectedColumnNames(const std::vector< std::string > &col_names) WARN_UNUSED_RESULT
Builder for Partitioner instances.
Definition: client.h:2732
const ResourceMetrics & GetResourceMetrics() const
A helper class to create a new table with the desired options.
Definition: client.h:755
The interface for all status callbacks.
Definition: callbacks.h:161
Status Flush() WARN_UNUSED_RESULT
KuduTableCreator * NewTableCreator()
KuduClientBuilder & default_rpc_timeout(const MonoDelta &timeout)
const MonoDelta & default_rpc_timeout() const
Status SetFaultTolerant() WARN_UNUSED_RESULT
Builder API for specifying or altering a column within a table schema.
Definition: schema.h:341
Status GetTableSchema(const std::string &table_name, KuduSchema *schema)
KuduTableCreator & add_hash_partitions(const std::vector< std::string > &columns, int32_t num_buckets)
KuduClientBuilder & add_master_server_addr(const std::string &addr)
const std::string & owner() const
Status AddLowerBound(const KuduPartialRow &key)
Status SetFaultTolerant() WARN_UNUSED_RESULT
Status SetBatchSizeBytes(uint32_t batch_size)
Status Serialize(std::string *buf) const WARN_UNUSED_RESULT
Status PartitionRow(const KuduPartialRow &row, int *partition)
KuduScanTokenBuilder(KuduTable *table)
Status Apply(KuduWriteOperation *write_op) WARN_UNUSED_RESULT
Status SetProjectedColumnNames(const std::vector< std::string > &col_names) WARN_UNUSED_RESULT
The interface for all logging callbacks.
Definition: callbacks.h:44
Status NextBatch(KuduColumnarScanBatch *batch)
KuduTableAlterer * DropColumn(const std::string &name)
const std::string & uuid() const
int64_t on_disk_size() const
KuduPredicate * NewIsNullPredicate(const Slice &col_name)
Status AddLowerBoundPartitionKeyRaw(const Slice &partition_key)
int NumPartitions() const
@ INCLUSIVE_BOUND
An inclusive bound.
Definition: client.h:838
KuduTableCreator & timeout(const MonoDelta &timeout)
Builds scan tokens for a table.
Definition: client.h:2607
KuduScanner(KuduTable *table)
KuduColumnSpec * AddColumn(const std::string &name)
A single row upsert to be sent to the cluster.
Definition: write_op.h:190
ReplicaSelection
Policy with which to choose amongst multiple replicas.
Definition: client.h:496
A single-row write operation to be sent to a Kudu table.
Definition: write_op.h:64
const std::string & name() const
const std::map< std::string, std::string > & extra_configs() const
FlushMode
Modes of flush operations.
Definition: client.h:1643
@ LEADER_ONLY
Select the LEADER replica.
Definition: client.h:497
In-memory representation of a remote tablet's replica.
Definition: client.h:696
A representation of a table's schema.
Definition: schema.h:597
A scan descriptor limited to a single physical contiguous location.
Definition: client.h:2547
@ AUTO_FLUSH_BACKGROUND
Definition: client.h:1676
A representation of an operation's outcome.
Definition: status.h:165
KuduClientBuilder & num_reactors(int num_reactors)
Set the number of reactors for the RPC messenger.
Status NextBatch(std::vector< KuduRowResult > *rows) ATTRIBUTE_DEPRECATED("use NextBatch(KuduScanBatch*) instead")
ComparisonOp
Supported comparison operators.
Definition: scan_predicate.h:46
Status OpenTable(const std::string &table_name, sp::shared_ptr< KuduTable > *table)
@ EXCLUSIVE_BOUND
An exclusive bound.
Definition: client.h:837
A representation of a table on a particular cluster.
Definition: client.h:1037
KuduColumnSpec * AlterColumn(const std::string &name)
@ AUTO_FLUSH_SYNC
Definition: client.h:1648
static const uint64_t kNoTimestamp
Definition: client.h:520
Status ListTabletServers(std::vector< KuduTabletServer * > *tablet_servers)
KuduTableCreator & wait(bool wait)
Status SetProjectedColumnIndexes(const std::vector< int > &col_indexes) WARN_UNUSED_RESULT
A single row delete to be sent to the cluster.
Definition: write_op.h:244
A representation of a time interval.
Definition: monotime.h:57
Status AddConjunctPredicate(KuduPredicate *pred) WARN_UNUSED_RESULT
KuduPredicate * NewInBloomFilterPredicate(const Slice &col_name, const std::vector< Slice > &bloom_filters)
uint64_t GetLatestObservedTimestamp() const
const KuduTablet & tablet() const
Status SetReadMode(KuduScanner::ReadMode read_mode) WARN_UNUSED_RESULT
Status SetFlushMode(FlushMode m) WARN_UNUSED_RESULT
Status AddUpperBound(const KuduPartialRow &key) WARN_UNUSED_RESULT
KuduPredicate * NewInListPredicate(const Slice &col_name, std::vector< KuduValue * > *values)
Status DeleteTable(const std::string &table_name)
Status AddLowerBoundRaw(const Slice &key) ATTRIBUTE_DEPRECATED("use AddLowerBound() instead")
void SetTimeoutMillis(int millis)
Status SetReadMode(ReadMode read_mode) WARN_UNUSED_RESULT
A wrapper around externally allocated data.
Definition: slice.h:51
KuduTableCreator & num_replicas(int n_replicas)
void SetLatestObservedTimestamp(uint64_t ht_timestamp)
Status ListTables(std::vector< std::string > *tables, const std::string &filter="")
KuduClientBuilder & default_admin_operation_timeout(const MonoDelta &timeout)
KuduTableAlterer * AddRangePartitionWithDimension(KuduPartialRow *lower_bound, KuduPartialRow *upper_bound, const std::string &dimension_label, KuduTableCreator::RangePartitionBound lower_bound_type=KuduTableCreator::INCLUSIVE_BOUND, KuduTableCreator::RangePartitionBound upper_bound_type=KuduTableCreator::EXCLUSIVE_BOUND)
A "factory" for KuduClient objects.
Definition: client.h:217
A batch of columnar data returned from a scanner.
Definition: columnar_scan_batch.h:51
A single row insert to be sent to the cluster.
Definition: write_op.h:137
Status SetBatchSizeBytes(uint32_t batch_size) WARN_UNUSED_RESULT
KuduPredicate * NewComparisonPredicate(const Slice &col_name, KuduPredicate::ComparisonOp op, KuduValue *value)
static Status DeserializeIntoScanner(KuduClient *client, const std::string &serialized_token, KuduScanner **scanner) WARN_UNUSED_RESULT
Status AddExclusiveUpperBoundRaw(const Slice &key) ATTRIBUTE_DEPRECATED("use AddExclusiveUpperBound() instead")
const KuduSchema & schema() const
KuduTableCreator & schema(const KuduSchema *schema)
KuduPartitionerBuilder(sp::shared_ptr< KuduTable > table)
KuduPredicate * NewInBloomFilterPredicate(const Slice &col_name, std::vector< KuduBloomFilter * > *bloom_filters)
A single row insert ignore to be sent to the cluster, duplicate row errors are ignored.
Definition: write_op.h:164
Status SetExternalConsistencyMode(ExternalConsistencyMode m) WARN_UNUSED_RESULT
Status AddConjunctPredicate(KuduPredicate *pred) WARN_UNUSED_RESULT
KuduTableCreator & set_owner(const std::string &owner)
Status SetOrderMode(OrderMode order_mode) WARN_UNUSED_RESULT ATTRIBUTE_DEPRECATED("use SetFaultTolerant() instead")
const PartitionSchema & partition_schema() const
Status SetProjectedColumnIndexes(const std::vector< int > &col_indexes) WARN_UNUSED_RESULT
Alters an existing table based on the provided steps.
Definition: client.h:1300
KuduClientBuilder & clear_master_server_addrs()
Status AddExclusiveUpperBound(const KuduPartialRow &key)
@ UNORDERED
Definition: client.h:2107
Status IntoKuduScanner(KuduScanner **scanner) const WARN_UNUSED_RESULT
const std::vector< const KuduReplica * > & replicas() const
@ READ_LATEST
Definition: client.h:2062
int64_t live_row_count() const
Status NextBatch(KuduScanBatch *batch)
OrderMode
Definition: client.h:2101
KuduClient * client() const
Status SetProjectedColumns(const std::vector< std::string > &col_names) WARN_UNUSED_RESULT ATTRIBUTE_DEPRECATED("use SetProjectedColumnNames() instead")
A handle for a connection to a cluster.
Definition: client.h:332