librdkafka
The Apache Kafka C/C++ client library
|
Go to the documentation of this file.
30 #ifndef _RDKAFKACPP_H_
31 #define _RDKAFKACPP_H_
57 #include <sys/types.h>
64 #ifndef _SSIZE_T_DEFINED
65 #define _SSIZE_T_DEFINED
66 typedef SSIZE_T ssize_t;
70 #ifdef LIBRDKAFKA_STATICLIB
73 #ifdef LIBRDKAFKACPP_EXPORTS
74 #define RD_EXPORT __declspec(dllexport)
76 #define RD_EXPORT __declspec(dllimport)
88 struct rd_kafka_topic_s;
90 struct rd_kafka_conf_s;
91 struct rd_kafka_topic_conf_s;
115 #define RD_KAFKA_VERSION 0x020600ff
129 std::string version_str();
136 std::string get_debug_contexts();
148 int wait_destroyed(
int timeout_ms);
161 void *mem_malloc(
size_t size);
177 void mem_free(
void *ptr);
368 #define ERR_GROUP_LOAD_IN_PROGRESS ERR_COORDINATOR_LOAD_IN_PROGRESS
372 #define ERR_GROUP_COORDINATOR_NOT_AVAILABLE ERR_COORDINATOR_NOT_AVAILABLE
376 #define ERR_NOT_COORDINATOR_FOR_GROUP ERR_NOT_COORDINATOR
594 class TopicPartition;
638 virtual std::string name()
const = 0;
643 virtual std::string str()
const = 0;
649 virtual bool is_fatal()
const = 0;
654 virtual bool is_retriable()
const = 0;
667 virtual bool txn_requires_abort()
const = 0;
706 virtual void dr_cb(
Message &message) = 0;
749 virtual void oauthbearer_token_refresh_cb(
751 const std::string &oauthbearer_config) = 0;
784 virtual int32_t partitioner_cb(
const Topic *topic,
785 const std::string *key,
786 int32_t partition_cnt,
787 void *msg_opaque) = 0;
810 int32_t partition_cnt,
811 void *msg_opaque) = 0;
834 virtual void event_cb(
Event &event) = 0;
856 EVENT_SEVERITY_EMERG = 0,
857 EVENT_SEVERITY_ALERT = 1,
858 EVENT_SEVERITY_CRITICAL = 2,
859 EVENT_SEVERITY_ERROR = 3,
860 EVENT_SEVERITY_WARNING = 4,
861 EVENT_SEVERITY_NOTICE = 5,
862 EVENT_SEVERITY_INFO = 6,
863 EVENT_SEVERITY_DEBUG = 7
877 virtual Type type()
const = 0;
883 virtual ErrorCode err()
const = 0;
889 virtual Severity severity()
const = 0;
895 virtual std::string fac()
const = 0;
905 virtual std::string str()
const = 0;
911 virtual int throttle_time()
const = 0;
917 virtual std::string broker_name()
const = 0;
923 virtual int broker_id()
const = 0;
931 virtual bool fatal()
const = 0;
948 virtual void consume_cb(
Message &message,
void *opaque) = 0;
1030 std::vector<TopicPartition *> &partitions) = 0;
1058 std::vector<TopicPartition *> &offsets) = 0;
1109 virtual bool ssl_cert_verify_cb(
const std::string &broker_name,
1115 std::string &errstr) = 0;
1141 virtual int socket_cb(
int domain,
int type,
int protocol) = 0;
1165 virtual int open_cb(
const std::string &path,
int flags,
int mode) = 0;
1214 static Conf *create(ConfType type);
1233 const std::string &value,
1234 std::string &errstr) = 0;
1238 DeliveryReportCb *dr_cb,
1239 std::string &errstr) = 0;
1243 const std::string &name,
1244 OAuthBearerTokenRefreshCb *oauthbearer_token_refresh_cb,
1245 std::string &errstr) = 0;
1250 std::string &errstr) = 0;
1260 const Conf *topic_conf,
1261 std::string &errstr) = 0;
1265 PartitionerCb *partitioner_cb,
1266 std::string &errstr) = 0;
1270 PartitionerKeyPointerCb *partitioner_kp_cb,
1271 std::string &errstr) = 0;
1275 SocketCb *socket_cb,
1276 std::string &errstr) = 0;
1281 std::string &errstr) = 0;
1285 RebalanceCb *rebalance_cb,
1286 std::string &errstr) = 0;
1290 OffsetCommitCb *offset_commit_cb,
1291 std::string &errstr) = 0;
1298 SslCertificateVerifyCb *ssl_cert_verify_cb,
1299 std::string &errstr) = 0;
1345 std::string &errstr) = 0;
1360 std::string &value)
const = 0;
1371 OAuthBearerTokenRefreshCb *&oauthbearer_token_refresh_cb)
const = 0;
1387 PartitionerKeyPointerCb *&partitioner_kp_cb)
const = 0;
1411 SslCertificateVerifyCb *&ssl_cert_verify_cb)
const = 0;
1415 virtual std::list<std::string> *dump() = 0;
1419 ConsumeCb *consume_cb,
1420 std::string &errstr) = 0;
1438 virtual struct rd_kafka_conf_s *c_ptr_global() = 0;
1457 virtual struct rd_kafka_topic_conf_s *c_ptr_topic() = 0;
1472 std::string &errstr) = 0;
1498 std::string &errstr) = 0;
1519 virtual std::string name()
const = 0;
1529 virtual std::string memberid()
const = 0;
1556 virtual int poll(
int timeout_ms) = 0;
1564 virtual int outq_len() = 0;
1581 virtual ErrorCode metadata(
bool all_topics,
1582 const Topic *only_rkt,
1584 int timeout_ms) = 0;
1596 virtual ErrorCode pause(std::vector<TopicPartition *> &partitions) = 0;
1608 virtual ErrorCode resume(std::vector<TopicPartition *> &partitions) = 0;
1619 virtual ErrorCode query_watermark_offsets(
const std::string &topic,
1623 int timeout_ms) = 0;
1642 virtual ErrorCode get_watermark_offsets(
const std::string &topic,
1669 virtual ErrorCode offsetsForTimes(std::vector<TopicPartition *> &offsets,
1670 int timeout_ms) = 0;
1712 virtual void yield() = 0;
1728 virtual std::string clusterid(
int timeout_ms) = 0;
1746 virtual struct rd_kafka_s *c_ptr() = 0;
1763 virtual int32_t controllerid(
int timeout_ms) = 0;
1787 virtual ErrorCode fatal_error(std::string &errstr)
const = 0;
1828 virtual ErrorCode oauthbearer_set_token(
1829 const std::string &token_value,
1830 int64_t md_lifetime_ms,
1831 const std::string &md_principal_name,
1832 const std::list<std::string> &extensions,
1833 std::string &errstr) = 0;
1852 virtual ErrorCode oauthbearer_set_token_failure(
1853 const std::string &errstr) = 0;
1862 virtual Error *sasl_background_callbacks_enable() = 0;
1870 virtual Queue *get_sasl_queue() = 0;
1875 virtual Queue *get_background_queue() = 0;
1904 virtual void mem_free(
void *ptr) = 0;
1920 virtual Error *sasl_set_credentials(
const std::string &username,
1921 const std::string &password) = 0;
1950 static TopicPartition *create(
const std::string &topic,
int partition);
1968 static void destroy(std::vector<TopicPartition *> &partitions);
1971 virtual const std::string &topic()
const = 0;
1974 virtual int partition()
const = 0;
1977 virtual int64_t offset()
const = 0;
1980 virtual void set_offset(int64_t offset) = 0;
1986 virtual int32_t get_leader_epoch() = 0;
1989 virtual void set_leader_epoch(int32_t leader_epoch) = 0;
1992 virtual std::vector<unsigned char> get_metadata() = 0;
1995 virtual void set_metadata(std::vector<unsigned char> &metadata) = 0;
2031 const std::string &topic_str,
2033 std::string &errstr);
2035 virtual ~
Topic() = 0;
2039 virtual std::string name()
const = 0;
2046 virtual bool partition_available(int32_t partition)
const = 0;
2064 virtual ErrorCode offset_store(int32_t partition, int64_t offset) = 0;
2082 virtual struct rd_kafka_topic_s *c_ptr() = 0;
2113 MSG_TIMESTAMP_LOG_APPEND_TIME
2154 Header(
const std::string &key,
const void *value,
size_t value_size) :
2155 key_(key), err_(
ERR_NO_ERROR), value_size_(value_size) {
2156 value_ = copy_value(value, value_size);
2176 key_(key), err_(err), value_(NULL), value_size_(value_size) {
2178 value_ = copy_value(value, value_size);
2187 key_(other.key_), err_(other.err_), value_size_(other.value_size_) {
2188 value_ = copy_value(other.value_, value_size_);
2197 if (&other ==
this) {
2203 value_size_ = other.value_size_;
2208 value_ = copy_value(other.value_, value_size_);
2231 return static_cast<const char *
>(value_);
2245 char *copy_value(
const void *value,
size_t value_size) {
2249 char *dest = (
char *)
mem_malloc(value_size + 1);
2250 memcpy(dest, (
const char *)value, value_size);
2251 dest[value_size] =
'\0';
2260 void *
operator new(size_t);
2268 static Headers *create();
2278 static Headers *create(
const std::vector<Header> &headers);
2289 virtual ErrorCode add(
const std::string &key,
2291 size_t value_size) = 0;
2303 virtual ErrorCode add(
const std::string &key,
const std::string &value) = 0;
2314 virtual ErrorCode add(
const Header &header) = 0;
2323 virtual ErrorCode remove(
const std::string &key) = 0;
2334 virtual std::vector<Header> get(
const std::string &key)
const = 0;
2346 virtual Header get_last(
const std::string &key)
const = 0;
2353 virtual std::vector<Header> get_all()
const = 0;
2358 virtual size_t size()
const = 0;
2381 MSG_STATUS_NOT_PERSISTED = 0,
2386 MSG_STATUS_POSSIBLY_PERSISTED = 1,
2391 MSG_STATUS_PERSISTED = 2,
2401 virtual std::string errstr()
const = 0;
2410 virtual Topic *topic()
const = 0;
2413 virtual std::string topic_name()
const = 0;
2416 virtual int32_t partition()
const = 0;
2419 virtual void *payload()
const = 0;
2422 virtual size_t len()
const = 0;
2425 virtual const std::string *key()
const = 0;
2428 virtual const void *key_pointer()
const = 0;
2431 virtual size_t key_len()
const = 0;
2434 virtual int64_t offset()
const = 0;
2440 virtual void *msg_opaque()
const = 0;
2446 virtual int64_t latency()
const = 0;
2469 virtual Status status()
const = 0;
2487 virtual int32_t broker_id()
const = 0;
2491 virtual int32_t leader_epoch()
const = 0;
2512 virtual Error *offset_store() = 0;
2569 virtual Message *consume(
int timeout_ms) = 0;
2578 virtual int poll(
int timeout_ms) = 0;
2580 virtual ~
Queue() = 0;
2597 virtual void io_event_enable(
int fd,
const void *payload,
size_t size) = 0;
2656 std::vector<RdKafka::TopicPartition *> &partitions) = 0;
2660 virtual ErrorCode subscription(std::vector<std::string> &topics) = 0;
2696 virtual ErrorCode subscribe(
const std::vector<std::string> &topics) = 0;
2707 virtual ErrorCode assign(
const std::vector<TopicPartition *> &partitions) = 0;
2738 virtual Message *consume(
int timeout_ms) = 0;
2793 virtual ErrorCode commitSync(std::vector<TopicPartition *> &offsets) = 0;
2805 const std::vector<TopicPartition *> &offsets) = 0;
2829 virtual ErrorCode commitSync(std::vector<TopicPartition *> &offsets,
2842 virtual ErrorCode committed(std::vector<TopicPartition *> &partitions,
2843 int timeout_ms) = 0;
2853 virtual ErrorCode position(std::vector<TopicPartition *> &partitions) = 0;
2921 virtual ErrorCode offsets_store(std::vector<TopicPartition *> &offsets) = 0;
2951 virtual bool assignment_lost() = 0;
2968 virtual std::string rebalance_protocol() = 0;
2986 virtual Error *incremental_assign(
2987 const std::vector<TopicPartition *> &partitions) = 0;
3005 virtual Error *incremental_unassign(
3006 const std::vector<TopicPartition *> &partitions) = 0;
3032 virtual bool closed() = 0;
3062 static Consumer *create(
const Conf *conf, std::string &errstr);
3086 virtual ErrorCode start(
Topic *topic, int32_t partition, int64_t offset) = 0;
3127 int timeout_ms) = 0;
3146 virtual Message *consume(
Topic *topic, int32_t partition,
int timeout_ms) = 0;
3169 virtual Message *consume(
Queue *queue,
int timeout_ms) = 0;
3190 virtual int consume_callback(
Topic *topic,
3202 virtual int consume_callback(
Queue *queue,
3216 static int64_t OffsetTail(int64_t offset);
3244 static Producer *create(
const Conf *conf, std::string &errstr);
3285 MSG_FREE = RK_MSG_FREE,
3286 MSG_COPY = RK_MSG_COPY
3352 const std::string *key,
3353 void *msg_opaque) = 0;
3366 void *msg_opaque) = 0;
3374 virtual ErrorCode produce(
const std::string topic_name,
3382 void *msg_opaque) = 0;
3391 virtual ErrorCode produce(
const std::string topic_name,
3400 void *msg_opaque) = 0;
3409 const std::vector<char> *payload,
3410 const std::vector<char> *key,
3411 void *msg_opaque) = 0;
3429 virtual ErrorCode flush(
int timeout_ms) = 0;
3459 virtual ErrorCode purge(
int purge_flags) = 0;
3467 PURGE_INFLIGHT = 0x2,
3474 PURGE_NON_BLOCKING = 0x4
3504 virtual Error *init_transactions(
int timeout_ms) = 0;
3519 virtual Error *begin_transaction() = 0;
3567 virtual Error *send_offsets_to_transaction(
3568 const std::vector<TopicPartition *> &offsets,
3570 int timeout_ms) = 0;
3600 virtual Error *commit_transaction(
int timeout_ms) = 0;
3632 virtual Error *abort_transaction(
int timeout_ms) = 0;
3653 virtual int32_t
id()
const = 0;
3656 virtual std::string
host()
const = 0;
3659 virtual int port()
const = 0;
3683 virtual int32_t
id()
const = 0;
3689 virtual int32_t
leader()
const = 0;
3692 virtual const std::vector<int32_t> *
replicas()
const = 0;
3697 virtual const std::vector<int32_t> *
isrs()
const = 0;
3715 virtual std::string
topic()
const = 0;
@ ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafkacpp.h:422
@ ERR__TIMED_OUT_QUEUE
Definition: rdkafkacpp.h:275
@ ERR_INVALID_SESSION_TIMEOUT
Definition: rdkafkacpp.h:396
@ ERR__AUTO_OFFSET_RESET
Definition: rdkafkacpp.h:327
@ ERR_NOT_COORDINATOR
Definition: rdkafkacpp.h:374
@ ERR_INVALID_PARTITIONS
Definition: rdkafkacpp.h:418
@ ERR_INVALID_PRINCIPAL_TYPE
Definition: rdkafkacpp.h:483
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1461
@ CERT_CA
Definition: rdkafkacpp.h:566
SASL/OAUTHBEARER token refresh callback class.
Definition: rdkafkacpp.h:740
@ ERR_DUPLICATE_RESOURCE
Definition: rdkafkacpp.h:536
static const int64_t OFFSET_END
Definition: rdkafkacpp.h:2016
@ ERR_NOT_LEADER_FOR_PARTITION
Definition: rdkafkacpp.h:350
@ ERR__UNDERFLOW
Definition: rdkafkacpp.h:297
High-level KafkaConsumer (for brokers 0.9 and later)
Definition: rdkafkacpp.h:2635
@ ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafkacpp.h:390
@ ERR_ELECTION_NOT_NEEDED
Definition: rdkafkacpp.h:518
@ ERR__QUEUE_FULL
Definition: rdkafkacpp.h:239
@ ERR_OPERATION_NOT_ATTEMPTED
Definition: rdkafkacpp.h:459
@ ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafkacpp.h:477
@ ERR__CONFLICT
Definition: rdkafkacpp.h:261
rd_kafka_resp_err_t err
Definition: rdkafka.h:1462
@ CONF_GLOBAL
Definition: rdkafkacpp.h:1197
Delivery Report callback class.
Definition: rdkafkacpp.h:701
@ ERR_INVALID_RECORD
Definition: rdkafkacpp.h:525
@ ERR__NOENT
Definition: rdkafkacpp.h:295
@ ERR__INCONSISTENT
Definition: rdkafkacpp.h:309
@ ERR_GROUP_ID_NOT_FOUND
Definition: rdkafkacpp.h:487
@ ERR_ILLEGAL_SASL_STATE
Definition: rdkafkacpp.h:412
@ ERR_NETWORK_EXCEPTION
Definition: rdkafkacpp.h:364
KafkaConsumer: Rebalance callback class
Definition: rdkafkacpp.h:958
@ ERR__BAD_MSG
Definition: rdkafkacpp.h:206
static const int64_t OFFSET_STORED
Definition: rdkafkacpp.h:2017
@ ERR_CLUSTER_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:406
@ ERR__KEY_SERIALIZATION
Definition: rdkafkacpp.h:283
@ ERR__CRIT_SYS_RESOURCE
Definition: rdkafkacpp.h:216
Queue interface.
Definition: rdkafkacpp.h:2538
@ ERR__FS
Definition: rdkafkacpp.h:229
@ ERR_INVALID_TXN_STATE
Definition: rdkafkacpp.h:440
@ ERR__UNSUPPORTED_FEATURE
Definition: rdkafkacpp.h:277
@ ERR_UNKNOWN_MEMBER_ID
Definition: rdkafkacpp.h:394
@ ERR__PURGE_QUEUE
Definition: rdkafkacpp.h:303
@ ERR_INVALID_GROUP_ID
Definition: rdkafkacpp.h:392
@ ERR_THROTTLING_QUOTA_EXCEEDED
Definition: rdkafkacpp.h:529
@ ERR_INVALID_TIMESTAMP
Definition: rdkafkacpp.h:408
@ ERR__FAIL
Definition: rdkafkacpp.h:212
@ EVENT_LOG
Definition: rdkafkacpp.h:850
@ ERR__MSG_TIMED_OUT
Definition: rdkafkacpp.h:220
@ ERR__VALUE_SERIALIZATION
Definition: rdkafkacpp.h:285
@ ERR__SSL
Definition: rdkafkacpp.h:245
Severity
EVENT_LOG severities (conforms to syslog(3) severities)
Definition: rdkafkacpp.h:855
ErrorCode
Error codes.
Definition: rdkafkacpp.h:201
@ ERR__RESOLVE
Definition: rdkafkacpp.h:218
@ ERR__NOT_CONFIGURED
Definition: rdkafkacpp.h:317
@ ERR__TRANSPORT
Definition: rdkafkacpp.h:214
@ ERR__ISR_INSUFF
Definition: rdkafkacpp.h:241
@ ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafkacpp.h:384
@ ERR__ASSIGN_PARTITIONS
Definition: rdkafkacpp.h:257
@ ERR_OFFSET_NOT_AVAILABLE
Definition: rdkafkacpp.h:505
@ ERR_NOT_ENOUGH_REPLICAS
Definition: rdkafkacpp.h:382
@ ERR__ALL_BROKERS_DOWN
Definition: rdkafkacpp.h:233
@ ERR_RESOURCE_NOT_FOUND
Definition: rdkafkacpp.h:534
@ ERR__NODE_UPDATE
Definition: rdkafkacpp.h:243
@ ERR_DELEGATION_TOKEN_EXPIRED
Definition: rdkafkacpp.h:481
@ ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafkacpp.h:410
@ ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafkacpp.h:453
@ ERR__BAD_COMPRESSION
Definition: rdkafkacpp.h:208
Type
Event type.
Definition: rdkafkacpp.h:847
Base handle, super class for specific clients.
Definition: rdkafkacpp.h:1513
@ ERR__PREV_IN_PROGRESS
Definition: rdkafkacpp.h:253
@ ERR_DELEGATION_TOKEN_OWNER_MISMATCH
Definition: rdkafkacpp.h:475
@ ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafkacpp.h:465
@ ERR__VALUE_DESERIALIZATION
Definition: rdkafkacpp.h:289
@ ERR_INVALID_PRODUCER_ID_MAPPING
Definition: rdkafkacpp.h:443
@ ERR__ASSIGNMENT_LOST
Definition: rdkafkacpp.h:323
@ ERR_PRODUCER_FENCED
Definition: rdkafkacpp.h:532
@ ERR__WAIT_CACHE
Definition: rdkafkacpp.h:279
@ ERR_UNKNOWN_PRODUCER_ID
Definition: rdkafkacpp.h:467
@ ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafkacpp.h:516
Offset Commit callback class.
Definition: rdkafkacpp.h:1040
@ ERR_INVALID_COMMIT_OFFSET_SIZE
Definition: rdkafkacpp.h:400
MessageTimestampType type
Definition: rdkafkacpp.h:2116
@ ERR_TOPIC_ALREADY_EXISTS
Definition: rdkafkacpp.h:416
@ ERR_FENCED_INSTANCE_ID
Definition: rdkafkacpp.h:514
@ ERR_STALE_CTRL_EPOCH
Definition: rdkafkacpp.h:360
@ ERR__OUTDATED
Definition: rdkafkacpp.h:273
@ ERR_MSG_SIZE_TOO_LARGE
Definition: rdkafkacpp.h:358
@ ERR_COORDINATOR_NOT_AVAILABLE
Definition: rdkafkacpp.h:370
@ ERR_INCONSISTENT_VOTER_SET
Definition: rdkafkacpp.h:541
@ EVENT_ERROR
Definition: rdkafkacpp.h:848
@ ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafkacpp.h:446
int64_t timestamp
Definition: rdkafkacpp.h:2117
@ ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafkacpp.h:499
@ ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafkacpp.h:469
@ ERR_INVALID_MSG_SIZE
Definition: rdkafkacpp.h:346
@ ERR__INVALID_ARG
Definition: rdkafkacpp.h:235
@ ERR__APPLICATION
Definition: rdkafkacpp.h:321
RD_EXPORT void * mem_malloc(size_t size)
Allocate memory using the same allocator librdkafka uses.
@ ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafkacpp.h:434
@ ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:402
@ ERR__NOT_IMPLEMENTED
Definition: rdkafkacpp.h:267
@ ERR__INTR
Definition: rdkafkacpp.h:281
@ ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafkacpp.h:380
@ ERR__END
Definition: rdkafkacpp.h:332
@ ERR_DELEGATION_TOKEN_NOT_FOUND
Definition: rdkafkacpp.h:473
@ ERR_KAFKA_STORAGE_ERROR
Definition: rdkafkacpp.h:461
@ ERR__IN_PROGRESS
Definition: rdkafkacpp.h:251
Event callback class.
Definition: rdkafkacpp.h:827
@ ERR__UNKNOWN_PARTITION
Definition: rdkafkacpp.h:227
@ ERR__GAPLESS_GUARANTEE
Definition: rdkafkacpp.h:311
@ ERR__EXISTING_SUBSCRIPTION
Definition: rdkafkacpp.h:255
@ ERR__TIMED_OUT
Definition: rdkafkacpp.h:237
Configuration interface.
Definition: rdkafkacpp.h:1191
@ ERR_INVALID_MSG
Definition: rdkafkacpp.h:342
@ MSG_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafkacpp.h:2111
@ ERR__STATE
Definition: rdkafkacpp.h:263
Variant partitioner with key pointer.
Definition: rdkafkacpp.h:797
@ ERR_INVALID_REQUEST
Definition: rdkafkacpp.h:428
@ ERR__FENCED
Definition: rdkafkacpp.h:319
@ ERR__REVOKE_PARTITIONS
Definition: rdkafkacpp.h:259
@ ERR_DUPLICATE_SEQUENCE_NUMBER
Definition: rdkafkacpp.h:436
CertificateEncoding
SSL certificate encoding.
Definition: rdkafkacpp.h:574
@ ERR_UNSUPPORTED_COMPRESSION_TYPE
Definition: rdkafkacpp.h:501
@ ERR_GROUP_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:404
Partitioner callback class.
Definition: rdkafkacpp.h:765
@ ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafkacpp.h:538
@ ERR__UNKNOWN_BROKER
Definition: rdkafkacpp.h:315
@ ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafkacpp.h:509
@ ERR__BEGIN
Definition: rdkafkacpp.h:204
@ ERR_UNKNOWN_TOPIC_OR_PART
Definition: rdkafkacpp.h:344
Consume callback class.
Definition: rdkafkacpp.h:939
ConfType
Configuration object type.
Definition: rdkafkacpp.h:1196
Simple Consumer (legacy)
Definition: rdkafkacpp.h:3050
@ ERR_FETCH_SESSION_ID_NOT_FOUND
Definition: rdkafkacpp.h:489
@ ERR_REPLICA_NOT_AVAILABLE
Definition: rdkafkacpp.h:356
Message object.
Definition: rdkafkacpp.h:2373
@ ERR_POLICY_VIOLATION
Definition: rdkafkacpp.h:432
@ ERR_BROKER_NOT_AVAILABLE
Definition: rdkafkacpp.h:354
@ ERR__DESTROY
Definition: rdkafkacpp.h:210
SSL broker certificate verification class.
Definition: rdkafkacpp.h:1071
@ ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafkacpp.h:362
@ ERR__FATAL
Definition: rdkafkacpp.h:307
The Error class is used as a return value from APIs to propagate an error. The error consists of an e...
Definition: rdkafkacpp.h:616
@ MSG_TIMESTAMP_CREATE_TIME
Definition: rdkafkacpp.h:2112
MessageTimestampType
Definition: rdkafkacpp.h:2110
static const int64_t OFFSET_BEGINNING
Special offsets.
Definition: rdkafkacpp.h:2015
Event object class as passed to the EventCb callback.
Definition: rdkafkacpp.h:844
RD_EXPORT void mem_free(void *ptr)
Free pointer returned by librdkafka.
@ ERR_UNSUPPORTED_VERSION
Definition: rdkafkacpp.h:414
@ ERR__INVALID_TYPE
Definition: rdkafkacpp.h:299
@ ERR_REBALANCE_IN_PROGRESS
Definition: rdkafkacpp.h:398
Topic handle.
Definition: rdkafkacpp.h:2004
static const int32_t PARTITION_UA
Unassigned partition.
Definition: rdkafkacpp.h:2012
@ ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafkacpp.h:547
@ ERR__UNKNOWN_GROUP
Definition: rdkafkacpp.h:249
@ ERR__RETRY
Definition: rdkafkacpp.h:301
@ ERR_NON_EMPTY_GROUP
Definition: rdkafkacpp.h:485
ConfResult
RdKafka::Conf::Set() result code.
Definition: rdkafkacpp.h:1204
Topic+Partition.
Definition: rdkafkacpp.h:1943
@ ERR_MEMBER_ID_REQUIRED
Definition: rdkafkacpp.h:507
@ ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:479
@ ERR__WAIT_COORD
Definition: rdkafkacpp.h:247
Status
Message persistence status can be used by the application to find out if a produced message was persi...
Definition: rdkafkacpp.h:2377
@ ERR__READ_ONLY
Definition: rdkafkacpp.h:293
@ ERR__NO_OFFSET
Definition: rdkafkacpp.h:271
Message timestamp object.
Definition: rdkafkacpp.h:2107
@ ERR_UNKNOWN
Definition: rdkafkacpp.h:336
@ ERR_LISTENER_NOT_FOUND
Definition: rdkafkacpp.h:493
@ CERT_ENC_DER
Definition: rdkafkacpp.h:576
@ ERR_LOG_DIR_NOT_FOUND
Definition: rdkafkacpp.h:463
@ ERR_NO_ERROR
Definition: rdkafkacpp.h:338
@ ERR__NOOP
Definition: rdkafkacpp.h:325
@ ERR__UNKNOWN_TOPIC
Definition: rdkafkacpp.h:231
@ ERR_OFFSET_OUT_OF_RANGE
Definition: rdkafkacpp.h:340
@ ERR__MAX_POLL_EXCEEDED
Definition: rdkafkacpp.h:313
@ ERR_NOT_CONTROLLER
Definition: rdkafkacpp.h:426
@ ERR__LOG_TRUNCATION
Definition: rdkafkacpp.h:329
@ CERT_ENC_PEM
Definition: rdkafkacpp.h:577
@ CERT_ENC_PKCS12
Definition: rdkafkacpp.h:575
@ ERR_ILLEGAL_GENERATION
Definition: rdkafkacpp.h:388
@ ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafkacpp.h:438
@ ERR_SECURITY_DISABLED
Definition: rdkafkacpp.h:457
@ ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
Definition: rdkafkacpp.h:455
@ ERR__AUTHENTICATION
Definition: rdkafkacpp.h:269
@ CERT_PRIVATE_KEY
Definition: rdkafkacpp.h:565
Producer.
Definition: rdkafkacpp.h:3232
@ ERR_LEADER_NOT_AVAILABLE
Definition: rdkafkacpp.h:348
@ ERR_GROUP_SUBSCRIBED_TO_TOPIC
Definition: rdkafkacpp.h:523
@ ERR_FEATURE_UPDATE_FAILED
Definition: rdkafkacpp.h:545
@ ERR_FENCED_LEADER_EPOCH
Definition: rdkafkacpp.h:497
@ ERR__PARTITION_EOF
Definition: rdkafkacpp.h:225
@ CERT_PUBLIC_KEY
Definition: rdkafkacpp.h:564
@ ERR_DELEGATION_TOKEN_AUTH_DISABLED
Definition: rdkafkacpp.h:471
@ ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafkacpp.h:527
@ ERR_INVALID_UPDATE_VERSION
Definition: rdkafkacpp.h:543
@ ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafkacpp.h:491
@ ERR_TOPIC_EXCEPTION
Definition: rdkafkacpp.h:378
@ ERR__KEY_DESERIALIZATION
Definition: rdkafkacpp.h:287
@ ERR__UNKNOWN_PROTOCOL
Definition: rdkafkacpp.h:265
@ ERR_NO_REASSIGNMENT_IN_PROGRESS
Definition: rdkafkacpp.h:520
@ ERR_COORDINATOR_LOAD_IN_PROGRESS
Definition: rdkafkacpp.h:366
Portability: SocketCb callback class
Definition: rdkafkacpp.h:1126
@ ERR_TOPIC_DELETION_DISABLED
Definition: rdkafkacpp.h:495
@ ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafkacpp.h:430
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 *.
@ ERR__PARTIAL
Definition: rdkafkacpp.h:291
@ ERR_REQUEST_TIMED_OUT
Definition: rdkafkacpp.h:352
@ ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafkacpp.h:511
@ ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafkacpp.h:449
static const int64_t OFFSET_INVALID
Definition: rdkafkacpp.h:2018
@ EVENT_STATS
Definition: rdkafkacpp.h:849
@ ERR_INVALID_REPLICATION_FACTOR
Definition: rdkafkacpp.h:420
@ ERR_STALE_BROKER_EPOCH
Definition: rdkafkacpp.h:503
@ ERR_INVALID_CONFIG
Definition: rdkafkacpp.h:424
@ ERR_INVALID_REQUIRED_ACKS
Definition: rdkafkacpp.h:386
@ ERR__PURGE_INFLIGHT
Definition: rdkafkacpp.h:305
CertificateType
SSL certificate types.
Definition: rdkafkacpp.h:563
Portability: OpenCb callback class
Definition: rdkafkacpp.h:1152