Kudu C++ client API
client.h
1 // Licensed to the Apache Software Foundation (ASF) under one
2 // or more contributor license agreements. See the NOTICE file
3 // distributed with this work for additional information
4 // regarding copyright ownership. The ASF licenses this file
5 // to you under the Apache License, Version 2.0 (the
6 // "License"); you may not use this file except in compliance
7 // with the License. You may obtain a copy of the License at
8 //
9 // http://www.apache.org/licenses/LICENSE-2.0
10 //
11 // Unless required by applicable law or agreed to in writing,
12 // software distributed under the License is distributed on an
13 // "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
14 // KIND, either express or implied. See the License for the
15 // specific language governing permissions and limitations
16 // under the License.
17 
24 
25 #ifndef KUDU_CLIENT_CLIENT_H
26 #define KUDU_CLIENT_CLIENT_H
27 
28 #include <stdint.h>
29 
30 #include <cstddef>
31 #include <map>
32 // Not safe to include <memory>; this header must remain compatible with C++98.
33 //
34 // IWYU pragma: no_include <memory>
35 #include <string>
36 #include <vector>
37 
38 #include "kudu/client/row_result.h"
39 #include "kudu/client/scan_predicate.h"
40 #include "kudu/client/schema.h"
41 #include "kudu/client/shared_ptr.h" // IWYU pragma: keep
42 #ifdef KUDU_HEADERS_NO_STUBS
43 #include <gtest/gtest_prod.h>
44 
45 #include "kudu/gutil/macros.h"
46 #include "kudu/gutil/port.h"
47 #else
48 #include "kudu/client/stubs.h"
49 #endif
50 #include "kudu/util/kudu_export.h"
51 #include "kudu/util/slice.h"
52 #include "kudu/util/status.h"
53 
54 namespace kudu {
55 
56 class AuthzTokenTest;
57 class ClientStressTest_TestUniqueClientIds_Test;
58 class KuduPartialRow;
59 class MonoDelta;
60 class Partition;
61 class PartitionSchema;
62 class SecurityUnknownTskTest;
63 
64 namespace client {
65 class KuduClient;
66 class KuduTable;
67 } // namespace client
68 
69 namespace tools {
70 class LeaderMasterProxy;
71 class RemoteKsckCluster;
72 } // namespace tools
73 
74 namespace client {
75 
76 class KuduColumnarScanBatch;
77 class KuduDelete;
78 class KuduInsert;
79 class KuduInsertIgnore;
80 class KuduLoggingCallback;
81 class KuduPartitioner;
82 class KuduScanBatch;
83 class KuduSession;
84 class KuduStatusCallback;
85 class KuduTableAlterer;
86 class KuduTableCreator;
87 class KuduTableStatistics;
88 class KuduTablet;
89 class KuduTabletServer;
90 class KuduUpdate;
91 class KuduUpsert;
92 class KuduValue;
93 class KuduWriteOperation;
94 class ResourceMetrics;
95 
96 namespace internal {
97 class Batcher;
98 class ErrorCollector;
99 class GetTableSchemaRpc;
100 class LookupRpc;
101 class MetaCache;
102 class RemoteTablet;
103 class RemoteTabletServer;
104 class ReplicaController;
105 class RetrieveAuthzTokenRpc;
106 class ScanBatchDataInterface;
107 class WriteRpc;
108 template <class ReqClass, class RespClass>
109 class AsyncLeaderMasterRpc; // IWYU pragma: keep
110 } // namespace internal
111 
124 void KUDU_EXPORT InstallLoggingCallback(KuduLoggingCallback* cb);
125 
132 void KUDU_EXPORT UninstallLoggingCallback();
133 
149 void KUDU_EXPORT SetVerboseLogLevel(int level);
150 
160 Status KUDU_EXPORT SetInternalSignalNumber(int signum) WARN_UNUSED_RESULT;
161 
173 Status KUDU_EXPORT DisableSaslInitialization() WARN_UNUSED_RESULT;
174 
175 
198 Status KUDU_EXPORT DisableOpenSSLInitialization() WARN_UNUSED_RESULT;
199 
202 std::string KUDU_EXPORT GetShortVersionString();
203 
206 std::string KUDU_EXPORT GetAllVersionInfo();
207 
212 class KUDU_EXPORT KuduClientBuilder {
213  public:
214  KuduClientBuilder();
215  ~KuduClientBuilder();
216 
220  KuduClientBuilder& clear_master_server_addrs();
221 
227  KuduClientBuilder& master_server_addrs(const std::vector<std::string>& addrs);
228 
236  KuduClientBuilder& add_master_server_addr(const std::string& addr);
237 
247  KuduClientBuilder& default_admin_operation_timeout(const MonoDelta& timeout);
248 
256  KuduClientBuilder& default_rpc_timeout(const MonoDelta& timeout);
257 
265  KuduClientBuilder& import_authentication_credentials(std::string authn_creds);
266 
276  KuduClientBuilder& num_reactors(int num_reactors);
277 
288  Status Build(sp::shared_ptr<KuduClient>* client);
289 
290  private:
291  class KUDU_NO_EXPORT Data;
292 
293  friend class internal::ReplicaController;
294 
295  // Owned.
296  Data* data_;
297 
298  DISALLOW_COPY_AND_ASSIGN(KuduClientBuilder);
299 };
300 
327 class KUDU_EXPORT KuduClient : public sp::enable_shared_from_this<KuduClient> {
328  public:
329  ~KuduClient();
330 
335  KuduTableCreator* NewTableCreator();
336 
345  Status IsCreateTableInProgress(const std::string& table_name,
346  bool* create_in_progress);
347 
353  Status DeleteTable(const std::string& table_name);
354 
356 
367  Status DeleteTableInCatalogs(const std::string& table_name,
368  bool modify_external_catalogs) KUDU_NO_EXPORT;
370 
377  KuduTableAlterer* NewTableAlterer(const std::string& table_name);
378 
387  Status IsAlterTableInProgress(const std::string& table_name,
388  bool* alter_in_progress);
396  Status GetTableSchema(const std::string& table_name,
397  KuduSchema* schema);
398 
405  Status ListTabletServers(std::vector<KuduTabletServer*>* tablet_servers);
406 
414  Status ListTables(std::vector<std::string>* tables,
415  const std::string& filter = "");
416 
424  Status TableExists(const std::string& table_name, bool* exists);
425 
442  Status OpenTable(const std::string& table_name,
443  sp::shared_ptr<KuduTable>* table);
444 
450  sp::shared_ptr<KuduSession> NewSession();
451 
453 
466  Status GetTablet(const std::string& tablet_id,
467  KuduTablet** tablet) KUDU_NO_EXPORT;
468 
476  Status GetTableStatistics(const std::string& table_name,
477  KuduTableStatistics** statistics);
478 
486  std::string GetMasterAddresses() const KUDU_NO_EXPORT;
487 
489 
493 
494  CLOSEST_REPLICA,
495 
500  FIRST_REPLICA
501  };
502 
505  bool IsMultiMaster() const;
506 
508  const MonoDelta& default_admin_operation_timeout() const;
509 
511  const MonoDelta& default_rpc_timeout() const;
512 
515  static const uint64_t kNoTimestamp;
516 
527  uint64_t GetLatestObservedTimestamp() const;
528 
543  void SetLatestObservedTimestamp(uint64_t ht_timestamp);
544 
556  Status ExportAuthenticationCredentials(std::string* authn_creds) const;
557 
559 
565  std::string GetHiveMetastoreUris() const KUDU_NO_EXPORT;
566 
572  bool GetHiveMetastoreSaslEnabled() const KUDU_NO_EXPORT;
573 
583  std::string GetHiveMetastoreUuid() const KUDU_NO_EXPORT;
584 
591  std::string location() const KUDU_NO_EXPORT;
593 
594  private:
595  class KUDU_NO_EXPORT Data;
596 
597  template <class ReqClass, class RespClass>
598  friend class internal::AsyncLeaderMasterRpc;
599 
600  friend class ClientTest;
601  friend class ConnectToClusterBaseTest;
602  friend class KuduClientBuilder;
603  friend class KuduPartitionerBuilder;
604  friend class KuduScanToken;
605  friend class KuduScanTokenBuilder;
606  friend class KuduScanner;
607  friend class KuduSession;
608  friend class KuduTable;
609  friend class KuduTableAlterer;
610  friend class KuduTableCreator;
611  friend class internal::Batcher;
612  friend class internal::GetTableSchemaRpc;
613  friend class internal::LookupRpc;
614  friend class internal::MetaCache;
615  friend class internal::RemoteTablet;
616  friend class internal::RemoteTabletServer;
617  friend class internal::RetrieveAuthzTokenRpc;
618  friend class internal::WriteRpc;
619  friend class kudu::AuthzTokenTest;
620  friend class kudu::SecurityUnknownTskTest;
621  friend class tools::LeaderMasterProxy;
622  friend class tools::RemoteKsckCluster;
623 
624  FRIEND_TEST(kudu::ClientStressTest, TestUniqueClientIds);
625  FRIEND_TEST(ClientTest, TestCacheAuthzTokens);
626  FRIEND_TEST(ClientTest, TestGetSecurityInfoFromMaster);
627  FRIEND_TEST(ClientTest, TestGetTabletServerBlacklist);
628  FRIEND_TEST(ClientTest, TestMasterDown);
629  FRIEND_TEST(ClientTest, TestMasterLookupPermits);
630  FRIEND_TEST(ClientTest, TestMetaCacheExpiry);
631  FRIEND_TEST(ClientTest, TestNonCoveringRangePartitions);
632  FRIEND_TEST(ClientTest, TestRetrieveAuthzTokenInParallel);
633  FRIEND_TEST(ClientTest, TestReplicatedTabletWritesWithLeaderElection);
634  FRIEND_TEST(ClientTest, TestScanFaultTolerance);
635  FRIEND_TEST(ClientTest, TestScanTimeout);
636  FRIEND_TEST(ClientTest, TestWriteWithDeadMaster);
637  FRIEND_TEST(MasterFailoverTest, TestPauseAfterCreateTableIssued);
638 
639  KuduClient();
640 
641  // Owned.
642  Data* data_;
643 
644  DISALLOW_COPY_AND_ASSIGN(KuduClient);
645 };
646 
648 class KUDU_EXPORT KuduTabletServer {
649  public:
650  ~KuduTabletServer();
651 
654  const std::string& uuid() const;
655 
658  const std::string& hostname() const;
659 
662  uint16_t port() const;
663 
665 
670  const std::string& location() const KUDU_NO_EXPORT;
672 
673  private:
674  class KUDU_NO_EXPORT Data;
675 
676  friend class KuduClient;
677  friend class KuduScanner;
678  friend class KuduScanTokenBuilder;
679 
680  KuduTabletServer();
681 
682  // Owned.
683  Data* data_;
684 
685  DISALLOW_COPY_AND_ASSIGN(KuduTabletServer);
686 };
687 
689 class KUDU_EXPORT KuduReplica {
690  public:
691  ~KuduReplica();
692 
697  bool is_leader() const;
698 
700  const KuduTabletServer& ts() const;
701 
702  private:
703  friend class KuduClient;
704  friend class KuduScanTokenBuilder;
705  friend class internal::ReplicaController;
706 
707  class KUDU_NO_EXPORT Data;
708 
709  KuduReplica();
710 
711  // Owned.
712  Data* data_;
713 
714  DISALLOW_COPY_AND_ASSIGN(KuduReplica);
715 };
716 
718 class KUDU_EXPORT KuduTablet {
719  public:
720  ~KuduTablet();
721 
724  const std::string& id() const;
725 
731  const std::vector<const KuduReplica*>& replicas() const;
732 
733  private:
734  friend class KuduClient;
735  friend class KuduScanTokenBuilder;
736 
737  class KUDU_NO_EXPORT Data;
738 
739  KuduTablet();
740 
741  // Owned.
742  Data* data_;
743 
744  DISALLOW_COPY_AND_ASSIGN(KuduTablet);
745 };
746 
748 class KUDU_EXPORT KuduTableCreator {
749  public:
750  ~KuduTableCreator();
751 
763  KuduTableCreator& table_name(const std::string& name);
764 
775  KuduTableCreator& schema(const KuduSchema* schema);
776 
793  KuduTableCreator& add_hash_partitions(const std::vector<std::string>& columns,
794  int32_t num_buckets);
795 
811  KuduTableCreator& add_hash_partitions(const std::vector<std::string>& columns,
812  int32_t num_buckets, int32_t seed);
813 
826  KuduTableCreator& set_range_partition_columns(const std::vector<std::string>& columns);
827 
832  };
833 
862  KuduTableCreator& add_range_partition(KuduPartialRow* lower_bound,
863  KuduPartialRow* upper_bound,
864  RangePartitionBound lower_bound_type = INCLUSIVE_BOUND,
865  RangePartitionBound upper_bound_type = EXCLUSIVE_BOUND);
866 
875  KuduTableCreator& add_range_partition_split(KuduPartialRow* split_row);
876 
882  KuduTableCreator& split_rows(const std::vector<const KuduPartialRow*>& split_rows)
883  ATTRIBUTE_DEPRECATED("use add_range_partition_split() instead");
884 
894  KuduTableCreator& num_replicas(int n_replicas);
895 
907  KuduTableCreator& dimension_label(const std::string& dimension_label);
908 
916  KuduTableCreator& extra_configs(const std::map<std::string, std::string>& extra_configs);
917 
927  KuduTableCreator& timeout(const MonoDelta& timeout);
928 
936  KuduTableCreator& wait(bool wait);
937 
951  Status Create();
952 
953  private:
954  class KUDU_NO_EXPORT Data;
955 
956  friend class KuduClient;
957 
958  explicit KuduTableCreator(KuduClient* client);
959 
960  // Owned.
961  Data* data_;
962 
963  DISALLOW_COPY_AND_ASSIGN(KuduTableCreator);
964 };
965 
967 class KUDU_EXPORT KuduTableStatistics {
968  public:
971 
976  int64_t on_disk_size() const;
977 
982  int64_t live_row_count() const;
983 
987  std::string ToString() const;
988 
989  private:
990  class KUDU_NO_EXPORT Data;
991 
992  friend class KuduClient;
993 
994  // Owned.
995  Data* data_;
996 
997  DISALLOW_COPY_AND_ASSIGN(KuduTableStatistics);
998 };
999 
1020 class KUDU_EXPORT KuduTable : public sp::enable_shared_from_this<KuduTable> {
1021  public:
1022  ~KuduTable();
1023 
1025  const std::string& name() const;
1026 
1034  const std::string& id() const;
1035 
1037  const KuduSchema& schema() const;
1038 
1040  int num_replicas() const;
1041 
1045  KuduInsert* NewInsert();
1046 
1050  KuduInsertIgnore* NewInsertIgnore();
1051 
1055  KuduUpsert* NewUpsert();
1056 
1060  KuduUpdate* NewUpdate();
1061 
1065  KuduDelete* NewDelete();
1066 
1090  KuduPredicate* NewComparisonPredicate(const Slice& col_name,
1092  KuduValue* value);
1093 
1120  KuduPredicate* NewInBloomFilterPredicate(const Slice& col_name,
1121  std::vector<KuduBloomFilter*>* bloom_filters);
1122 
1157  KuduPredicate* NewInBloomFilterPredicate(const Slice& col_name,
1158  const Slice& allocator,
1159  const std::vector<Slice>& bloom_filters);
1161 
1185  KuduPredicate* NewInListPredicate(const Slice& col_name,
1186  std::vector<KuduValue*>* values);
1187 
1198  KuduPredicate* NewIsNotNullPredicate(const Slice& col_name);
1199 
1210  KuduPredicate* NewIsNullPredicate(const Slice& col_name);
1211 
1214  KuduClient* client() const;
1215 
1217  const PartitionSchema& partition_schema() const;
1218 
1220  const std::map<std::string, std::string>& extra_configs() const;
1221 
1223 
1233  Status ListPartitions(std::vector<Partition>* partitions) KUDU_NO_EXPORT;
1234 
1236 
1237  private:
1238  class KUDU_NO_EXPORT Data;
1239 
1240  friend class KuduClient;
1241  friend class KuduPartitioner;
1242 
1243  KuduTable(const sp::shared_ptr<KuduClient>& client,
1244  const std::string& name,
1245  const std::string& id,
1246  int num_replicas,
1247  const KuduSchema& schema,
1248  const PartitionSchema& partition_schema,
1249  const std::map<std::string, std::string>& extra_configs);
1250 
1251  // Owned.
1252  Data* data_;
1253 
1254  DISALLOW_COPY_AND_ASSIGN(KuduTable);
1255 };
1256 
1268 class KUDU_EXPORT KuduTableAlterer {
1269  public:
1270  ~KuduTableAlterer();
1271 
1277  KuduTableAlterer* RenameTo(const std::string& new_name);
1278 
1288  KuduColumnSpec* AddColumn(const std::string& name);
1289 
1298  KuduColumnSpec* AlterColumn(const std::string& name);
1299 
1307  KuduTableAlterer* DropColumn(const std::string& name);
1308 
1340  KuduTableAlterer* AddRangePartition(
1341  KuduPartialRow* lower_bound,
1342  KuduPartialRow* upper_bound,
1345 
1380  KuduTableAlterer* AddRangePartitionWithDimension(
1381  KuduPartialRow* lower_bound,
1382  KuduPartialRow* upper_bound,
1383  const std::string& dimension_label,
1386 
1414  KuduTableAlterer* DropRangePartition(
1415  KuduPartialRow* lower_bound,
1416  KuduPartialRow* upper_bound,
1419 
1429  KuduTableAlterer* AlterExtraConfig(const std::map<std::string, std::string>& extra_configs);
1430 
1440  KuduTableAlterer* timeout(const MonoDelta& timeout);
1441 
1453  KuduTableAlterer* wait(bool wait);
1454 
1456 
1465  KuduTableAlterer* modify_external_catalogs(bool modify_external_catalogs) KUDU_NO_EXPORT;
1466 
1468 
1473  Status Alter();
1474 
1475  private:
1476  class KUDU_NO_EXPORT Data;
1477 
1478  friend class KuduClient;
1479 
1480  KuduTableAlterer(KuduClient* client,
1481  const std::string& name);
1482 
1483  // Owned.
1484  Data* data_;
1485 
1486  DISALLOW_COPY_AND_ASSIGN(KuduTableAlterer);
1487 };
1488 
1494 class KUDU_EXPORT KuduError {
1495  public:
1496  ~KuduError();
1497 
1499  const Status& status() const;
1500 
1502  const KuduWriteOperation& failed_op() const;
1503 
1511  KuduWriteOperation* release_failed_op();
1512 
1523  bool was_possibly_successful() const;
1524 
1525  private:
1526  class KUDU_NO_EXPORT Data;
1527 
1528  friend class internal::Batcher;
1529  friend class internal::ErrorCollector;
1530  friend class KuduSession;
1531 
1532  KuduError(KuduWriteOperation* failed_op, const Status& error);
1533 
1534  // Owned.
1535  Data* data_;
1536 
1537  DISALLOW_COPY_AND_ASSIGN(KuduError);
1538 };
1539 
1540 
1599 class KUDU_EXPORT KuduSession : public sp::enable_shared_from_this<KuduSession> {
1600  public:
1601  ~KuduSession();
1602 
1604  enum FlushMode {
1610 
1638 
1651  MANUAL_FLUSH
1652  };
1653 
1662  Status SetFlushMode(FlushMode m) WARN_UNUSED_RESULT;
1663 
1685 
1700  COMMIT_WAIT
1701  };
1702 
1708  Status SetExternalConsistencyMode(ExternalConsistencyMode m)
1709  WARN_UNUSED_RESULT;
1710 
1728  Status SetMutationBufferSpace(size_t size_bytes) WARN_UNUSED_RESULT;
1729 
1753  Status SetMutationBufferFlushWatermark(double watermark_pct)
1754  WARN_UNUSED_RESULT;
1755 
1777  Status SetMutationBufferFlushInterval(unsigned int millis) WARN_UNUSED_RESULT;
1778 
1803  Status SetMutationBufferMaxNum(unsigned int max_num) WARN_UNUSED_RESULT;
1804 
1810  void SetTimeoutMillis(int millis);
1811 
1815 
1839  Status Apply(KuduWriteOperation* write_op) WARN_UNUSED_RESULT;
1840 
1855  Status Flush() WARN_UNUSED_RESULT;
1856 
1897  void FlushAsync(KuduStatusCallback* cb);
1898 
1901  Status Close() WARN_UNUSED_RESULT;
1902 
1911  bool HasPendingOperations() const;
1912 
1934  int CountBufferedOperations() const
1935  ATTRIBUTE_DEPRECATED("this method is experimental and will disappear "
1936  "in a future release");
1937 
1962  Status SetErrorBufferSpace(size_t size_bytes);
1963 
1973  int CountPendingErrors() const;
1974 
1985  void GetPendingErrors(std::vector<KuduError*>* errors, bool* overflowed);
1986 
1988  KuduClient* client() const;
1989 
1990  private:
1991  class KUDU_NO_EXPORT Data;
1992 
1993  friend class KuduClient;
1994  friend class internal::Batcher;
1995  friend class ClientTest;
1996  FRIEND_TEST(ClientTest, TestAutoFlushBackgroundApplyBlocks);
1997  FRIEND_TEST(ClientTest, TestAutoFlushBackgroundAndErrorCollector);
1998 
1999  explicit KuduSession(const sp::shared_ptr<KuduClient>& client);
2000 
2001  // Owned.
2002  Data* data_;
2003 
2004  DISALLOW_COPY_AND_ASSIGN(KuduSession);
2005 };
2006 
2007 
2012 class KUDU_EXPORT KuduScanner {
2013  public:
2015  enum ReadMode {
2024 
2045 
2056  READ_YOUR_WRITES
2057  };
2058 
2062  enum OrderMode {
2069 
2074  ORDERED
2075  };
2076 
2080  enum { kScanTimeoutMillis = 30000 };
2081 
2087  explicit KuduScanner(KuduTable* table);
2088  ~KuduScanner();
2089 
2099  Status SetProjectedColumnNames(const std::vector<std::string>& col_names)
2100  WARN_UNUSED_RESULT;
2101 
2111  Status SetProjectedColumnIndexes(const std::vector<int>& col_indexes)
2112  WARN_UNUSED_RESULT;
2113 
2119  Status SetProjectedColumns(const std::vector<std::string>& col_names)
2120  WARN_UNUSED_RESULT
2121  ATTRIBUTE_DEPRECATED("use SetProjectedColumnNames() instead");
2122 
2131  Status AddConjunctPredicate(KuduPredicate* pred) WARN_UNUSED_RESULT;
2132 
2141  Status AddLowerBound(const KuduPartialRow& key);
2142 
2150  Status AddLowerBoundRaw(const Slice& key)
2151  ATTRIBUTE_DEPRECATED("use AddLowerBound() instead");
2152 
2161  Status AddExclusiveUpperBound(const KuduPartialRow& key);
2162 
2170  Status AddExclusiveUpperBoundRaw(const Slice& key)
2171  ATTRIBUTE_DEPRECATED("use AddExclusiveUpperBound() instead");
2172 
2181  Status AddLowerBoundPartitionKeyRaw(const Slice& partition_key);
2182 
2191  Status AddExclusiveUpperBoundPartitionKeyRaw(const Slice& partition_key);
2192 
2199  Status SetCacheBlocks(bool cache_blocks);
2200 
2202  Status Open();
2203 
2222  Status KeepAlive();
2223 
2232  void Close();
2233 
2242  bool HasMoreRows() const;
2243 
2255  Status NextBatch(std::vector<KuduRowResult>* rows)
2256  ATTRIBUTE_DEPRECATED("use NextBatch(KuduScanBatch*) instead");
2257 
2269  Status NextBatch(KuduScanBatch* batch);
2270 
2282  Status NextBatch(KuduColumnarScanBatch* batch);
2283 
2292  Status GetCurrentServer(KuduTabletServer** server);
2293 
2295  const ResourceMetrics& GetResourceMetrics() const;
2296 
2303  Status SetBatchSizeBytes(uint32_t batch_size);
2304 
2312  Status SetSelection(KuduClient::ReplicaSelection selection)
2313  WARN_UNUSED_RESULT;
2314 
2320  Status SetReadMode(ReadMode read_mode) WARN_UNUSED_RESULT;
2321 
2327  Status SetOrderMode(OrderMode order_mode) WARN_UNUSED_RESULT
2328  ATTRIBUTE_DEPRECATED("use SetFaultTolerant() instead");
2329 
2342  Status SetFaultTolerant() WARN_UNUSED_RESULT;
2343 
2349  Status SetSnapshotMicros(uint64_t snapshot_timestamp_micros) WARN_UNUSED_RESULT;
2350 
2360  Status SetSnapshotRaw(uint64_t snapshot_timestamp) WARN_UNUSED_RESULT;
2361 
2363 
2378  Status SetDiffScan(uint64_t start_timestamp, uint64_t end_timestamp)
2379  WARN_UNUSED_RESULT KUDU_NO_EXPORT;
2380 
2382 
2388  Status SetTimeoutMillis(int millis);
2389 
2391  KuduSchema GetProjectionSchema() const;
2392 
2394  //
2400  static const uint64_t NO_FLAGS = 0;
2408  static const uint64_t PAD_UNIXTIME_MICROS_TO_16_BYTES = 1 << 0;
2409 
2417  static const uint64_t COLUMNAR_LAYOUT = 1 << 1;
2418 
2446  Status SetRowFormatFlags(uint64_t flags);
2448 
2454  Status SetLimit(int64_t limit) WARN_UNUSED_RESULT;
2455 
2461  std::string ToString() const;
2462 
2463  private:
2464  class KUDU_NO_EXPORT Data;
2465 
2466  Status NextBatch(internal::ScanBatchDataInterface* batch);
2467 
2468  friend class KuduScanToken;
2469  FRIEND_TEST(ClientTest, TestBlockScannerHijackingAttempts);
2470  FRIEND_TEST(ClientTest, TestScanCloseProxy);
2471  FRIEND_TEST(ClientTest, TestScanFaultTolerance);
2472  FRIEND_TEST(ClientTest, TestScanNoBlockCaching);
2473  FRIEND_TEST(ClientTest, TestScanTimeout);
2474  FRIEND_TEST(ClientTest, TestReadAtSnapshotNoTimestampSet);
2475  FRIEND_TEST(ConsistencyITest, TestSnapshotScanTimestampReuse);
2476  FRIEND_TEST(ScanTokenTest, TestScanTokens);
2477 
2478  // Owned.
2479  Data* data_;
2480 
2481  DISALLOW_COPY_AND_ASSIGN(KuduScanner);
2482 };
2483 
2504 class KUDU_EXPORT KuduScanToken {
2505  public:
2506 
2507  ~KuduScanToken();
2508 
2519  Status IntoKuduScanner(KuduScanner** scanner) const WARN_UNUSED_RESULT;
2520 
2522  const KuduTablet& tablet() const;
2523 
2531  Status Serialize(std::string* buf) const WARN_UNUSED_RESULT;
2532 
2544  static Status DeserializeIntoScanner(KuduClient* client,
2545  const std::string& serialized_token,
2546  KuduScanner** scanner) WARN_UNUSED_RESULT;
2547 
2548  private:
2549  class KUDU_NO_EXPORT Data;
2550 
2551  friend class KuduScanTokenBuilder;
2552 
2553  KuduScanToken();
2554 
2555  // Owned.
2556  Data* data_;
2557 
2558  DISALLOW_COPY_AND_ASSIGN(KuduScanToken);
2559 };
2560 
2564 class KUDU_EXPORT KuduScanTokenBuilder {
2565  public:
2566 
2572  explicit KuduScanTokenBuilder(KuduTable* table);
2574 
2584  Status SetProjectedColumnNames(const std::vector<std::string>& col_names)
2585  WARN_UNUSED_RESULT;
2586 
2588  Status SetProjectedColumnIndexes(const std::vector<int>& col_indexes)
2589  WARN_UNUSED_RESULT;
2590 
2592  Status AddConjunctPredicate(KuduPredicate* pred) WARN_UNUSED_RESULT;
2593 
2595  Status AddLowerBound(const KuduPartialRow& key) WARN_UNUSED_RESULT;
2596 
2605  Status AddUpperBound(const KuduPartialRow& key) WARN_UNUSED_RESULT;
2606 
2608  Status SetCacheBlocks(bool cache_blocks) WARN_UNUSED_RESULT;
2609 
2616  Status SetBatchSizeBytes(uint32_t batch_size) WARN_UNUSED_RESULT;
2617 
2625  Status SetSelection(KuduClient::ReplicaSelection selection)
2626  WARN_UNUSED_RESULT;
2627 
2629  Status SetReadMode(KuduScanner::ReadMode read_mode) WARN_UNUSED_RESULT;
2630 
2632  Status SetFaultTolerant() WARN_UNUSED_RESULT;
2633 
2635  Status SetSnapshotMicros(uint64_t snapshot_timestamp_micros)
2636  WARN_UNUSED_RESULT;
2637 
2639  Status SetSnapshotRaw(uint64_t snapshot_timestamp) WARN_UNUSED_RESULT;
2640 
2642 
2644  Status SetDiffScan(uint64_t start_timestamp, uint64_t end_timestamp)
2645  WARN_UNUSED_RESULT KUDU_NO_EXPORT;
2647 
2649  Status SetTimeoutMillis(int millis) WARN_UNUSED_RESULT;
2650 
2659  Status Build(std::vector<KuduScanToken*>* tokens) WARN_UNUSED_RESULT;
2660 
2661  private:
2662  class KUDU_NO_EXPORT Data;
2663 
2664  // Owned.
2665  Data* data_;
2666 
2667  DISALLOW_COPY_AND_ASSIGN(KuduScanTokenBuilder);
2668 };
2669 
2671 class KUDU_EXPORT KuduPartitionerBuilder {
2672  public:
2677  explicit KuduPartitionerBuilder(sp::shared_ptr<KuduTable> table);
2679 
2685  KuduPartitionerBuilder* SetBuildTimeout(MonoDelta timeout);
2686 
2705  Status Build(KuduPartitioner** partitioner);
2706  private:
2707  class KUDU_NO_EXPORT Data;
2708 
2709  // Owned.
2710  Data* data_;
2711 
2712  DISALLOW_COPY_AND_ASSIGN(KuduPartitionerBuilder);
2713 };
2714 
2725 class KUDU_EXPORT KuduPartitioner {
2726  public:
2727  ~KuduPartitioner();
2728 
2732  int NumPartitions() const;
2733 
2745  Status PartitionRow(const KuduPartialRow& row, int* partition);
2746  private:
2747  class KUDU_NO_EXPORT Data;
2748 
2749  friend class KuduPartitionerBuilder;
2750 
2751  explicit KuduPartitioner(Data* data);
2752  Data* data_; // Owned.
2753 };
2754 
2755 
2756 } // namespace client
2757 } // namespace kudu
2758 #endif
The interface for all logging callbacks.
Definition: callbacks.h:44
A single row update to be sent to the cluster.
Definition: write_op.h:224
A representation of a table&#39;s schema.
Definition: schema.h:594
A single row insert ignore to be sent to the cluster, duplicate row errors are ignored.
Definition: write_op.h:171
A representation of an operation&#39;s outcome.
Definition: status.h:165
A constant cell value with a specific type.
Definition: value.h:35
Builder for Partitioner instances.
Definition: client.h:2671
ExternalConsistencyMode
The possible external consistency modes on which Kudu operates.
Definition: client.h:1665
Definition: callbacks.h:28
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:144
Builder API for specifying or altering a column within a table schema.
Definition: schema.h:338
static const uint64_t kNoTimestamp
Definition: client.h:515
ReplicaSelection
Policy with which to choose amongst multiple replicas.
Definition: client.h:491
A single row upsert to be sent to the cluster.
Definition: write_op.h:197
Builds scan tokens for a table.
Definition: client.h:2564
ComparisonOp
Supported comparison operators.
Definition: scan_predicate.h:46
Alters an existing table based on the provided steps.
Definition: client.h:1268
OrderMode
Definition: client.h:2062
Definition: client.h:2068
Smart pointer typedefs for externally-faced code.
Definition: client.h:2725
A representation of comparison predicate for Kudu queries.
Definition: scan_predicate.h:43
An exclusive bound.
Definition: client.h:830
This class represents an error which occurred in a write operation.
Definition: client.h:1494
A handle for a connection to a cluster.
Definition: client.h:327
An inclusive bound.
Definition: client.h:831
In-memory representation of a remote tablet server.
Definition: client.h:648
The interface for all status callbacks.
Definition: callbacks.h:161
A wrapper around externally allocated data.
Definition: slice.h:51
A representation of a table on a particular cluster.
Definition: client.h:1020
This class is a representation of a single scan.
Definition: client.h:2012
ReadMode
The read modes for scanners.
Definition: client.h:2015
A "factory" for KuduClient objects.
Definition: client.h:212
A single-row write operation to be sent to a Kudu table.
Definition: write_op.h:66
Select the LEADER replica.
Definition: client.h:492
RangePartitionBound
Range partition bound type.
Definition: client.h:829
In-memory statistics of table.
Definition: client.h:967
In-memory representation of a remote tablet.
Definition: client.h:718
In-memory representation of a remote tablet&#39;s replica.
Definition: client.h:689
FlushMode
Modes of flush operations.
Definition: client.h:1604
A scan descriptor limited to a single physical contiguous location.
Definition: client.h:2504
A single row delete to be sent to the cluster.
Definition: write_op.h:251
A generic catalog of simple metrics.
Definition: resource_metrics.h:39
A helper class to create a new table with the desired options.
Definition: client.h:748
A row which may only contain values for a subset of the columns.
Definition: partial_row.h:72
Representation of a Kudu client session.
Definition: client.h:1599
A batch of zero or more rows returned by a scan operation.
Definition: scan_batch.h:84
A representation of a time interval.
Definition: monotime.h:57