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 #pragma once
30 
49 #include <string>
50 #include <list>
51 #include <vector>
52 #include <stdint.h>
53 
54 
55 #ifdef _MSC_VER
56 #undef RD_EXPORT
57 #ifdef LIBRDKAFKA_STATICLIB
58 #define RD_EXPORT
59 #else
60 #ifdef LIBRDKAFKACPP_EXPORTS
61 #define RD_EXPORT __declspec(dllexport)
62 #else
63 #define RD_EXPORT __declspec(dllimport)
64 #endif
65 #endif
66 #else
67 #define RD_EXPORT
68 #endif
69 
72 namespace RdKafka {
73 
74 
94 #define RD_KAFKA_VERSION 0x000905ff
95 
101 RD_EXPORT
102 int version ();
103 
107 RD_EXPORT
108 std::string version_str();
109 
114 RD_EXPORT
115 std::string get_debug_contexts();
116 
126 RD_EXPORT
127 int wait_destroyed(int timeout_ms);
128 
129 
152 enum ErrorCode {
153  /* Internal errors to rdkafka: */
155  ERR__BEGIN = -200,
157  ERR__BAD_MSG = -199,
161  ERR__DESTROY = -197,
163  ERR__FAIL = -196,
169  ERR__RESOLVE = -193,
178  ERR__FS = -189,
194  ERR__SSL = -181,
212  ERR__STATE = -172,
230  ERR__END = -100,
231 
232  /* Kafka broker errors: */
299 };
300 
301 
305 RD_EXPORT
306 std::string err2str(RdKafka::ErrorCode err);
307 
308 
314 /* Forward declarations */
315 class Producer;
316 class Message;
317 class Queue;
318 class Event;
319 class Topic;
320 class TopicPartition;
321 class Metadata;
322 class KafkaConsumer;
354 class RD_EXPORT DeliveryReportCb {
355  public:
359  virtual void dr_cb (Message &message) = 0;
360 
361  virtual ~DeliveryReportCb() { }
362 };
363 
364 
372 class RD_EXPORT PartitionerCb {
373  public:
390  virtual int32_t partitioner_cb (const Topic *topic,
391  const std::string *key,
392  int32_t partition_cnt,
393  void *msg_opaque) = 0;
394 
395  virtual ~PartitionerCb() { }
396 };
397 
403  public:
412  virtual int32_t partitioner_cb (const Topic *topic,
413  const void *key,
414  size_t key_len,
415  int32_t partition_cnt,
416  void *msg_opaque) = 0;
417 
418  virtual ~PartitionerKeyPointerCb() { }
419 };
420 
421 
422 
431 class RD_EXPORT EventCb {
432  public:
438  virtual void event_cb (Event &event) = 0;
439 
440  virtual ~EventCb() { }
441 };
442 
443 
447 class RD_EXPORT Event {
448  public:
450  enum Type {
454  EVENT_THROTTLE
455  };
456 
458  enum Severity {
459  EVENT_SEVERITY_EMERG = 0,
460  EVENT_SEVERITY_ALERT = 1,
461  EVENT_SEVERITY_CRITICAL = 2,
462  EVENT_SEVERITY_ERROR = 3,
463  EVENT_SEVERITY_WARNING = 4,
464  EVENT_SEVERITY_NOTICE = 5,
465  EVENT_SEVERITY_INFO = 6,
466  EVENT_SEVERITY_DEBUG = 7
467  };
468 
469  virtual ~Event () { }
470 
471  /*
472  * Event Accessor methods
473  */
474 
479  virtual Type type () const = 0;
480 
485  virtual ErrorCode err () const = 0;
486 
491  virtual Severity severity () const = 0;
492 
497  virtual std::string fac () const = 0;
498 
507  virtual std::string str () const = 0;
508 
513  virtual int throttle_time () const = 0;
514 
519  virtual std::string broker_name () const = 0;
520 
525  virtual int broker_id () const = 0;
526 };
527 
528 
529 
533 class RD_EXPORT ConsumeCb {
534  public:
542  virtual void consume_cb (Message &message, void *opaque) = 0;
543 
544  virtual ~ConsumeCb() { }
545 };
546 
547 
551 class RD_EXPORT RebalanceCb {
552 public:
602  virtual void rebalance_cb (RdKafka::KafkaConsumer *consumer,
603  RdKafka::ErrorCode err,
604  std::vector<TopicPartition*>&partitions) = 0;
605 
606  virtual ~RebalanceCb() { }
607 };
608 
609 
613 class RD_EXPORT OffsetCommitCb {
614 public:
630  virtual void offset_commit_cb(RdKafka::ErrorCode err,
631  std::vector<TopicPartition*>&offsets) = 0;
632 
633  virtual ~OffsetCommitCb() { }
634 };
635 
636 
637 
642 class RD_EXPORT SocketCb {
643  public:
657  virtual int socket_cb (int domain, int type, int protocol) = 0;
658 
659  virtual ~SocketCb() { }
660 };
661 
662 
667 class RD_EXPORT OpenCb {
668  public:
680  virtual int open_cb (const std::string &path, int flags, int mode) = 0;
681 
682  virtual ~OpenCb() { }
683 };
684 
685 
706 class RD_EXPORT Conf {
707  public:
711  enum ConfType {
713  CONF_TOPIC
714  };
715 
719  enum ConfResult {
720  CONF_UNKNOWN = -2,
721  CONF_INVALID = -1,
722  CONF_OK = 0
723  };
724 
725 
729  static Conf *create (ConfType type);
730 
731  virtual ~Conf () { }
732 
746  virtual Conf::ConfResult set (const std::string &name,
747  const std::string &value,
748  std::string &errstr) = 0;
749 
751  virtual Conf::ConfResult set (const std::string &name, ConsumeCb *consume_cb,
752  std::string &errstr) = 0;
753 
755  virtual Conf::ConfResult set (const std::string &name,
756  DeliveryReportCb *dr_cb,
757  std::string &errstr) = 0;
758 
760  virtual Conf::ConfResult set (const std::string &name,
761  EventCb *event_cb,
762  std::string &errstr) = 0;
763 
771  virtual Conf::ConfResult set (const std::string &name,
772  const Conf *topic_conf,
773  std::string &errstr) = 0;
774 
776  virtual Conf::ConfResult set (const std::string &name,
777  PartitionerCb *partitioner_cb,
778  std::string &errstr) = 0;
779 
781  virtual Conf::ConfResult set (const std::string &name,
782  PartitionerKeyPointerCb *partitioner_kp_cb,
783  std::string &errstr) = 0;
784 
786  virtual Conf::ConfResult set (const std::string &name, SocketCb *socket_cb,
787  std::string &errstr) = 0;
788 
790  virtual Conf::ConfResult set (const std::string &name, OpenCb *open_cb,
791  std::string &errstr) = 0;
792 
794  virtual Conf::ConfResult set (const std::string &name,
795  RebalanceCb *rebalance_cb,
796  std::string &errstr) = 0;
797 
799  virtual Conf::ConfResult set (const std::string &name,
800  OffsetCommitCb *offset_commit_cb,
801  std::string &errstr) = 0;
802 
814  virtual Conf::ConfResult get(const std::string &name,
815  std::string &value) const = 0;
816 
820  virtual Conf::ConfResult get(DeliveryReportCb *&dr_cb) const = 0;
821 
825  virtual Conf::ConfResult get(EventCb *&event_cb) const = 0;
826 
830  virtual Conf::ConfResult get(PartitionerCb *&partitioner_cb) const = 0;
831 
835  virtual Conf::ConfResult get(PartitionerKeyPointerCb *&partitioner_kp_cb) const = 0;
836 
840  virtual Conf::ConfResult get(SocketCb *&socket_cb) const = 0;
841 
845  virtual Conf::ConfResult get(OpenCb *&open_cb) const = 0;
846 
850  virtual Conf::ConfResult get(RebalanceCb *&rebalance_cb) const = 0;
851 
855  virtual Conf::ConfResult get(OffsetCommitCb *&offset_commit_cb) const = 0;
856 
859  virtual std::list<std::string> *dump () = 0;
860 };
861 
874 class RD_EXPORT Handle {
875  public:
876  virtual ~Handle() { }
877 
879  virtual const std::string name () const = 0;
880 
889  virtual const std::string memberid () const = 0;
890 
891 
914  virtual int poll (int timeout_ms) = 0;
915 
922  virtual int outq_len () = 0;
923 
939  virtual ErrorCode metadata (bool all_topics, const Topic *only_rkt,
940  Metadata **metadatap, int timeout_ms) = 0;
941 
942 
952  virtual ErrorCode pause (std::vector<TopicPartition*> &partitions) = 0;
953 
954 
964  virtual ErrorCode resume (std::vector<TopicPartition*> &partitions) = 0;
965 
966 
975  virtual ErrorCode query_watermark_offsets (const std::string &topic,
976  int32_t partition,
977  int64_t *low, int64_t *high,
978  int timeout_ms) = 0;
979 
997  virtual ErrorCode get_watermark_offsets (const std::string &topic,
998  int32_t partition,
999  int64_t *low, int64_t *high) = 0;
1000 
1001 
1021  virtual ErrorCode offsetsForTimes (std::vector<TopicPartition*> &offsets,
1022  int timeout_ms) = 0;
1023 
1024 
1033  virtual Queue *get_partition_queue (const TopicPartition *partition) = 0;
1034 
1051  virtual ErrorCode set_log_queue (Queue *queue) = 0;
1052 };
1053 
1054 
1073 class RD_EXPORT TopicPartition {
1074 public:
1081  static TopicPartition *create (const std::string &topic, int partition);
1082  static TopicPartition *create (const std::string &topic, int partition,
1083  int64_t offset);
1084 
1085  virtual ~TopicPartition() = 0;
1086 
1091  static void destroy (std::vector<TopicPartition*> &partitions);
1092 
1094  virtual const std::string &topic () const = 0;
1095 
1097  virtual int partition () const = 0;
1098 
1100  virtual int64_t offset () const = 0;
1101 
1103  virtual void set_offset (int64_t offset) = 0;
1104 
1106  virtual ErrorCode err () const = 0;
1107 };
1108 
1109 
1110 
1115 class RD_EXPORT Topic {
1116  public:
1123  static const int32_t PARTITION_UA;
1124 
1126  static const int64_t OFFSET_BEGINNING;
1127  static const int64_t OFFSET_END;
1128  static const int64_t OFFSET_STORED;
1129  static const int64_t OFFSET_INVALID;
1141  static Topic *create (Handle *base, const std::string &topic_str,
1142  Conf *conf, std::string &errstr);
1143 
1144  virtual ~Topic () = 0;
1145 
1146 
1148  virtual const std::string name () const = 0;
1149 
1155  virtual bool partition_available (int32_t partition) const = 0;
1156 
1168  virtual ErrorCode offset_store (int32_t partition, int64_t offset) = 0;
1169 };
1170 
1171 
1193 class RD_EXPORT MessageTimestamp {
1194 public:
1198  MSG_TIMESTAMP_LOG_APPEND_TIME
1199  };
1200 
1202  int64_t timestamp;
1203 };
1204 
1205 
1206 
1218 class RD_EXPORT Message {
1219  public:
1227  virtual std::string errstr() const = 0;
1228 
1230  virtual ErrorCode err () const = 0;
1231 
1236  virtual Topic *topic () const = 0;
1237 
1239  virtual std::string topic_name () const = 0;
1240 
1242  virtual int32_t partition () const = 0;
1243 
1245  virtual void *payload () const = 0 ;
1246 
1248  virtual size_t len () const = 0;
1249 
1251  virtual const std::string *key () const = 0;
1252 
1254  virtual const void *key_pointer () const = 0 ;
1255 
1257  virtual size_t key_len () const = 0;
1258 
1260  virtual int64_t offset () const = 0;
1261 
1263  virtual MessageTimestamp timestamp () const = 0;
1264 
1266  virtual void *msg_opaque () const = 0;
1267 
1268  virtual ~Message () = 0;
1269 };
1270 
1294 class RD_EXPORT Queue {
1295  public:
1299  static Queue *create (Handle *handle);
1300 
1311  virtual ErrorCode forward (Queue *dst) = 0;
1312 
1313 
1325  virtual Message *consume (int timeout_ms) = 0;
1326 
1334  virtual int poll (int timeout_ms) = 0;
1335 
1336  virtual ~Queue () = 0;
1337 };
1338 
1357 class RD_EXPORT KafkaConsumer : public virtual Handle {
1358 public:
1370  static KafkaConsumer *create (Conf *conf, std::string &errstr);
1371 
1372  virtual ~KafkaConsumer () = 0;
1373 
1374 
1377  virtual ErrorCode assignment (std::vector<RdKafka::TopicPartition*> &partitions) = 0;
1378 
1381  virtual ErrorCode subscription (std::vector<std::string> &topics) = 0;
1382 
1407  virtual ErrorCode subscribe (const std::vector<std::string> &topics) = 0;
1408 
1410  virtual ErrorCode unsubscribe () = 0;
1411 
1418  virtual ErrorCode assign (const std::vector<TopicPartition*> &partitions) = 0;
1419 
1423  virtual ErrorCode unassign () = 0;
1424 
1449  virtual Message *consume (int timeout_ms) = 0;
1450 
1464  virtual ErrorCode commitSync () = 0;
1465 
1471  virtual ErrorCode commitAsync () = 0;
1472 
1480  virtual ErrorCode commitSync (Message *message) = 0;
1481 
1489  virtual ErrorCode commitAsync (Message *message) = 0;
1490 
1496  virtual ErrorCode commitSync (std::vector<TopicPartition*> &offsets) = 0;
1497 
1503  virtual ErrorCode commitAsync (const std::vector<TopicPartition*> &offsets) = 0;
1504 
1515  virtual ErrorCode commitSync (OffsetCommitCb *offset_commit_cb) = 0;
1516 
1527  virtual ErrorCode commitSync (std::vector<TopicPartition*> &offsets,
1528  OffsetCommitCb *offset_commit_cb) = 0;
1529 
1530 
1531 
1532 
1541  virtual ErrorCode committed (std::vector<TopicPartition*> &partitions,
1542  int timeout_ms) = 0;
1543 
1552  virtual ErrorCode position (std::vector<TopicPartition*> &partitions) = 0;
1553 
1554 
1577  virtual ErrorCode close () = 0;
1578 };
1579 
1580 
1595 class RD_EXPORT Consumer : public virtual Handle {
1596  public:
1607  static Consumer *create (Conf *conf, std::string &errstr);
1608 
1609  virtual ~Consumer () = 0;
1610 
1611 
1631  virtual ErrorCode start (Topic *topic, int32_t partition, int64_t offset) = 0;
1632 
1639  virtual ErrorCode start (Topic *topic, int32_t partition, int64_t offset,
1640  Queue *queue) = 0;
1641 
1651  virtual ErrorCode stop (Topic *topic, int32_t partition) = 0;
1652 
1667  virtual ErrorCode seek (Topic *topic, int32_t partition, int64_t offset,
1668  int timeout_ms) = 0;
1669 
1687  virtual Message *consume (Topic *topic, int32_t partition,
1688  int timeout_ms) = 0;
1689 
1711  virtual Message *consume (Queue *queue, int timeout_ms) = 0;
1712 
1732  virtual int consume_callback (Topic *topic, int32_t partition,
1733  int timeout_ms,
1734  ConsumeCb *consume_cb,
1735  void *opaque) = 0;
1736 
1743  virtual int consume_callback (Queue *queue, int timeout_ms,
1744  RdKafka::ConsumeCb *consume_cb,
1745  void *opaque) = 0;
1746 
1756  static int64_t OffsetTail(int64_t offset);
1757 };
1758 
1772 class RD_EXPORT Producer : public virtual Handle {
1773  public:
1784  static Producer *create (Conf *conf, std::string &errstr);
1785 
1786 
1787  virtual ~Producer () = 0;
1788 
1794  enum {
1795  RK_MSG_FREE = 0x1,
1797  RK_MSG_COPY = 0x2,
1801  RK_MSG_BLOCK = 0x4
1818  /* For backwards compatibility: */
1819 #ifndef MSG_COPY /* defined in sys/msg.h */
1820  ,
1823  MSG_FREE = RK_MSG_FREE,
1824  MSG_COPY = RK_MSG_COPY
1825 #endif
1826 
1827  };
1828 
1883  virtual ErrorCode produce (Topic *topic, int32_t partition,
1884  int msgflags,
1885  void *payload, size_t len,
1886  const std::string *key,
1887  void *msg_opaque) = 0;
1888 
1893  virtual ErrorCode produce (Topic *topic, int32_t partition,
1894  int msgflags,
1895  void *payload, size_t len,
1896  const void *key, size_t key_len,
1897  void *msg_opaque) = 0;
1898 
1905  virtual ErrorCode produce (const std::string topic_name, int32_t partition,
1906  int msgflags,
1907  void *payload, size_t len,
1908  const void *key, size_t key_len,
1909  int64_t timestamp,
1910  void *msg_opaque) = 0;
1911 
1912 
1917  virtual ErrorCode produce (Topic *topic, int32_t partition,
1918  const std::vector<char> *payload,
1919  const std::vector<char> *key,
1920  void *msg_opaque) = 0;
1921 
1922 
1934  virtual ErrorCode flush (int timeout_ms) = 0;
1935 };
1936 
1951  public:
1953  virtual int32_t id() const = 0;
1954 
1956  virtual const std::string host() const = 0;
1957 
1959  virtual int port() const = 0;
1960 
1961  virtual ~BrokerMetadata() = 0;
1962 };
1963 
1964 
1965 
1970  public:
1972  typedef std::vector<int32_t> ReplicasVector;
1974  typedef std::vector<int32_t> ISRSVector;
1975 
1977  typedef ReplicasVector::const_iterator ReplicasIterator;
1979  typedef ISRSVector::const_iterator ISRSIterator;
1980 
1981 
1983  virtual int32_t id() const = 0;
1984 
1986  virtual ErrorCode err() const = 0;
1987 
1989  virtual int32_t leader() const = 0;
1990 
1992  virtual const std::vector<int32_t> *replicas() const = 0;
1993 
1997  virtual const std::vector<int32_t> *isrs() const = 0;
1998 
1999  virtual ~PartitionMetadata() = 0;
2000 };
2001 
2002 
2003 
2008  public:
2010  typedef std::vector<const PartitionMetadata*> PartitionMetadataVector;
2012  typedef PartitionMetadataVector::const_iterator PartitionMetadataIterator;
2013 
2015  virtual const std::string topic() const = 0;
2016 
2018  virtual const PartitionMetadataVector *partitions() const = 0;
2019 
2021  virtual ErrorCode err() const = 0;
2022 
2023  virtual ~TopicMetadata() = 0;
2024 };
2025 
2026 
2030 class Metadata {
2031  public:
2033  typedef std::vector<const BrokerMetadata*> BrokerMetadataVector;
2035  typedef std::vector<const TopicMetadata*> TopicMetadataVector;
2036 
2038  typedef BrokerMetadataVector::const_iterator BrokerMetadataIterator;
2040  typedef TopicMetadataVector::const_iterator TopicMetadataIterator;
2041 
2042 
2044  virtual const BrokerMetadataVector *brokers() const = 0;
2045 
2047  virtual const TopicMetadataVector *topics() const = 0;
2048 
2050  virtual int32_t orig_broker_id() const = 0;
2051 
2053  virtual const std::string orig_broker_name() const = 0;
2054 
2055  virtual ~Metadata() = 0;
2056 };
2057 
2060 }
2061 
Definition: rdkafkacpp.h:178
Definition: rdkafkacpp.h:282
ISRSVector::const_iterator ISRSIterator
ISRs iterator.
Definition: rdkafkacpp.h:1979
Definition: rdkafkacpp.h:171
Definition: rdkafkacpp.h:284
Definition: rdkafkacpp.h:224
Definition: rdkafkacpp.h:226
High-level KafkaConsumer (for brokers 0.9 and later)
Definition: rdkafkacpp.h:1357
static const int64_t OFFSET_BEGINNING
Special offsets.
Definition: rdkafkacpp.h:1126
Definition: rdkafkacpp.h:194
ConfType
Configuration object type.
Definition: rdkafkacpp.h:711
Partitioner callback class.
Definition: rdkafkacpp.h:372
Definition: rdkafkacpp.h:262
Definition: rdkafkacpp.h:248
Definition: rdkafkacpp.h:288
Type
Event type.
Definition: rdkafkacpp.h:450
Definition: rdkafkacpp.h:165
Definition: rdkafkacpp.h:222
Definition: rdkafkacpp.h:274
Definition: rdkafkacpp.h:169
Definition: rdkafkacpp.h:206
Definition: rdkafkacpp.h:286
Definition: rdkafkacpp.h:167
std::vector< const PartitionMetadata * > PartitionMetadataVector
Partitions.
Definition: rdkafkacpp.h:2010
static const int32_t PARTITION_UA
Unassigned partition.
Definition: rdkafkacpp.h:1123
Definition: rdkafkacpp.h:210
Message object.
Definition: rdkafkacpp.h:1218
Definition: rdkafkacpp.h:188
Definition: rdkafkacpp.h:163
Severity
EVENT_LOG severities (conforms to syslog(3) severities)
Definition: rdkafkacpp.h:458
Event callback class.
Definition: rdkafkacpp.h:431
Definition: rdkafkacpp.h:272
ReplicasVector::const_iterator ReplicasIterator
Replicas iterator.
Definition: rdkafkacpp.h:1977
Definition: rdkafkacpp.h:298
Definition: rdkafkacpp.h:192
Definition: rdkafkacpp.h:216
Definition: rdkafkacpp.h:200
KafkaConsunmer: Rebalance callback class
Definition: rdkafkacpp.h:551
std::vector< int32_t > ReplicasVector
Replicas.
Definition: rdkafkacpp.h:1972
Definition: rdkafkacpp.h:157
Definition: rdkafkacpp.h:182
Definition: rdkafkacpp.h:184
Definition: rdkafkacpp.h:202
Definition: rdkafkacpp.h:190
int64_t timestamp
Definition: rdkafkacpp.h:1202
Definition: rdkafkacpp.h:292
Variant partitioner with key pointer.
Definition: rdkafkacpp.h:402
Definition: rdkafkacpp.h:258
BrokerMetadataVector::const_iterator BrokerMetadataIterator
Brokers iterator.
Definition: rdkafkacpp.h:2038
Definition: rdkafkacpp.h:264
Definition: rdkafkacpp.h:159
Definition: rdkafkacpp.h:256
Definition: rdkafkacpp.h:453
Definition: rdkafkacpp.h:176
Definition: rdkafkacpp.h:208
Definition: rdkafkacpp.h:228
static const int64_t OFFSET_INVALID
Definition: rdkafkacpp.h:1129
Definition: rdkafkacpp.h:252
Topic handle.
Definition: rdkafkacpp.h:1115
Definition: rdkafkacpp.h:161
Definition: rdkafkacpp.h:244
Metadata: Partition information.
Definition: rdkafkacpp.h:1969
Definition: rdkafkacpp.h:294
Producer.
Definition: rdkafkacpp.h:1772
Definition: rdkafkacpp.h:155
Definition: rdkafkacpp.h:268
Metadata: Topic information.
Definition: rdkafkacpp.h:2007
Definition: rdkafkacpp.h:712
Definition: rdkafkacpp.h:451
Definition: rdkafkacpp.h:204
Definition: rdkafkacpp.h:212
std::vector< const BrokerMetadata * > BrokerMetadataVector
Brokers.
Definition: rdkafkacpp.h:2033
ConfResult
RdKafka::Conf::Set() result code.
Definition: rdkafkacpp.h:719
Definition: rdkafkacpp.h:72
Queue interface.
Definition: rdkafkacpp.h:1294
MessageTimestampType type
Definition: rdkafkacpp.h:1201
Message timestamp object.
Definition: rdkafkacpp.h:1193
std::vector< int32_t > ISRSVector
ISRs (In-Sync-Replicas)
Definition: rdkafkacpp.h:1974
Definition: rdkafkacpp.h:296
Definition: rdkafkacpp.h:238
Definition: rdkafkacpp.h:230
Definition: rdkafkacpp.h:180
Definition: rdkafkacpp.h:242
static const int64_t OFFSET_END
Definition: rdkafkacpp.h:1127
Definition: rdkafkacpp.h:198
Definition: rdkafkacpp.h:186
PartitionMetadataVector::const_iterator PartitionMetadataIterator
Partitions iterator.
Definition: rdkafkacpp.h:2012
Portability: OpenCb callback class
Definition: rdkafkacpp.h:667
Definition: rdkafkacpp.h:196
Definition: rdkafkacpp.h:240
Definition: rdkafkacpp.h:234
Definition: rdkafkacpp.h:260
Metadata: Broker information.
Definition: rdkafkacpp.h:1950
Definition: rdkafkacpp.h:452
Definition: rdkafkacpp.h:214
static const int64_t OFFSET_STORED
Definition: rdkafkacpp.h:1128
Definition: rdkafkacpp.h:218
Definition: rdkafkacpp.h:290
Configuration interface.
Definition: rdkafkacpp.h:706
Definition: rdkafkacpp.h:246
Definition: rdkafkacpp.h:254
Definition: rdkafkacpp.h:174
Offset Commit callback class.
Definition: rdkafkacpp.h:613
Base handle, super class for specific clients.
Definition: rdkafkacpp.h:874
Definition: rdkafkacpp.h:220
RD_EXPORT std::string err2str(RdKafka::ErrorCode err)
Returns a human readable representation of a kafka error.
std::vector< const TopicMetadata * > TopicMetadataVector
Topics.
Definition: rdkafkacpp.h:2035
Definition: rdkafkacpp.h:280
Topic+Partition.
Definition: rdkafkacpp.h:1073
Definition: rdkafkacpp.h:270
Consume callback class.
Definition: rdkafkacpp.h:533
Simple Consumer (legacy)
Definition: rdkafkacpp.h:1595
MessageTimestampType
Definition: rdkafkacpp.h:1195
TopicMetadataVector::const_iterator TopicMetadataIterator
Topics iterator.
Definition: rdkafkacpp.h:2040
Definition: rdkafkacpp.h:236
Event object class as passed to the EventCb callback.
Definition: rdkafkacpp.h:447
Metadata container.
Definition: rdkafkacpp.h:2030
ErrorCode
Error codes.
Definition: rdkafkacpp.h:152
Definition: rdkafkacpp.h:278
Definition: rdkafkacpp.h:250
Portability: SocketCb callback class
Definition: rdkafkacpp.h:642
Delivery Report callback class.
Definition: rdkafkacpp.h:354