librdkafka
The Apache Kafka C/C++ client library
rdkafkacpp.h
Go to the documentation of this file.
1 /*
2  * librdkafka - Apache Kafka C/C++ library
3  *
4  * Copyright (c) 2014 Magnus Edenhill
5  * All rights reserved.
6  *
7  * Redistribution and use in source and binary forms, with or without
8  * modification, are permitted provided that the following conditions are met:
9  *
10  * 1. Redistributions of source code must retain the above copyright notice,
11  * this list of conditions and the following disclaimer.
12  * 2. Redistributions in binary form must reproduce the above copyright notice,
13  * this list of conditions and the following disclaimer in the documentation
14  * and/or other materials provided with the distribution.
15  *
16  * THIS SOFTWARE IS PROVIDED BY THE COPYRIGHT HOLDERS AND CONTRIBUTORS "AS IS"
17  * AND ANY EXPRESS OR IMPLIED WARRANTIES, INCLUDING, BUT NOT LIMITED TO, THE
18  * IMPLIED WARRANTIES OF MERCHANTABILITY AND FITNESS FOR A PARTICULAR PURPOSE
19  * ARE DISCLAIMED. IN NO EVENT SHALL THE COPYRIGHT OWNER OR CONTRIBUTORS BE
20  * LIABLE FOR ANY DIRECT, INDIRECT, INCIDENTAL, SPECIAL, EXEMPLARY, OR
21  * CONSEQUENTIAL DAMAGES (INCLUDING, BUT NOT LIMITED TO, PROCUREMENT OF
22  * SUBSTITUTE GOODS OR SERVICES; LOSS OF USE, DATA, OR PROFITS; OR BUSINESS
23  * INTERRUPTION) HOWEVER CAUSED AND ON ANY THEORY OF LIABILITY, WHETHER IN
24  * CONTRACT, STRICT LIABILITY, OR TORT (INCLUDING NEGLIGENCE OR OTHERWISE)
25  * ARISING IN ANY WAY OUT OF THE USE OF THIS SOFTWARE, EVEN IF ADVISED OF THE
26  * POSSIBILITY OF SUCH DAMAGE.
27  */
28 
29 #ifndef _RDKAFKACPP_H_
30 #define _RDKAFKACPP_H_
31 
50 #include <string>
51 #include <list>
52 #include <vector>
53 #include <cstdlib>
54 #include <cstring>
55 #include <stdint.h>
56 #include <sys/types.h>
57 
58 #ifdef _WIN32
59 #ifndef ssize_t
60 #ifndef _BASETSD_H_
61 #include <basetsd.h>
62 #endif
63 #ifndef _SSIZE_T_DEFINED
64 #define _SSIZE_T_DEFINED
65 typedef SSIZE_T ssize_t;
66 #endif
67 #endif
68 #undef RD_EXPORT
69 #ifdef LIBRDKAFKA_STATICLIB
70 #define RD_EXPORT
71 #else
72 #ifdef LIBRDKAFKACPP_EXPORTS
73 #define RD_EXPORT __declspec(dllexport)
74 #else
75 #define RD_EXPORT __declspec(dllimport)
76 #endif
77 #endif
78 #else
79 #define RD_EXPORT
80 #endif
81 
84 extern "C" {
85  /* Forward declarations */
86  struct rd_kafka_s;
87  struct rd_kafka_topic_s;
88  struct rd_kafka_message_s;
89  struct rd_kafka_conf_s;
90  struct rd_kafka_topic_conf_s;
91 }
92 
93 namespace RdKafka {
94 
114 #define RD_KAFKA_VERSION 0x010802ff
115 
121 RD_EXPORT
122 int version ();
123 
127 RD_EXPORT
128 std::string version_str();
129 
134 RD_EXPORT
135 std::string get_debug_contexts();
136 
146 RD_EXPORT
147 int wait_destroyed(int timeout_ms);
148 
159 RD_EXPORT
160 void *mem_malloc (size_t size);
161 
175 RD_EXPORT
176 void mem_free (void *ptr);
177 
200 enum ErrorCode {
201  /* Internal errors to rdkafka: */
203  ERR__BEGIN = -200,
205  ERR__BAD_MSG = -199,
209  ERR__DESTROY = -197,
211  ERR__FAIL = -196,
217  ERR__RESOLVE = -193,
228  ERR__FS = -189,
244  ERR__SSL = -181,
262  ERR__STATE = -172,
280  ERR__INTR = -163,
290  ERR__PARTIAL = -158,
294  ERR__NOENT = -156,
300  ERR__RETRY = -153,
306  ERR__FATAL = -150,
318  ERR__FENCED = -144,
324  ERR__NOOP = -141,
327 
329  ERR__END = -100,
330 
331  /* Kafka broker errors: */
365 #define ERR_GROUP_LOAD_IN_PROGRESS ERR_COORDINATOR_LOAD_IN_PROGRESS
366 
369 #define ERR_GROUP_COORDINATOR_NOT_AVAILABLE ERR_COORDINATOR_NOT_AVAILABLE
370 
373 #define ERR_NOT_COORDINATOR_FOR_GROUP ERR_NOT_COORDINATOR
374 
545 };
546 
547 
551 RD_EXPORT
552 std::string err2str(RdKafka::ErrorCode err);
553 
554 
555 
564  CERT__CNT
565 };
566 
575  CERT_ENC__CNT
576 };
577 
583 /* Forward declarations */
584 class Handle;
585 class Producer;
586 class Message;
587 class Headers;
588 class Queue;
589 class Event;
590 class Topic;
591 class TopicPartition;
592 class Metadata;
593 class KafkaConsumer;
613 class RD_EXPORT Error {
614  public:
615 
619  static Error *create (ErrorCode code, const std::string *errstr);
620 
621  virtual ~Error () { }
622 
623  /*
624  * Error accessor methods
625  */
626 
630  virtual ErrorCode code () const = 0;
631 
635  virtual std::string name () const = 0;
636 
640  virtual std::string str () const = 0;
641 
646  virtual bool is_fatal () const = 0;
647 
651  virtual bool is_retriable () const = 0;
652 
664  virtual bool txn_requires_abort () const = 0;
665 };
666 
698 class RD_EXPORT DeliveryReportCb {
699  public:
703  virtual void dr_cb (Message &message) = 0;
704 
705  virtual ~DeliveryReportCb() { }
706 };
707 
708 
736 class RD_EXPORT OAuthBearerTokenRefreshCb {
737  public:
745  virtual void oauthbearer_token_refresh_cb (RdKafka::Handle* handle,
746  const std::string &oauthbearer_config) = 0;
747 
748  virtual ~OAuthBearerTokenRefreshCb() { }
749 };
750 
751 
759 class RD_EXPORT PartitionerCb {
760  public:
777  virtual int32_t partitioner_cb (const Topic *topic,
778  const std::string *key,
779  int32_t partition_cnt,
780  void *msg_opaque) = 0;
781 
782  virtual ~PartitionerCb() { }
783 };
784 
790  public:
799  virtual int32_t partitioner_cb (const Topic *topic,
800  const void *key,
801  size_t key_len,
802  int32_t partition_cnt,
803  void *msg_opaque) = 0;
804 
805  virtual ~PartitionerKeyPointerCb() { }
806 };
807 
808 
809 
818 class RD_EXPORT EventCb {
819  public:
825  virtual void event_cb (Event &event) = 0;
826 
827  virtual ~EventCb() { }
828 };
829 
830 
834 class RD_EXPORT Event {
835  public:
837  enum Type {
841  EVENT_THROTTLE
842  };
843 
845  enum Severity {
846  EVENT_SEVERITY_EMERG = 0,
847  EVENT_SEVERITY_ALERT = 1,
848  EVENT_SEVERITY_CRITICAL = 2,
849  EVENT_SEVERITY_ERROR = 3,
850  EVENT_SEVERITY_WARNING = 4,
851  EVENT_SEVERITY_NOTICE = 5,
852  EVENT_SEVERITY_INFO = 6,
853  EVENT_SEVERITY_DEBUG = 7
854  };
855 
856  virtual ~Event () { }
857 
858  /*
859  * Event Accessor methods
860  */
861 
866  virtual Type type () const = 0;
867 
872  virtual ErrorCode err () const = 0;
873 
878  virtual Severity severity () const = 0;
879 
884  virtual std::string fac () const = 0;
885 
894  virtual std::string str () const = 0;
895 
900  virtual int throttle_time () const = 0;
901 
906  virtual std::string broker_name () const = 0;
907 
912  virtual int broker_id () const = 0;
913 
914 
920  virtual bool fatal () const = 0;
921 };
922 
923 
924 
928 class RD_EXPORT ConsumeCb {
929  public:
937  virtual void consume_cb (Message &message, void *opaque) = 0;
938 
939  virtual ~ConsumeCb() { }
940 };
941 
942 
946 class RD_EXPORT RebalanceCb {
947 public:
1016  virtual void rebalance_cb (RdKafka::KafkaConsumer *consumer,
1017  RdKafka::ErrorCode err,
1018  std::vector<TopicPartition*>&partitions) = 0;
1019 
1020  virtual ~RebalanceCb() { }
1021 };
1022 
1023 
1027 class RD_EXPORT OffsetCommitCb {
1028 public:
1044  virtual void offset_commit_cb(RdKafka::ErrorCode err,
1045  std::vector<TopicPartition*>&offsets) = 0;
1046 
1047  virtual ~OffsetCommitCb() { }
1048 };
1049 
1050 
1051 
1057 class RD_EXPORT SslCertificateVerifyCb {
1058 public:
1095  virtual bool ssl_cert_verify_cb (const std::string &broker_name,
1096  int32_t broker_id,
1097  int *x509_error,
1098  int depth,
1099  const char *buf, size_t size,
1100  std::string &errstr) = 0;
1101 
1102  virtual ~SslCertificateVerifyCb() {}
1103 };
1104 
1105 
1110 class RD_EXPORT SocketCb {
1111  public:
1125  virtual int socket_cb (int domain, int type, int protocol) = 0;
1126 
1127  virtual ~SocketCb() { }
1128 };
1129 
1130 
1135 class RD_EXPORT OpenCb {
1136  public:
1148  virtual int open_cb (const std::string &path, int flags, int mode) = 0;
1149 
1150  virtual ~OpenCb() { }
1151 };
1152 
1153 
1174 class RD_EXPORT Conf {
1175  public:
1179  enum ConfType {
1181  CONF_TOPIC
1182  };
1183 
1187  enum ConfResult {
1188  CONF_UNKNOWN = -2,
1189  CONF_INVALID = -1,
1190  CONF_OK = 0
1191  };
1192 
1193 
1197  static Conf *create (ConfType type);
1198 
1199  virtual ~Conf () { }
1200 
1214  virtual Conf::ConfResult set (const std::string &name,
1215  const std::string &value,
1216  std::string &errstr) = 0;
1217 
1219  virtual Conf::ConfResult set (const std::string &name,
1220  DeliveryReportCb *dr_cb,
1221  std::string &errstr) = 0;
1222 
1224  virtual Conf::ConfResult set (const std::string &name,
1225  OAuthBearerTokenRefreshCb *oauthbearer_token_refresh_cb,
1226  std::string &errstr) = 0;
1227 
1229  virtual Conf::ConfResult set (const std::string &name,
1230  EventCb *event_cb,
1231  std::string &errstr) = 0;
1232 
1240  virtual Conf::ConfResult set (const std::string &name,
1241  const Conf *topic_conf,
1242  std::string &errstr) = 0;
1243 
1245  virtual Conf::ConfResult set (const std::string &name,
1246  PartitionerCb *partitioner_cb,
1247  std::string &errstr) = 0;
1248 
1250  virtual Conf::ConfResult set (const std::string &name,
1251  PartitionerKeyPointerCb *partitioner_kp_cb,
1252  std::string &errstr) = 0;
1253 
1255  virtual Conf::ConfResult set (const std::string &name, SocketCb *socket_cb,
1256  std::string &errstr) = 0;
1257 
1259  virtual Conf::ConfResult set (const std::string &name, OpenCb *open_cb,
1260  std::string &errstr) = 0;
1261 
1263  virtual Conf::ConfResult set (const std::string &name,
1264  RebalanceCb *rebalance_cb,
1265  std::string &errstr) = 0;
1266 
1268  virtual Conf::ConfResult set (const std::string &name,
1269  OffsetCommitCb *offset_commit_cb,
1270  std::string &errstr) = 0;
1271 
1276  virtual Conf::ConfResult set(const std::string &name,
1277  SslCertificateVerifyCb *ssl_cert_verify_cb,
1278  std::string &errstr) = 0;
1279 
1312  virtual Conf::ConfResult set_ssl_cert (RdKafka::CertificateType cert_type,
1314  const void *buffer, size_t size,
1315  std::string &errstr) = 0;
1316 
1328  virtual Conf::ConfResult get(const std::string &name,
1329  std::string &value) const = 0;
1330 
1334  virtual Conf::ConfResult get(DeliveryReportCb *&dr_cb) const = 0;
1335 
1339  virtual Conf::ConfResult get(
1340  OAuthBearerTokenRefreshCb *&oauthbearer_token_refresh_cb) const = 0;
1341 
1345  virtual Conf::ConfResult get(EventCb *&event_cb) const = 0;
1346 
1350  virtual Conf::ConfResult get(PartitionerCb *&partitioner_cb) const = 0;
1351 
1355  virtual Conf::ConfResult get(PartitionerKeyPointerCb *&partitioner_kp_cb) const = 0;
1356 
1360  virtual Conf::ConfResult get(SocketCb *&socket_cb) const = 0;
1361 
1365  virtual Conf::ConfResult get(OpenCb *&open_cb) const = 0;
1366 
1370  virtual Conf::ConfResult get(RebalanceCb *&rebalance_cb) const = 0;
1371 
1375  virtual Conf::ConfResult get(OffsetCommitCb *&offset_commit_cb) const = 0;
1376 
1378  virtual Conf::ConfResult get(SslCertificateVerifyCb *&ssl_cert_verify_cb) const = 0;
1379 
1382  virtual std::list<std::string> *dump () = 0;
1383 
1385  virtual Conf::ConfResult set (const std::string &name, ConsumeCb *consume_cb,
1386  std::string &errstr) = 0;
1387 
1404  virtual struct rd_kafka_conf_s *c_ptr_global () = 0;
1405 
1423  virtual struct rd_kafka_topic_conf_s *c_ptr_topic () = 0;
1424 
1437  virtual Conf::ConfResult set_engine_callback_data (void *value,
1438  std::string &errstr) = 0;
1439 };
1440 
1453 class RD_EXPORT Handle {
1454  public:
1455  virtual ~Handle() { }
1456 
1458  virtual const std::string name () const = 0;
1459 
1468  virtual const std::string memberid () const = 0;
1469 
1470 
1493  virtual int poll (int timeout_ms) = 0;
1494 
1501  virtual int outq_len () = 0;
1502 
1518  virtual ErrorCode metadata (bool all_topics, const Topic *only_rkt,
1519  Metadata **metadatap, int timeout_ms) = 0;
1520 
1521 
1531  virtual ErrorCode pause (std::vector<TopicPartition*> &partitions) = 0;
1532 
1533 
1543  virtual ErrorCode resume (std::vector<TopicPartition*> &partitions) = 0;
1544 
1545 
1554  virtual ErrorCode query_watermark_offsets (const std::string &topic,
1555  int32_t partition,
1556  int64_t *low, int64_t *high,
1557  int timeout_ms) = 0;
1558 
1576  virtual ErrorCode get_watermark_offsets (const std::string &topic,
1577  int32_t partition,
1578  int64_t *low, int64_t *high) = 0;
1579 
1580 
1602  virtual ErrorCode offsetsForTimes (std::vector<TopicPartition*> &offsets,
1603  int timeout_ms) = 0;
1604 
1605 
1614  virtual Queue *get_partition_queue (const TopicPartition *partition) = 0;
1615 
1632  virtual ErrorCode set_log_queue (Queue *queue) = 0;
1633 
1645  virtual void yield () = 0;
1646 
1661  virtual const std::string clusterid (int timeout_ms) = 0;
1662 
1679  virtual struct rd_kafka_s *c_ptr () = 0;
1680 
1696  virtual int32_t controllerid (int timeout_ms) = 0;
1697 
1698 
1720  virtual ErrorCode fatal_error (std::string &errstr) const = 0;
1721 
1761  virtual ErrorCode oauthbearer_set_token (const std::string &token_value,
1762  int64_t md_lifetime_ms,
1763  const std::string &md_principal_name,
1764  const std::list<std::string> &extensions,
1765  std::string &errstr) = 0;
1766 
1784  virtual ErrorCode oauthbearer_set_token_failure (const std::string &errstr) = 0;
1785 
1796  virtual void *mem_malloc (size_t size) = 0;
1797 
1811  virtual void mem_free (void *ptr) = 0;
1812 };
1813 
1814 
1833 class RD_EXPORT TopicPartition {
1834 public:
1840  static TopicPartition *create (const std::string &topic, int partition);
1841 
1848  static TopicPartition *create (const std::string &topic, int partition,
1849  int64_t offset);
1850 
1851  virtual ~TopicPartition() = 0;
1852 
1857  static void destroy (std::vector<TopicPartition*> &partitions);
1858 
1860  virtual const std::string &topic () const = 0;
1861 
1863  virtual int partition () const = 0;
1864 
1866  virtual int64_t offset () const = 0;
1867 
1869  virtual void set_offset (int64_t offset) = 0;
1870 
1872  virtual ErrorCode err () const = 0;
1873 };
1874 
1875 
1876 
1881 class RD_EXPORT Topic {
1882  public:
1889  static const int32_t PARTITION_UA;
1890 
1892  static const int64_t OFFSET_BEGINNING;
1893  static const int64_t OFFSET_END;
1894  static const int64_t OFFSET_STORED;
1895  static const int64_t OFFSET_INVALID;
1907  static Topic *create (Handle *base, const std::string &topic_str,
1908  const Conf *conf, std::string &errstr);
1909 
1910  virtual ~Topic () = 0;
1911 
1912 
1914  virtual const std::string name () const = 0;
1915 
1921  virtual bool partition_available (int32_t partition) const = 0;
1922 
1934  virtual ErrorCode offset_store (int32_t partition, int64_t offset) = 0;
1935 
1952  virtual struct rd_kafka_topic_s *c_ptr () = 0;
1953 };
1954 
1955 
1977 class RD_EXPORT MessageTimestamp {
1978 public:
1983  MSG_TIMESTAMP_LOG_APPEND_TIME
1984  };
1985 
1987  int64_t timestamp;
1988 };
1989 
1990 
2000 class RD_EXPORT Headers {
2001 public:
2002  virtual ~Headers() = 0;
2003 
2012  class Header {
2013  public:
2024  Header(const std::string &key,
2025  const void *value,
2026  size_t value_size):
2027  key_(key), err_(ERR_NO_ERROR), value_size_(value_size) {
2028  value_ = copy_value(value, value_size);
2029  }
2030 
2044  Header(const std::string &key,
2045  const void *value,
2046  size_t value_size,
2047  const RdKafka::ErrorCode err):
2048  key_(key), err_(err), value_(NULL), value_size_(value_size) {
2049  if (err == ERR_NO_ERROR)
2050  value_ = copy_value(value, value_size);
2051  }
2052 
2058  Header(const Header &other):
2059  key_(other.key_), err_(other.err_), value_size_(other.value_size_) {
2060  value_ = copy_value(other.value_, value_size_);
2061  }
2062 
2068  Header& operator=(const Header &other)
2069  {
2070  if (&other == this) {
2071  return *this;
2072  }
2073 
2074  key_ = other.key_;
2075  err_ = other.err_;
2076  value_size_ = other.value_size_;
2077 
2078  if (value_ != NULL)
2079  mem_free(value_);
2080 
2081  value_ = copy_value(other.value_, value_size_);
2082 
2083  return *this;
2084  }
2085 
2086  ~Header() {
2087  if (value_ != NULL)
2088  mem_free(value_);
2089  }
2090 
2092  std::string key() const {
2093  return key_;
2094  }
2095 
2097  const void *value() const {
2098  return value_;
2099  }
2100 
2103  const char *value_string() const {
2104  return static_cast<const char *>(value_);
2105  }
2106 
2108  size_t value_size() const {
2109  return value_size_;
2110  }
2111 
2114  return err_;
2115  }
2116 
2117  private:
2118  char *copy_value(const void *value, size_t value_size) {
2119  if (!value)
2120  return NULL;
2121 
2122  char *dest = (char *)mem_malloc(value_size + 1);
2123  memcpy(dest, (const char *)value, value_size);
2124  dest[value_size] = '\0';
2125 
2126  return dest;
2127  }
2128 
2129  std::string key_;
2130  RdKafka::ErrorCode err_;
2131  char *value_;
2132  size_t value_size_;
2133  void *operator new(size_t); /* Prevent dynamic allocation */
2134  };
2135 
2141  static Headers *create();
2142 
2151  static Headers *create(const std::vector<Header> &headers);
2152 
2162  virtual ErrorCode add(const std::string &key, const void *value,
2163  size_t value_size) = 0;
2164 
2175  virtual ErrorCode add(const std::string &key, const std::string &value) = 0;
2176 
2186  virtual ErrorCode add(const Header &header) = 0;
2187 
2195  virtual ErrorCode remove(const std::string &key) = 0;
2196 
2206  virtual std::vector<Header> get(const std::string &key) const = 0;
2207 
2218  virtual Header get_last(const std::string &key) const = 0;
2219 
2225  virtual std::vector<Header> get_all() const = 0;
2226 
2230  virtual size_t size() const = 0;
2231 };
2232 
2233 
2245 class RD_EXPORT Message {
2246  public:
2249  enum Status {
2253  MSG_STATUS_NOT_PERSISTED = 0,
2254 
2258  MSG_STATUS_POSSIBLY_PERSISTED = 1,
2259 
2263  MSG_STATUS_PERSISTED = 2,
2264  };
2265 
2273  virtual std::string errstr() const = 0;
2274 
2276  virtual ErrorCode err () const = 0;
2277 
2282  virtual Topic *topic () const = 0;
2283 
2285  virtual std::string topic_name () const = 0;
2286 
2288  virtual int32_t partition () const = 0;
2289 
2291  virtual void *payload () const = 0 ;
2292 
2294  virtual size_t len () const = 0;
2295 
2297  virtual const std::string *key () const = 0;
2298 
2300  virtual const void *key_pointer () const = 0 ;
2301 
2303  virtual size_t key_len () const = 0;
2304 
2306  virtual int64_t offset () const = 0;
2307 
2309  virtual MessageTimestamp timestamp () const = 0;
2310 
2312  virtual void *msg_opaque () const = 0;
2313 
2314  virtual ~Message () = 0;
2315 
2318  virtual int64_t latency () const = 0;
2319 
2336  virtual struct rd_kafka_message_s *c_ptr () = 0;
2337 
2341  virtual Status status () const = 0;
2342 
2347  virtual RdKafka::Headers *headers () = 0;
2348 
2355  virtual RdKafka::Headers *headers (RdKafka::ErrorCode *err) = 0;
2356 
2359  virtual int32_t broker_id () const = 0;
2360 };
2361 
2385 class RD_EXPORT Queue {
2386  public:
2390  static Queue *create (Handle *handle);
2391 
2402  virtual ErrorCode forward (Queue *dst) = 0;
2403 
2404 
2416  virtual Message *consume (int timeout_ms) = 0;
2417 
2425  virtual int poll (int timeout_ms) = 0;
2426 
2427  virtual ~Queue () = 0;
2428 
2444  virtual void io_event_enable (int fd, const void *payload, size_t size) = 0;
2445 };
2446 
2460 class RD_EXPORT ConsumerGroupMetadata {
2461 public:
2462  virtual ~ConsumerGroupMetadata () = 0;
2463 };
2464 
2482 class RD_EXPORT KafkaConsumer : public virtual Handle {
2483 public:
2495  static KafkaConsumer *create (const Conf *conf, std::string &errstr);
2496 
2497  virtual ~KafkaConsumer () = 0;
2498 
2499 
2502  virtual ErrorCode assignment (std::vector<RdKafka::TopicPartition*> &partitions) = 0;
2503 
2506  virtual ErrorCode subscription (std::vector<std::string> &topics) = 0;
2507 
2542  virtual ErrorCode subscribe (const std::vector<std::string> &topics) = 0;
2543 
2545  virtual ErrorCode unsubscribe () = 0;
2546 
2553  virtual ErrorCode assign (const std::vector<TopicPartition*> &partitions) = 0;
2554 
2558  virtual ErrorCode unassign () = 0;
2559 
2584  virtual Message *consume (int timeout_ms) = 0;
2585 
2599  virtual ErrorCode commitSync () = 0;
2600 
2606  virtual ErrorCode commitAsync () = 0;
2607 
2617  virtual ErrorCode commitSync (Message *message) = 0;
2618 
2628  virtual ErrorCode commitAsync (Message *message) = 0;
2629 
2639  virtual ErrorCode commitSync (std::vector<TopicPartition*> &offsets) = 0;
2640 
2650  virtual ErrorCode commitAsync (const std::vector<TopicPartition*> &offsets) = 0;
2651 
2662  virtual ErrorCode commitSync (OffsetCommitCb *offset_commit_cb) = 0;
2663 
2674  virtual ErrorCode commitSync (std::vector<TopicPartition*> &offsets,
2675  OffsetCommitCb *offset_commit_cb) = 0;
2676 
2677 
2678 
2679 
2688  virtual ErrorCode committed (std::vector<TopicPartition*> &partitions,
2689  int timeout_ms) = 0;
2690 
2699  virtual ErrorCode position (std::vector<TopicPartition*> &partitions) = 0;
2700 
2701 
2724  virtual ErrorCode close () = 0;
2725 
2726 
2744  virtual ErrorCode seek (const TopicPartition &partition, int timeout_ms) = 0;
2745 
2746 
2764  virtual ErrorCode offsets_store (std::vector<TopicPartition*> &offsets) = 0;
2765 
2766 
2777  virtual ConsumerGroupMetadata *groupMetadata () = 0;
2778 
2779 
2794  virtual bool assignment_lost () = 0;
2795 
2811  virtual std::string rebalance_protocol () = 0;
2812 
2813 
2829  virtual Error *incremental_assign (const std::vector<TopicPartition*> &partitions) = 0;
2830 
2831 
2847  virtual Error *incremental_unassign (const std::vector<TopicPartition*> &partitions) = 0;
2848 
2849 };
2850 
2851 
2866 class RD_EXPORT Consumer : public virtual Handle {
2867  public:
2878  static Consumer *create (const Conf *conf, std::string &errstr);
2879 
2880  virtual ~Consumer () = 0;
2881 
2882 
2902  virtual ErrorCode start (Topic *topic, int32_t partition, int64_t offset) = 0;
2903 
2910  virtual ErrorCode start (Topic *topic, int32_t partition, int64_t offset,
2911  Queue *queue) = 0;
2912 
2922  virtual ErrorCode stop (Topic *topic, int32_t partition) = 0;
2923 
2938  virtual ErrorCode seek (Topic *topic, int32_t partition, int64_t offset,
2939  int timeout_ms) = 0;
2940 
2958  virtual Message *consume (Topic *topic, int32_t partition,
2959  int timeout_ms) = 0;
2960 
2982  virtual Message *consume (Queue *queue, int timeout_ms) = 0;
2983 
3003  virtual int consume_callback (Topic *topic, int32_t partition,
3004  int timeout_ms,
3005  ConsumeCb *consume_cb,
3006  void *opaque) = 0;
3007 
3014  virtual int consume_callback (Queue *queue, int timeout_ms,
3015  RdKafka::ConsumeCb *consume_cb,
3016  void *opaque) = 0;
3017 
3027  static int64_t OffsetTail(int64_t offset);
3028 };
3029 
3043 class RD_EXPORT Producer : public virtual Handle {
3044  public:
3055  static Producer *create (const Conf *conf, std::string &errstr);
3056 
3057 
3058  virtual ~Producer () = 0;
3059 
3065  enum {
3066  RK_MSG_FREE = 0x1,
3069  RK_MSG_COPY = 0x2,
3074  RK_MSG_BLOCK = 0x4
3091  /* For backwards compatibility: */
3092 #ifndef MSG_COPY /* defined in sys/msg.h */
3093  ,
3096  MSG_FREE = RK_MSG_FREE,
3097  MSG_COPY = RK_MSG_COPY
3098 #endif
3099 
3100  };
3101 
3158  virtual ErrorCode produce (Topic *topic, int32_t partition,
3159  int msgflags,
3160  void *payload, size_t len,
3161  const std::string *key,
3162  void *msg_opaque) = 0;
3163 
3168  virtual ErrorCode produce (Topic *topic, int32_t partition,
3169  int msgflags,
3170  void *payload, size_t len,
3171  const void *key, size_t key_len,
3172  void *msg_opaque) = 0;
3173 
3180  virtual ErrorCode produce (const std::string topic_name, int32_t partition,
3181  int msgflags,
3182  void *payload, size_t len,
3183  const void *key, size_t key_len,
3184  int64_t timestamp, void *msg_opaque) = 0;
3185 
3193  virtual ErrorCode produce (const std::string topic_name, int32_t partition,
3194  int msgflags,
3195  void *payload, size_t len,
3196  const void *key, size_t key_len,
3197  int64_t timestamp,
3198  RdKafka::Headers *headers,
3199  void *msg_opaque) = 0;
3200 
3201 
3206  virtual ErrorCode produce (Topic *topic, int32_t partition,
3207  const std::vector<char> *payload,
3208  const std::vector<char> *key,
3209  void *msg_opaque) = 0;
3210 
3211 
3227  virtual ErrorCode flush (int timeout_ms) = 0;
3228 
3229 
3257  virtual ErrorCode purge (int purge_flags) = 0;
3258 
3262  enum {
3263  PURGE_QUEUE = 0x1,
3265  PURGE_INFLIGHT = 0x2,
3272  PURGE_NON_BLOCKING = 0x4 /* Don't wait for background queue
3273  * purging to finish. */
3274  };
3275 
3302  virtual Error *init_transactions (int timeout_ms) = 0;
3303 
3304 
3317  virtual Error *begin_transaction () = 0;
3318 
3365  virtual Error *send_offsets_to_transaction (
3366  const std::vector<TopicPartition*> &offsets,
3367  const ConsumerGroupMetadata *group_metadata,
3368  int timeout_ms) = 0;
3369 
3398  virtual Error *commit_transaction (int timeout_ms) = 0;
3399 
3430  virtual Error *abort_transaction (int timeout_ms) = 0;
3431 
3433 };
3434 
3449  public:
3451  virtual int32_t id() const = 0;
3452 
3454  virtual const std::string host() const = 0;
3455 
3457  virtual int port() const = 0;
3458 
3459  virtual ~BrokerMetadata() = 0;
3460 };
3461 
3462 
3463 
3468  public:
3470  typedef std::vector<int32_t> ReplicasVector;
3472  typedef std::vector<int32_t> ISRSVector;
3473 
3475  typedef ReplicasVector::const_iterator ReplicasIterator;
3477  typedef ISRSVector::const_iterator ISRSIterator;
3478 
3479 
3481  virtual int32_t id() const = 0;
3482 
3484  virtual ErrorCode err() const = 0;
3485 
3487  virtual int32_t leader() const = 0;
3488 
3490  virtual const std::vector<int32_t> *replicas() const = 0;
3491 
3495  virtual const std::vector<int32_t> *isrs() const = 0;
3496 
3497  virtual ~PartitionMetadata() = 0;
3498 };
3499 
3500 
3501 
3506  public:
3508  typedef std::vector<const PartitionMetadata*> PartitionMetadataVector;
3510  typedef PartitionMetadataVector::const_iterator PartitionMetadataIterator;
3511 
3513  virtual const std::string topic() const = 0;
3514 
3516  virtual const PartitionMetadataVector *partitions() const = 0;
3517 
3519  virtual ErrorCode err() const = 0;
3520 
3521  virtual ~TopicMetadata() = 0;
3522 };
3523 
3524 
3528 class Metadata {
3529  public:
3531  typedef std::vector<const BrokerMetadata*> BrokerMetadataVector;
3533  typedef std::vector<const TopicMetadata*> TopicMetadataVector;
3534 
3536  typedef BrokerMetadataVector::const_iterator BrokerMetadataIterator;
3538  typedef TopicMetadataVector::const_iterator TopicMetadataIterator;
3539 
3540 
3546  virtual const BrokerMetadataVector *brokers() const = 0;
3547 
3553  virtual const TopicMetadataVector *topics() const = 0;
3554 
3556  virtual int32_t orig_broker_id() const = 0;
3557 
3559  virtual const std::string orig_broker_name() const = 0;
3560 
3561  virtual ~Metadata() = 0;
3562 };
3563 
3566 }
3567 
3568 
3569 #endif /* _RDKAFKACPP_H_ */
RdKafka::ERR_INVALID_REPLICA_ASSIGNMENT
@ ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafkacpp.h:419
RdKafka::ERR__TIMED_OUT_QUEUE
@ ERR__TIMED_OUT_QUEUE
Definition: rdkafkacpp.h:274
RdKafka::ERR_INVALID_SESSION_TIMEOUT
@ ERR_INVALID_SESSION_TIMEOUT
Definition: rdkafkacpp.h:393
RdKafka::BrokerMetadata::port
virtual int port() const =0
RdKafka::PartitionMetadata
Metadata: Partition information.
Definition: rdkafkacpp.h:3467
RdKafka::ERR__AUTO_OFFSET_RESET
@ ERR__AUTO_OFFSET_RESET
Definition: rdkafkacpp.h:326
RdKafka::ERR_NOT_COORDINATOR
@ ERR_NOT_COORDINATOR
Definition: rdkafkacpp.h:371
RdKafka::ERR_INVALID_PARTITIONS
@ ERR_INVALID_PARTITIONS
Definition: rdkafkacpp.h:415
RdKafka::PartitionMetadata::replicas
virtual const std::vector< int32_t > * replicas() const =0
RdKafka::ERR_INVALID_PRINCIPAL_TYPE
@ ERR_INVALID_PRINCIPAL_TYPE
Definition: rdkafkacpp.h:480
rd_kafka_message_s
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1393
RdKafka::CERT_CA
@ CERT_CA
Definition: rdkafkacpp.h:563
RdKafka::OAuthBearerTokenRefreshCb
SASL/OAUTHBEARER token refresh callback class.
Definition: rdkafkacpp.h:736
RdKafka::ERR_DUPLICATE_RESOURCE
@ ERR_DUPLICATE_RESOURCE
Definition: rdkafkacpp.h:533
RdKafka::TopicMetadata::err
virtual ErrorCode err() const =0
RdKafka::Topic::OFFSET_END
static const int64_t OFFSET_END
Definition: rdkafkacpp.h:1893
RdKafka::BrokerMetadata::host
virtual const std::string host() const =0
RdKafka::ERR_NOT_LEADER_FOR_PARTITION
@ ERR_NOT_LEADER_FOR_PARTITION
Definition: rdkafkacpp.h:347
RdKafka::PartitionMetadata::isrs
virtual const std::vector< int32_t > * isrs() const =0
RdKafka::ERR__UNDERFLOW
@ ERR__UNDERFLOW
Definition: rdkafkacpp.h:296
RdKafka::KafkaConsumer
High-level KafkaConsumer (for brokers 0.9 and later)
Definition: rdkafkacpp.h:2482
RdKafka::ERR_INCONSISTENT_GROUP_PROTOCOL
@ ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafkacpp.h:387
RdKafka::Headers::Header::key
std::string key() const
Definition: rdkafkacpp.h:2092
RdKafka::Metadata::TopicMetadataVector
std::vector< const TopicMetadata * > TopicMetadataVector
Topics.
Definition: rdkafkacpp.h:3533
RdKafka::ERR_ELECTION_NOT_NEEDED
@ ERR_ELECTION_NOT_NEEDED
Definition: rdkafkacpp.h:515
RdKafka::ERR__QUEUE_FULL
@ ERR__QUEUE_FULL
Definition: rdkafkacpp.h:238
RdKafka::ERR_OPERATION_NOT_ATTEMPTED
@ ERR_OPERATION_NOT_ATTEMPTED
Definition: rdkafkacpp.h:456
RdKafka::ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
@ ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafkacpp.h:474
RdKafka::ERR__CONFLICT
@ ERR__CONFLICT
Definition: rdkafkacpp.h:260
rd_kafka_message_s::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:1394
RdKafka::Conf::CONF_GLOBAL
@ CONF_GLOBAL
Definition: rdkafkacpp.h:1180
RdKafka::DeliveryReportCb
Delivery Report callback class.
Definition: rdkafkacpp.h:698
RdKafka::ERR_INVALID_RECORD
@ ERR_INVALID_RECORD
Definition: rdkafkacpp.h:522
RdKafka::ERR__NOENT
@ ERR__NOENT
Definition: rdkafkacpp.h:294
RdKafka::ERR__INCONSISTENT
@ ERR__INCONSISTENT
Definition: rdkafkacpp.h:308
RdKafka::ERR_GROUP_ID_NOT_FOUND
@ ERR_GROUP_ID_NOT_FOUND
Definition: rdkafkacpp.h:484
RdKafka::ERR_ILLEGAL_SASL_STATE
@ ERR_ILLEGAL_SASL_STATE
Definition: rdkafkacpp.h:409
RdKafka::ERR_NETWORK_EXCEPTION
@ ERR_NETWORK_EXCEPTION
Definition: rdkafkacpp.h:361
RdKafka::RebalanceCb
KafkaConsumer: Rebalance callback class
Definition: rdkafkacpp.h:946
RdKafka::ERR__BAD_MSG
@ ERR__BAD_MSG
Definition: rdkafkacpp.h:205
RdKafka::Topic::OFFSET_STORED
static const int64_t OFFSET_STORED
Definition: rdkafkacpp.h:1894
RdKafka::ERR_CLUSTER_AUTHORIZATION_FAILED
@ ERR_CLUSTER_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:403
RdKafka::ERR__KEY_SERIALIZATION
@ ERR__KEY_SERIALIZATION
Definition: rdkafkacpp.h:282
RdKafka::ERR__CRIT_SYS_RESOURCE
@ ERR__CRIT_SYS_RESOURCE
Definition: rdkafkacpp.h:215
RdKafka::Queue
Queue interface.
Definition: rdkafkacpp.h:2385
RdKafka::ERR__FS
@ ERR__FS
Definition: rdkafkacpp.h:228
RdKafka::ERR_INVALID_TXN_STATE
@ ERR_INVALID_TXN_STATE
Definition: rdkafkacpp.h:437
RdKafka::ERR__UNSUPPORTED_FEATURE
@ ERR__UNSUPPORTED_FEATURE
Definition: rdkafkacpp.h:276
RdKafka::ERR_UNKNOWN_MEMBER_ID
@ ERR_UNKNOWN_MEMBER_ID
Definition: rdkafkacpp.h:391
RdKafka::ERR__PURGE_QUEUE
@ ERR__PURGE_QUEUE
Definition: rdkafkacpp.h:302
RdKafka::ERR_INVALID_GROUP_ID
@ ERR_INVALID_GROUP_ID
Definition: rdkafkacpp.h:389
RdKafka::Metadata::BrokerMetadataVector
std::vector< const BrokerMetadata * > BrokerMetadataVector
Brokers.
Definition: rdkafkacpp.h:3531
RdKafka::ERR_THROTTLING_QUOTA_EXCEEDED
@ ERR_THROTTLING_QUOTA_EXCEEDED
Definition: rdkafkacpp.h:526
RdKafka::TopicMetadata::partitions
virtual const PartitionMetadataVector * partitions() const =0
RdKafka::ERR_INVALID_TIMESTAMP
@ ERR_INVALID_TIMESTAMP
Definition: rdkafkacpp.h:405
RdKafka::ERR__FAIL
@ ERR__FAIL
Definition: rdkafkacpp.h:211
RdKafka::Metadata::brokers
virtual const BrokerMetadataVector * brokers() const =0
Broker list.
RdKafka::Event::EVENT_LOG
@ EVENT_LOG
Definition: rdkafkacpp.h:840
RdKafka::ERR__MSG_TIMED_OUT
@ ERR__MSG_TIMED_OUT
Definition: rdkafkacpp.h:219
RdKafka::TopicMetadata::topic
virtual const std::string topic() const =0
RdKafka::ERR__VALUE_SERIALIZATION
@ ERR__VALUE_SERIALIZATION
Definition: rdkafkacpp.h:284
RdKafka::ERR__SSL
@ ERR__SSL
Definition: rdkafkacpp.h:244
RdKafka::Event::Severity
Severity
EVENT_LOG severities (conforms to syslog(3) severities)
Definition: rdkafkacpp.h:845
RdKafka::ErrorCode
ErrorCode
Error codes.
Definition: rdkafkacpp.h:200
RdKafka::ERR__RESOLVE
@ ERR__RESOLVE
Definition: rdkafkacpp.h:217
RdKafka::ERR__NOT_CONFIGURED
@ ERR__NOT_CONFIGURED
Definition: rdkafkacpp.h:316
RdKafka::ERR__TRANSPORT
@ ERR__TRANSPORT
Definition: rdkafkacpp.h:213
RdKafka::ERR__ISR_INSUFF
@ ERR__ISR_INSUFF
Definition: rdkafkacpp.h:240
RdKafka::ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
@ ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafkacpp.h:381
RdKafka::ERR__ASSIGN_PARTITIONS
@ ERR__ASSIGN_PARTITIONS
Definition: rdkafkacpp.h:256
RdKafka::ERR_OFFSET_NOT_AVAILABLE
@ ERR_OFFSET_NOT_AVAILABLE
Definition: rdkafkacpp.h:502
RdKafka::ERR_NOT_ENOUGH_REPLICAS
@ ERR_NOT_ENOUGH_REPLICAS
Definition: rdkafkacpp.h:379
RdKafka::Headers::Header::operator=
Header & operator=(const Header &other)
Assignment operator.
Definition: rdkafkacpp.h:2068
RdKafka::ERR__ALL_BROKERS_DOWN
@ ERR__ALL_BROKERS_DOWN
Definition: rdkafkacpp.h:232
RdKafka::ERR_RESOURCE_NOT_FOUND
@ ERR_RESOURCE_NOT_FOUND
Definition: rdkafkacpp.h:531
RdKafka::ERR__NODE_UPDATE
@ ERR__NODE_UPDATE
Definition: rdkafkacpp.h:242
RdKafka::Headers::Header::Header
Header(const std::string &key, const void *value, size_t value_size)
Header object to encapsulate a single Header.
Definition: rdkafkacpp.h:2024
RdKafka::ERR_DELEGATION_TOKEN_EXPIRED
@ ERR_DELEGATION_TOKEN_EXPIRED
Definition: rdkafkacpp.h:478
RdKafka::ERR_UNSUPPORTED_SASL_MECHANISM
@ ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafkacpp.h:407
RdKafka::ERR_TRANSACTION_COORDINATOR_FENCED
@ ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafkacpp.h:450
RdKafka::ERR__BAD_COMPRESSION
@ ERR__BAD_COMPRESSION
Definition: rdkafkacpp.h:207
RdKafka::Event::Type
Type
Event type.
Definition: rdkafkacpp.h:837
RdKafka::Handle
Base handle, super class for specific clients.
Definition: rdkafkacpp.h:1453
RdKafka::ERR__PREV_IN_PROGRESS
@ ERR__PREV_IN_PROGRESS
Definition: rdkafkacpp.h:252
RdKafka::ERR_DELEGATION_TOKEN_OWNER_MISMATCH
@ ERR_DELEGATION_TOKEN_OWNER_MISMATCH
Definition: rdkafkacpp.h:472
RdKafka::ERR_SASL_AUTHENTICATION_FAILED
@ ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafkacpp.h:462
RdKafka::ERR__VALUE_DESERIALIZATION
@ ERR__VALUE_DESERIALIZATION
Definition: rdkafkacpp.h:288
RdKafka::ERR_INVALID_PRODUCER_ID_MAPPING
@ ERR_INVALID_PRODUCER_ID_MAPPING
Definition: rdkafkacpp.h:440
RdKafka::ERR__ASSIGNMENT_LOST
@ ERR__ASSIGNMENT_LOST
Definition: rdkafkacpp.h:322
RdKafka::ERR_PRODUCER_FENCED
@ ERR_PRODUCER_FENCED
Definition: rdkafkacpp.h:529
RdKafka::ERR__WAIT_CACHE
@ ERR__WAIT_CACHE
Definition: rdkafkacpp.h:278
RdKafka::ERR_UNKNOWN_PRODUCER_ID
@ ERR_UNKNOWN_PRODUCER_ID
Definition: rdkafkacpp.h:464
RdKafka::ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
@ ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafkacpp.h:513
RdKafka::OffsetCommitCb
Offset Commit callback class.
Definition: rdkafkacpp.h:1027
RdKafka::ERR_INVALID_COMMIT_OFFSET_SIZE
@ ERR_INVALID_COMMIT_OFFSET_SIZE
Definition: rdkafkacpp.h:397
RdKafka::Headers::Header
Header object.
Definition: rdkafkacpp.h:2012
RdKafka::MessageTimestamp::type
MessageTimestampType type
Definition: rdkafkacpp.h:1986
RdKafka::ERR_TOPIC_ALREADY_EXISTS
@ ERR_TOPIC_ALREADY_EXISTS
Definition: rdkafkacpp.h:413
RdKafka::ERR_FENCED_INSTANCE_ID
@ ERR_FENCED_INSTANCE_ID
Definition: rdkafkacpp.h:511
RdKafka::ERR_STALE_CTRL_EPOCH
@ ERR_STALE_CTRL_EPOCH
Definition: rdkafkacpp.h:357
RdKafka::ERR__OUTDATED
@ ERR__OUTDATED
Definition: rdkafkacpp.h:272
RdKafka::ERR_MSG_SIZE_TOO_LARGE
@ ERR_MSG_SIZE_TOO_LARGE
Definition: rdkafkacpp.h:355
RdKafka::ERR_COORDINATOR_NOT_AVAILABLE
@ ERR_COORDINATOR_NOT_AVAILABLE
Definition: rdkafkacpp.h:367
RdKafka::Headers::Header::err
RdKafka::ErrorCode err() const
Definition: rdkafkacpp.h:2113
RdKafka::ERR_INCONSISTENT_VOTER_SET
@ ERR_INCONSISTENT_VOTER_SET
Definition: rdkafkacpp.h:538
RdKafka::Event::EVENT_ERROR
@ EVENT_ERROR
Definition: rdkafkacpp.h:838
RdKafka::ERR_INVALID_TRANSACTION_TIMEOUT
@ ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafkacpp.h:443
RdKafka::MessageTimestamp::timestamp
int64_t timestamp
Definition: rdkafkacpp.h:1987
RdKafka::ERR_UNKNOWN_LEADER_EPOCH
@ ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafkacpp.h:496
RdKafka::PartitionMetadata::id
virtual int32_t id() const =0
RdKafka::ERR_REASSIGNMENT_IN_PROGRESS
@ ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafkacpp.h:466
RdKafka::ERR_INVALID_MSG_SIZE
@ ERR_INVALID_MSG_SIZE
Definition: rdkafkacpp.h:343
RdKafka::ERR__INVALID_ARG
@ ERR__INVALID_ARG
Definition: rdkafkacpp.h:234
RdKafka::ERR__APPLICATION
@ ERR__APPLICATION
Definition: rdkafkacpp.h:320
RdKafka::mem_malloc
RD_EXPORT void * mem_malloc(size_t size)
Allocate memory using the same allocator librdkafka uses.
RdKafka::ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
@ ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafkacpp.h:431
RdKafka::ERR_TOPIC_AUTHORIZATION_FAILED
@ ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:399
RdKafka::ERR__NOT_IMPLEMENTED
@ ERR__NOT_IMPLEMENTED
Definition: rdkafkacpp.h:266
RdKafka::ERR__INTR
@ ERR__INTR
Definition: rdkafkacpp.h:280
RdKafka::ERR_RECORD_LIST_TOO_LARGE
@ ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafkacpp.h:377
RdKafka::PartitionMetadata::ReplicasIterator
ReplicasVector::const_iterator ReplicasIterator
Replicas iterator.
Definition: rdkafkacpp.h:3475
RdKafka::ERR__END
@ ERR__END
Definition: rdkafkacpp.h:329
RdKafka::ERR_DELEGATION_TOKEN_NOT_FOUND
@ ERR_DELEGATION_TOKEN_NOT_FOUND
Definition: rdkafkacpp.h:470
RdKafka::ERR_KAFKA_STORAGE_ERROR
@ ERR_KAFKA_STORAGE_ERROR
Definition: rdkafkacpp.h:458
RdKafka::ERR__IN_PROGRESS
@ ERR__IN_PROGRESS
Definition: rdkafkacpp.h:250
RdKafka::EventCb
Event callback class.
Definition: rdkafkacpp.h:818
RdKafka::ERR__UNKNOWN_PARTITION
@ ERR__UNKNOWN_PARTITION
Definition: rdkafkacpp.h:226
RdKafka::ERR__GAPLESS_GUARANTEE
@ ERR__GAPLESS_GUARANTEE
Definition: rdkafkacpp.h:310
RdKafka::ERR__EXISTING_SUBSCRIPTION
@ ERR__EXISTING_SUBSCRIPTION
Definition: rdkafkacpp.h:254
RdKafka::TopicMetadata::PartitionMetadataVector
std::vector< const PartitionMetadata * > PartitionMetadataVector
Partitions.
Definition: rdkafkacpp.h:3508
RdKafka::ERR__TIMED_OUT
@ ERR__TIMED_OUT
Definition: rdkafkacpp.h:236
RdKafka::Conf
Configuration interface.
Definition: rdkafkacpp.h:1174
RdKafka::Headers::Header::value_size
size_t value_size() const
Definition: rdkafkacpp.h:2108
RdKafka::ERR_INVALID_MSG
@ ERR_INVALID_MSG
Definition: rdkafkacpp.h:339
RdKafka::MessageTimestamp::MSG_TIMESTAMP_NOT_AVAILABLE
@ MSG_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafkacpp.h:1981
RdKafka::ERR__STATE
@ ERR__STATE
Definition: rdkafkacpp.h:262
RdKafka::PartitionerKeyPointerCb
Variant partitioner with key pointer.
Definition: rdkafkacpp.h:789
RdKafka::ERR_INVALID_REQUEST
@ ERR_INVALID_REQUEST
Definition: rdkafkacpp.h:425
RdKafka::ERR__FENCED
@ ERR__FENCED
Definition: rdkafkacpp.h:318
RdKafka::ERR__REVOKE_PARTITIONS
@ ERR__REVOKE_PARTITIONS
Definition: rdkafkacpp.h:258
RdKafka::ERR_DUPLICATE_SEQUENCE_NUMBER
@ ERR_DUPLICATE_SEQUENCE_NUMBER
Definition: rdkafkacpp.h:433
RdKafka::CertificateEncoding
CertificateEncoding
SSL certificate encoding.
Definition: rdkafkacpp.h:571
RdKafka::ERR_UNSUPPORTED_COMPRESSION_TYPE
@ ERR_UNSUPPORTED_COMPRESSION_TYPE
Definition: rdkafkacpp.h:498
RdKafka::ConsumerGroupMetadata
ConsumerGroupMetadata holds a consumer instance's group metadata state.
Definition: rdkafkacpp.h:2460
RdKafka::ERR_GROUP_AUTHORIZATION_FAILED
@ ERR_GROUP_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:401
RdKafka::PartitionerCb
Partitioner callback class.
Definition: rdkafkacpp.h:759
RdKafka::Metadata::topics
virtual const TopicMetadataVector * topics() const =0
Topic list.
RdKafka::ERR_UNACCEPTABLE_CREDENTIAL
@ ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafkacpp.h:535
RdKafka::ERR__UNKNOWN_BROKER
@ ERR__UNKNOWN_BROKER
Definition: rdkafkacpp.h:314
RdKafka::ERR_PREFERRED_LEADER_NOT_AVAILABLE
@ ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafkacpp.h:506
RdKafka::ERR__BEGIN
@ ERR__BEGIN
Definition: rdkafkacpp.h:203
RdKafka::ERR_UNKNOWN_TOPIC_OR_PART
@ ERR_UNKNOWN_TOPIC_OR_PART
Definition: rdkafkacpp.h:341
RdKafka::ConsumeCb
Consume callback class.
Definition: rdkafkacpp.h:928
RdKafka::Conf::ConfType
ConfType
Configuration object type.
Definition: rdkafkacpp.h:1179
RdKafka::Consumer
Simple Consumer (legacy)
Definition: rdkafkacpp.h:2866
RdKafka::ERR_FETCH_SESSION_ID_NOT_FOUND
@ ERR_FETCH_SESSION_ID_NOT_FOUND
Definition: rdkafkacpp.h:486
RdKafka::ERR_REPLICA_NOT_AVAILABLE
@ ERR_REPLICA_NOT_AVAILABLE
Definition: rdkafkacpp.h:353
RdKafka::Message
Message object.
Definition: rdkafkacpp.h:2245
RdKafka::ERR_POLICY_VIOLATION
@ ERR_POLICY_VIOLATION
Definition: rdkafkacpp.h:429
RdKafka::TopicMetadata
Metadata: Topic information.
Definition: rdkafkacpp.h:3505
RdKafka::ERR_BROKER_NOT_AVAILABLE
@ ERR_BROKER_NOT_AVAILABLE
Definition: rdkafkacpp.h:351
RdKafka::ERR__DESTROY
@ ERR__DESTROY
Definition: rdkafkacpp.h:209
RdKafka::Headers::Header::Header
Header(const std::string &key, const void *value, size_t value_size, const RdKafka::ErrorCode err)
Header object to encapsulate a single Header.
Definition: rdkafkacpp.h:2044
RdKafka::SslCertificateVerifyCb
SSL broker certificate verification class.
Definition: rdkafkacpp.h:1057
RdKafka::ERR_OFFSET_METADATA_TOO_LARGE
@ ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafkacpp.h:359
RdKafka::ERR__FATAL
@ ERR__FATAL
Definition: rdkafkacpp.h:306
RdKafka::Metadata::TopicMetadataIterator
TopicMetadataVector::const_iterator TopicMetadataIterator
Topics iterator.
Definition: rdkafkacpp.h:3538
RdKafka::Error
The Error class is used as a return value from APIs to propagate an error. The error consists of an e...
Definition: rdkafkacpp.h:613
RdKafka::PartitionMetadata::err
virtual ErrorCode err() const =0
RdKafka::MessageTimestamp::MSG_TIMESTAMP_CREATE_TIME
@ MSG_TIMESTAMP_CREATE_TIME
Definition: rdkafkacpp.h:1982
RdKafka::MessageTimestamp::MessageTimestampType
MessageTimestampType
Definition: rdkafkacpp.h:1980
RdKafka::Topic::OFFSET_BEGINNING
static const int64_t OFFSET_BEGINNING
Special offsets.
Definition: rdkafkacpp.h:1892
RdKafka::Metadata::orig_broker_id
virtual int32_t orig_broker_id() const =0
Broker (id) originating this metadata.
RdKafka::Event
Event object class as passed to the EventCb callback.
Definition: rdkafkacpp.h:834
RdKafka::BrokerMetadata::id
virtual int32_t id() const =0
RdKafka::mem_free
RD_EXPORT void mem_free(void *ptr)
Free pointer returned by librdkafka.
RdKafka::ERR_UNSUPPORTED_VERSION
@ ERR_UNSUPPORTED_VERSION
Definition: rdkafkacpp.h:411
RdKafka::ERR__INVALID_TYPE
@ ERR__INVALID_TYPE
Definition: rdkafkacpp.h:298
RdKafka::ERR_REBALANCE_IN_PROGRESS
@ ERR_REBALANCE_IN_PROGRESS
Definition: rdkafkacpp.h:395
RdKafka::Topic
Topic handle.
Definition: rdkafkacpp.h:1881
RdKafka::PartitionMetadata::ISRSVector
std::vector< int32_t > ISRSVector
ISRs (In-Sync-Replicas)
Definition: rdkafkacpp.h:3472
RdKafka::Topic::PARTITION_UA
static const int32_t PARTITION_UA
Unassigned partition.
Definition: rdkafkacpp.h:1889
RdKafka::ERR_PRINCIPAL_DESERIALIZATION_FAILURE
@ ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafkacpp.h:544
RdKafka::ERR__UNKNOWN_GROUP
@ ERR__UNKNOWN_GROUP
Definition: rdkafkacpp.h:248
RdKafka::ERR__RETRY
@ ERR__RETRY
Definition: rdkafkacpp.h:300
RdKafka::Headers::Header::value
const void * value() const
Definition: rdkafkacpp.h:2097
RdKafka::ERR_NON_EMPTY_GROUP
@ ERR_NON_EMPTY_GROUP
Definition: rdkafkacpp.h:482
RdKafka::Conf::ConfResult
ConfResult
RdKafka::Conf::Set() result code.
Definition: rdkafkacpp.h:1187
RdKafka::TopicPartition
Topic+Partition.
Definition: rdkafkacpp.h:1833
RdKafka::ERR_MEMBER_ID_REQUIRED
@ ERR_MEMBER_ID_REQUIRED
Definition: rdkafkacpp.h:504
RdKafka::ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
@ ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:476
RdKafka::ERR__WAIT_COORD
@ ERR__WAIT_COORD
Definition: rdkafkacpp.h:246
RdKafka::Message::Status
Status
Message persistence status can be used by the application to find out if a produced message was persi...
Definition: rdkafkacpp.h:2249
RdKafka::ERR__READ_ONLY
@ ERR__READ_ONLY
Definition: rdkafkacpp.h:292
RdKafka::ERR__NO_OFFSET
@ ERR__NO_OFFSET
Definition: rdkafkacpp.h:270
RdKafka::MessageTimestamp
Message timestamp object.
Definition: rdkafkacpp.h:1977
RdKafka::ERR_UNKNOWN
@ ERR_UNKNOWN
Definition: rdkafkacpp.h:333
RdKafka::ERR_LISTENER_NOT_FOUND
@ ERR_LISTENER_NOT_FOUND
Definition: rdkafkacpp.h:490
RdKafka::CERT_ENC_DER
@ CERT_ENC_DER
Definition: rdkafkacpp.h:573
RdKafka::ERR_LOG_DIR_NOT_FOUND
@ ERR_LOG_DIR_NOT_FOUND
Definition: rdkafkacpp.h:460
RdKafka::ERR_NO_ERROR
@ ERR_NO_ERROR
Definition: rdkafkacpp.h:335
RdKafka::ERR__NOOP
@ ERR__NOOP
Definition: rdkafkacpp.h:324
RdKafka::ERR__UNKNOWN_TOPIC
@ ERR__UNKNOWN_TOPIC
Definition: rdkafkacpp.h:230
RdKafka::BrokerMetadata
Metadata: Broker information.
Definition: rdkafkacpp.h:3448
RdKafka::ERR_OFFSET_OUT_OF_RANGE
@ ERR_OFFSET_OUT_OF_RANGE
Definition: rdkafkacpp.h:337
RdKafka::ERR__MAX_POLL_EXCEEDED
@ ERR__MAX_POLL_EXCEEDED
Definition: rdkafkacpp.h:312
RdKafka::ERR_NOT_CONTROLLER
@ ERR_NOT_CONTROLLER
Definition: rdkafkacpp.h:423
RdKafka::PartitionMetadata::leader
virtual int32_t leader() const =0
RdKafka::CERT_ENC_PEM
@ CERT_ENC_PEM
Definition: rdkafkacpp.h:574
RdKafka::CERT_ENC_PKCS12
@ CERT_ENC_PKCS12
Definition: rdkafkacpp.h:572
RdKafka::ERR_ILLEGAL_GENERATION
@ ERR_ILLEGAL_GENERATION
Definition: rdkafkacpp.h:385
RdKafka::ERR_INVALID_PRODUCER_EPOCH
@ ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafkacpp.h:435
RdKafka::ERR_SECURITY_DISABLED
@ ERR_SECURITY_DISABLED
Definition: rdkafkacpp.h:454
RdKafka::ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
@ ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:452
RdKafka::ERR__AUTHENTICATION
@ ERR__AUTHENTICATION
Definition: rdkafkacpp.h:268
RdKafka::CERT_PRIVATE_KEY
@ CERT_PRIVATE_KEY
Definition: rdkafkacpp.h:562
RdKafka::Producer
Producer.
Definition: rdkafkacpp.h:3043
RdKafka::ERR_LEADER_NOT_AVAILABLE
@ ERR_LEADER_NOT_AVAILABLE
Definition: rdkafkacpp.h:345
RdKafka::ERR_GROUP_SUBSCRIBED_TO_TOPIC
@ ERR_GROUP_SUBSCRIBED_TO_TOPIC
Definition: rdkafkacpp.h:520
RdKafka::ERR_FEATURE_UPDATE_FAILED
@ ERR_FEATURE_UPDATE_FAILED
Definition: rdkafkacpp.h:542
RdKafka::ERR_FENCED_LEADER_EPOCH
@ ERR_FENCED_LEADER_EPOCH
Definition: rdkafkacpp.h:494
RdKafka::ERR__PARTITION_EOF
@ ERR__PARTITION_EOF
Definition: rdkafkacpp.h:224
RdKafka::Metadata
Metadata container.
Definition: rdkafkacpp.h:3528
RdKafka::CERT_PUBLIC_KEY
@ CERT_PUBLIC_KEY
Definition: rdkafkacpp.h:561
RdKafka::Metadata::BrokerMetadataIterator
BrokerMetadataVector::const_iterator BrokerMetadataIterator
Brokers iterator.
Definition: rdkafkacpp.h:3536
RdKafka::ERR_DELEGATION_TOKEN_AUTH_DISABLED
@ ERR_DELEGATION_TOKEN_AUTH_DISABLED
Definition: rdkafkacpp.h:468
RdKafka::ERR_UNSTABLE_OFFSET_COMMIT
@ ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafkacpp.h:524
RdKafka::ERR_INVALID_UPDATE_VERSION
@ ERR_INVALID_UPDATE_VERSION
Definition: rdkafkacpp.h:540
RdKafka::ERR_INVALID_FETCH_SESSION_EPOCH
@ ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafkacpp.h:488
RdKafka::ERR_TOPIC_EXCEPTION
@ ERR_TOPIC_EXCEPTION
Definition: rdkafkacpp.h:375
RdKafka::ERR__KEY_DESERIALIZATION
@ ERR__KEY_DESERIALIZATION
Definition: rdkafkacpp.h:286
RdKafka::ERR__UNKNOWN_PROTOCOL
@ ERR__UNKNOWN_PROTOCOL
Definition: rdkafkacpp.h:264
RdKafka::ERR_NO_REASSIGNMENT_IN_PROGRESS
@ ERR_NO_REASSIGNMENT_IN_PROGRESS
Definition: rdkafkacpp.h:517
RdKafka::ERR_COORDINATOR_LOAD_IN_PROGRESS
@ ERR_COORDINATOR_LOAD_IN_PROGRESS
Definition: rdkafkacpp.h:363
RdKafka::SocketCb
Portability: SocketCb callback class
Definition: rdkafkacpp.h:1110
RdKafka::ERR_TOPIC_DELETION_DISABLED
@ ERR_TOPIC_DELETION_DISABLED
Definition: rdkafkacpp.h:492
RdKafka::ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
@ ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafkacpp.h:427
RdKafka::TopicMetadata::PartitionMetadataIterator
PartitionMetadataVector::const_iterator PartitionMetadataIterator
Partitions iterator.
Definition: rdkafkacpp.h:3510
RdKafka::PartitionerKeyPointerCb::partitioner_cb
virtual int32_t partitioner_cb(const Topic *topic, const void *key, size_t key_len, int32_t partition_cnt, void *msg_opaque)=0
Variant partitioner callback that gets key as pointer and length instead of as a const std::string *.
RdKafka::ERR__PARTIAL
@ ERR__PARTIAL
Definition: rdkafkacpp.h:290
RdKafka::ERR_REQUEST_TIMED_OUT
@ ERR_REQUEST_TIMED_OUT
Definition: rdkafkacpp.h:349
RdKafka::ERR_GROUP_MAX_SIZE_REACHED
@ ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafkacpp.h:508
RdKafka::ERR_CONCURRENT_TRANSACTIONS
@ ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafkacpp.h:446
RdKafka::Headers::Header::value_string
const char * value_string() const
Definition: rdkafkacpp.h:2103
RdKafka::Headers
Headers object.
Definition: rdkafkacpp.h:2000
RdKafka::Topic::OFFSET_INVALID
static const int64_t OFFSET_INVALID
Definition: rdkafkacpp.h:1895
RdKafka::Event::EVENT_STATS
@ EVENT_STATS
Definition: rdkafkacpp.h:839
RdKafka::Headers::Header::Header
Header(const Header &other)
Copy constructor.
Definition: rdkafkacpp.h:2058
RdKafka::ERR_INVALID_REPLICATION_FACTOR
@ ERR_INVALID_REPLICATION_FACTOR
Definition: rdkafkacpp.h:417
RdKafka::ERR_STALE_BROKER_EPOCH
@ ERR_STALE_BROKER_EPOCH
Definition: rdkafkacpp.h:500
RdKafka::ERR_INVALID_CONFIG
@ ERR_INVALID_CONFIG
Definition: rdkafkacpp.h:421
RdKafka::ERR_INVALID_REQUIRED_ACKS
@ ERR_INVALID_REQUIRED_ACKS
Definition: rdkafkacpp.h:383
RdKafka::Metadata::orig_broker_name
virtual const std::string orig_broker_name() const =0
Broker (name) originating this metadata.
RdKafka::ERR__PURGE_INFLIGHT
@ ERR__PURGE_INFLIGHT
Definition: rdkafkacpp.h:304
RdKafka::PartitionMetadata::ISRSIterator
ISRSVector::const_iterator ISRSIterator
ISRs iterator.
Definition: rdkafkacpp.h:3477
RdKafka::CertificateType
CertificateType
SSL certificate types.
Definition: rdkafkacpp.h:560
RdKafka::PartitionMetadata::ReplicasVector
std::vector< int32_t > ReplicasVector
Replicas.
Definition: rdkafkacpp.h:3470
RdKafka::OpenCb
Portability: OpenCb callback class
Definition: rdkafkacpp.h:1135