librdkafka
The Apache Kafka C/C++ client library
rdkafka.h
Go to the documentation of this file.
1 /*
2  * librdkafka - Apache Kafka C library
3  *
4  * Copyright (c) 2012-2022 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 
43 /* @cond NO_DOC */
44 #ifndef _RDKAFKA_H_
45 #define _RDKAFKA_H_
46 
47 #include <stdio.h>
48 #include <inttypes.h>
49 #include <sys/types.h>
50 
51 #ifdef __cplusplus
52 extern "C" {
53 #if 0
54 } /* Restore indent */
55 #endif
56 #endif
57 
58 #ifdef _WIN32
59 #include <basetsd.h>
60 #ifndef WIN32_MEAN_AND_LEAN
61 #define WIN32_MEAN_AND_LEAN
62 #endif
63 #include <winsock2.h> /* for sockaddr, .. */
64 #ifndef _SSIZE_T_DEFINED
65 #define _SSIZE_T_DEFINED
66 typedef SSIZE_T ssize_t;
67 #endif
68 #define RD_UNUSED
69 #define RD_INLINE __inline
70 #define RD_DEPRECATED __declspec(deprecated)
71 #define RD_FORMAT(...)
72 #undef RD_EXPORT
73 #ifdef LIBRDKAFKA_STATICLIB
74 #define RD_EXPORT
75 #else
76 #ifdef LIBRDKAFKA_EXPORTS
77 #define RD_EXPORT __declspec(dllexport)
78 #else
79 #define RD_EXPORT __declspec(dllimport)
80 #endif
81 #ifndef LIBRDKAFKA_TYPECHECKS
82 #define LIBRDKAFKA_TYPECHECKS 0
83 #endif
84 #endif
85 
86 #else
87 #include <sys/socket.h> /* for sockaddr, .. */
88 
89 #define RD_UNUSED __attribute__((unused))
90 #define RD_INLINE inline
91 #define RD_EXPORT
92 #define RD_DEPRECATED __attribute__((deprecated))
93 
94 #if defined(__clang__) || defined(__GNUC__) || defined(__GNUG__)
95 #define RD_HAS_STATEMENT_EXPRESSIONS
96 #define RD_FORMAT(...) __attribute__((format(__VA_ARGS__)))
97 #else
98 #define RD_FORMAT(...)
99 #endif
100 
101 #ifndef LIBRDKAFKA_TYPECHECKS
102 #define LIBRDKAFKA_TYPECHECKS 1
103 #endif
104 #endif
105 
106 
112 #if LIBRDKAFKA_TYPECHECKS
113 #define _LRK_TYPECHECK(RET, TYPE, ARG) \
114  ({ \
115  if (0) { \
116  TYPE __t RD_UNUSED = (ARG); \
117  } \
118  RET; \
119  })
120 
121 #define _LRK_TYPECHECK2(RET, TYPE, ARG, TYPE2, ARG2) \
122  ({ \
123  if (0) { \
124  TYPE __t RD_UNUSED = (ARG); \
125  TYPE2 __t2 RD_UNUSED = (ARG2); \
126  } \
127  RET; \
128  })
129 
130 #define _LRK_TYPECHECK3(RET, TYPE, ARG, TYPE2, ARG2, TYPE3, ARG3) \
131  ({ \
132  if (0) { \
133  TYPE __t RD_UNUSED = (ARG); \
134  TYPE2 __t2 RD_UNUSED = (ARG2); \
135  TYPE3 __t3 RD_UNUSED = (ARG3); \
136  } \
137  RET; \
138  })
139 #else
140 #define _LRK_TYPECHECK(RET, TYPE, ARG) (RET)
141 #define _LRK_TYPECHECK2(RET, TYPE, ARG, TYPE2, ARG2) (RET)
142 #define _LRK_TYPECHECK3(RET, TYPE, ARG, TYPE2, ARG2, TYPE3, ARG3) (RET)
143 #endif
144 
145 /* @endcond */
146 
147 
169 #define RD_KAFKA_VERSION 0x020100ff
170 
179 RD_EXPORT
181 
187 RD_EXPORT
188 const char *rd_kafka_version_str(void);
189 
208 typedef enum rd_kafka_type_t {
212 
213 
224 
225 
226 
233 RD_EXPORT
234 const char *rd_kafka_get_debug_contexts(void);
235 
243 #define RD_KAFKA_DEBUG_CONTEXTS \
244  "all,generic,broker,topic,metadata,feature,queue,msg,protocol,cgrp," \
245  "security,fetch,interceptor,plugin,consumer,admin,eos,mock,assignor," \
246  "conf"
247 
248 
249 /* @cond NO_DOC */
250 /* Private types to provide ABI compatibility */
251 typedef struct rd_kafka_s rd_kafka_t;
252 typedef struct rd_kafka_topic_s rd_kafka_topic_t;
253 typedef struct rd_kafka_conf_s rd_kafka_conf_t;
254 typedef struct rd_kafka_topic_conf_s rd_kafka_topic_conf_t;
255 typedef struct rd_kafka_queue_s rd_kafka_queue_t;
256 typedef struct rd_kafka_op_s rd_kafka_event_t;
257 typedef struct rd_kafka_topic_result_s rd_kafka_topic_result_t;
258 typedef struct rd_kafka_consumer_group_metadata_s
259  rd_kafka_consumer_group_metadata_t;
260 typedef struct rd_kafka_error_s rd_kafka_error_t;
261 typedef struct rd_kafka_headers_s rd_kafka_headers_t;
262 typedef struct rd_kafka_group_result_s rd_kafka_group_result_t;
263 typedef struct rd_kafka_acl_result_s rd_kafka_acl_result_t;
264 /* @endcond */
265 
266 
279 typedef enum {
280  /* Internal errors to rdkafka: */
408 
411 
412  /* Kafka broker errors: */
428 #define RD_KAFKA_RESP_ERR_NOT_LEADER_OR_FOLLOWER \
429  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION
430  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION = 6,
448 #define RD_KAFKA_RESP_ERR_GROUP_LOAD_IN_PROGRESS \
449  RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
453 #define RD_KAFKA_RESP_ERR_GROUP_COORDINATOR_NOT_AVAILABLE \
454  RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
458 #define RD_KAFKA_RESP_ERR_NOT_COORDINATOR_FOR_GROUP \
459  RD_KAFKA_RESP_ERR_NOT_COORDINATOR
632 
633  RD_KAFKA_RESP_ERR_END_ALL,
635 
636 
644  const char *name;
645  const char *desc;
646 };
647 
648 
652 RD_EXPORT
653 void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs,
654  size_t *cntp);
655 
656 
657 
663 RD_EXPORT
665 
666 
667 
673 RD_EXPORT
675 
676 
702 RD_EXPORT
704 
705 
730 RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox);
731 
732 
745 RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void);
746 
747 
748 
775 RD_EXPORT
777 rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size);
778 
779 
799  const char *reason);
800 
801 
806 RD_EXPORT
807 rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error);
808 
817 RD_EXPORT
818 const char *rd_kafka_error_name(const rd_kafka_error_t *error);
819 
826 RD_EXPORT
827 const char *rd_kafka_error_string(const rd_kafka_error_t *error);
828 
829 
834 RD_EXPORT
835 int rd_kafka_error_is_fatal(const rd_kafka_error_t *error);
836 
837 
842 RD_EXPORT
843 int rd_kafka_error_is_retriable(const rd_kafka_error_t *error);
844 
845 
856 RD_EXPORT
857 int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error);
858 
864 RD_EXPORT
865 void rd_kafka_error_destroy(rd_kafka_error_t *error);
866 
867 
876 RD_EXPORT
878  const char *fmt,
879  ...) RD_FORMAT(printf, 2, 3);
880 
881 
897 typedef struct rd_kafka_topic_partition_s {
898  char *topic;
899  int32_t partition;
900  int64_t offset;
901  void *metadata;
902  size_t metadata_size;
903  void *opaque;
905  void *_private;
910 
911 
916 RD_EXPORT
918 
919 
928 RD_EXPORT
931  int32_t leader_epoch);
932 
941 RD_EXPORT
943  const rd_kafka_topic_partition_t *rktpar);
944 
949 typedef struct rd_kafka_topic_partition_list_s {
950  int cnt;
951  int size;
954 
955 
970 RD_EXPORT
972 
973 
977 RD_EXPORT
980 
990 RD_EXPORT
993  const char *topic,
994  int32_t partition);
995 
996 
1005 RD_EXPORT
1007  rd_kafka_topic_partition_list_t *rktparlist,
1008  const char *topic,
1009  int32_t start,
1010  int32_t stop);
1011 
1012 
1013 
1025 RD_EXPORT
1027  rd_kafka_topic_partition_list_t *rktparlist,
1028  const char *topic,
1029  int32_t partition);
1030 
1031 
1039 RD_EXPORT
1041  rd_kafka_topic_partition_list_t *rktparlist,
1042  int idx);
1043 
1044 
1052 RD_EXPORT
1055 
1056 
1057 
1065 RD_EXPORT
1067  rd_kafka_topic_partition_list_t *rktparlist,
1068  const char *topic,
1069  int32_t partition,
1070  int64_t offset);
1071 
1072 
1073 
1079 RD_EXPORT
1081  const rd_kafka_topic_partition_list_t *rktparlist,
1082  const char *topic,
1083  int32_t partition);
1084 
1085 
1096  rd_kafka_topic_partition_list_t *rktparlist,
1097  int (*cmp)(const void *a, const void *b, void *cmp_opaque),
1098  void *cmp_opaque);
1099 
1100 
1118 typedef enum rd_kafka_vtype_t {
1136 
1137 
1144 typedef struct rd_kafka_vu_s {
1147  union {
1148  const char *cstr;
1149  rd_kafka_topic_t *rkt;
1150  int i;
1151  int32_t i32;
1152  int64_t i64;
1153  struct {
1154  void *ptr;
1155  size_t size;
1156  } mem;
1157  struct {
1158  const char *name;
1159  const void *val;
1160  ssize_t size;
1161  } header;
1162  rd_kafka_headers_t *headers;
1163  void *ptr;
1164  char _pad[64];
1165  } u;
1166 } rd_kafka_vu_t;
1167 
1176 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
1177 
1183 #define RD_KAFKA_V_TOPIC(topic) \
1184  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
1185  (const char *)topic
1191 #define RD_KAFKA_V_RKT(rkt) \
1192  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
1193  (rd_kafka_topic_t *)rkt
1199 #define RD_KAFKA_V_PARTITION(partition) \
1200  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
1201  (int32_t)partition
1207 #define RD_KAFKA_V_VALUE(VALUE, LEN) \
1208  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
1209  (void *)VALUE, (size_t)LEN
1215 #define RD_KAFKA_V_KEY(KEY, LEN) \
1216  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
1217  (void *)KEY, (size_t)LEN
1225 #define RD_KAFKA_V_OPAQUE(msg_opaque) \
1226  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, msg_opaque), \
1227  (void *)msg_opaque
1234 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
1235  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), (int)msgflags
1242 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
1243  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
1244  (int64_t)timestamp
1253 #define RD_KAFKA_V_HEADER(NAME, VALUE, LEN) \
1254  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
1255  const void *, VALUE, ssize_t, LEN), \
1256  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
1257 
1269 #define RD_KAFKA_V_HEADERS(HDRS) \
1270  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
1271  (rd_kafka_headers_t *)HDRS
1272 
1273 
1304 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new(size_t initial_count);
1305 
1310 RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs);
1311 
1315 RD_EXPORT rd_kafka_headers_t *
1316 rd_kafka_headers_copy(const rd_kafka_headers_t *src);
1317 
1335 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs,
1336  const char *name,
1337  ssize_t name_size,
1338  const void *value,
1339  ssize_t value_size);
1340 
1348 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs,
1349  const char *name);
1350 
1351 
1369 RD_EXPORT rd_kafka_resp_err_t
1370 rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs,
1371  const char *name,
1372  const void **valuep,
1373  size_t *sizep);
1374 
1388 RD_EXPORT rd_kafka_resp_err_t
1389 rd_kafka_header_get(const rd_kafka_headers_t *hdrs,
1390  size_t idx,
1391  const char *name,
1392  const void **valuep,
1393  size_t *sizep);
1394 
1395 
1403 RD_EXPORT rd_kafka_resp_err_t
1404 rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs,
1405  size_t idx,
1406  const char **namep,
1407  const void **valuep,
1408  size_t *sizep);
1409 
1410 
1411 
1424 // FIXME: This doesn't show up in docs for some reason
1425 // "Compound rd_kafka_message_t is not documented."
1426 
1440 typedef struct rd_kafka_message_s {
1442  rd_kafka_topic_t *rkt;
1443  int32_t partition;
1444  void *payload;
1448  size_t len;
1451  void *key;
1453  size_t key_len;
1455  int64_t offset;
1463  void *_private;
1471 
1472 
1476 RD_EXPORT
1478 
1479 
1480 
1487 RD_EXPORT
1488 const char *rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage);
1489 
1490 
1502 RD_EXPORT
1504  rd_kafka_timestamp_type_t *tstype);
1505 
1506 
1507 
1514 RD_EXPORT
1516 
1517 
1524 RD_EXPORT
1526 
1527 
1544 RD_EXPORT rd_kafka_resp_err_t
1546  rd_kafka_headers_t **hdrsp);
1547 
1559 RD_EXPORT rd_kafka_resp_err_t
1561  rd_kafka_headers_t **hdrsp);
1562 
1563 
1576 RD_EXPORT
1578  rd_kafka_headers_t *hdrs);
1579 
1580 
1586 RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs);
1587 
1588 
1594 typedef enum {
1599 
1604 
1610 
1611 
1618 RD_EXPORT rd_kafka_msg_status_t
1620 
1621 
1629 RD_EXPORT int32_t
1631 
1632 
1648 typedef enum {
1650  RD_KAFKA_CONF_INVALID = -1,
1653  RD_KAFKA_CONF_OK = 0
1655 
1656 
1690 RD_EXPORT
1691 rd_kafka_conf_t *rd_kafka_conf_new(void);
1692 
1693 
1697 RD_EXPORT
1698 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1699 
1700 
1707 RD_EXPORT
1708 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1709 
1710 
1715 RD_EXPORT
1716 rd_kafka_conf_t *rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf,
1717  size_t filter_cnt,
1718  const char **filter);
1719 
1720 
1721 
1730 RD_EXPORT
1731 const rd_kafka_conf_t *rd_kafka_conf(rd_kafka_t *rk);
1732 
1733 
1753 RD_EXPORT
1755  const char *name,
1756  const char *value,
1757  char *errstr,
1758  size_t errstr_size);
1759 
1760 
1766 RD_EXPORT
1767 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1768 
1769 
1806  rd_kafka_conf_t *conf,
1807  void (*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque));
1808 
1809 
1813 RD_EXPORT
1814 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1815  void (*dr_cb)(rd_kafka_t *rk,
1816  void *payload,
1817  size_t len,
1818  rd_kafka_resp_err_t err,
1819  void *opaque,
1820  void *msg_opaque));
1821 
1852 RD_EXPORT
1854  rd_kafka_conf_t *conf,
1855  void (*dr_msg_cb)(rd_kafka_t *rk,
1856  const rd_kafka_message_t *rkmessage,
1857  void *opaque));
1858 
1859 
1867 RD_EXPORT
1869  rd_kafka_conf_t *conf,
1870  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque));
1871 
1976 RD_EXPORT
1978  rd_kafka_conf_t *conf,
1979  void (*rebalance_cb)(rd_kafka_t *rk,
1980  rd_kafka_resp_err_t err,
1981  rd_kafka_topic_partition_list_t *partitions,
1982  void *opaque));
1983 
1984 
1985 
2003 RD_EXPORT
2005  rd_kafka_conf_t *conf,
2006  void (*offset_commit_cb)(rd_kafka_t *rk,
2007  rd_kafka_resp_err_t err,
2009  void *opaque));
2010 
2011 
2034 RD_EXPORT
2035 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
2036  void (*error_cb)(rd_kafka_t *rk,
2037  int err,
2038  const char *reason,
2039  void *opaque));
2040 
2058 RD_EXPORT
2059 void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf,
2060  void (*throttle_cb)(rd_kafka_t *rk,
2061  const char *broker_name,
2062  int32_t broker_id,
2063  int throttle_time_ms,
2064  void *opaque));
2065 
2066 
2083 RD_EXPORT
2084 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
2085  void (*log_cb)(const rd_kafka_t *rk,
2086  int level,
2087  const char *fac,
2088  const char *buf));
2089 
2090 
2113 RD_EXPORT
2115  rd_kafka_conf_t *conf,
2116  int (*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque));
2117 
2167 RD_EXPORT
2169  rd_kafka_conf_t *conf,
2170  void (*oauthbearer_token_refresh_cb)(rd_kafka_t *rk,
2171  const char *oauthbearer_config,
2172  void *opaque));
2173 
2201 RD_EXPORT
2202 void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable);
2203 
2204 
2222 RD_EXPORT
2224  rd_kafka_conf_t *conf,
2225  int (*socket_cb)(int domain, int type, int protocol, void *opaque));
2226 
2227 
2228 
2244 RD_EXPORT void
2245 rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf,
2246  int (*connect_cb)(int sockfd,
2247  const struct sockaddr *addr,
2248  int addrlen,
2249  const char *id,
2250  void *opaque));
2251 
2263  rd_kafka_conf_t *conf,
2264  int (*closesocket_cb)(int sockfd, void *opaque));
2265 
2266 
2267 
2268 #ifndef _WIN32
2286 RD_EXPORT
2288  rd_kafka_conf_t *conf,
2289  int (*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque));
2290 #endif
2291 
2293 struct addrinfo;
2294 
2314 RD_EXPORT void
2315 rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf,
2316  int (*resolve_cb)(const char *node,
2317  const char *service,
2318  const struct addrinfo *hints,
2319  struct addrinfo **res,
2320  void *opaque));
2321 
2363 RD_EXPORT
2365  rd_kafka_conf_t *conf,
2366  int (*ssl_cert_verify_cb)(rd_kafka_t *rk,
2367  const char *broker_name,
2368  int32_t broker_id,
2369  int *x509_error,
2370  int depth,
2371  const char *buf,
2372  size_t size,
2373  char *errstr,
2374  size_t errstr_size,
2375  void *opaque));
2376 
2377 
2385 typedef enum rd_kafka_cert_type_t {
2389  RD_KAFKA_CERT__CNT,
2391 
2399 typedef enum rd_kafka_cert_enc_t {
2403  RD_KAFKA_CERT_ENC__CNT,
2405 
2406 
2449 RD_EXPORT rd_kafka_conf_res_t
2450 rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf,
2451  rd_kafka_cert_type_t cert_type,
2452  rd_kafka_cert_enc_t cert_enc,
2453  const void *buffer,
2454  size_t size,
2455  char *errstr,
2456  size_t errstr_size);
2457 
2458 
2473 RD_EXPORT
2474 void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf,
2475  void *callback_data);
2476 
2477 
2483 RD_EXPORT
2484 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
2485 
2490 RD_EXPORT
2491 void *rd_kafka_opaque(const rd_kafka_t *rk);
2492 
2493 
2494 
2508 RD_EXPORT
2509 void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf,
2510  rd_kafka_topic_conf_t *tconf);
2511 
2524 RD_EXPORT rd_kafka_topic_conf_t *
2526 
2527 
2547 RD_EXPORT
2548 rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf,
2549  const char *name,
2550  char *dest,
2551  size_t *dest_size);
2552 
2553 
2559 RD_EXPORT
2560 rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf,
2561  const char *name,
2562  char *dest,
2563  size_t *dest_size);
2564 
2565 
2574 RD_EXPORT
2575 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
2576 
2577 
2586 RD_EXPORT
2587 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
2588  size_t *cntp);
2589 
2594 RD_EXPORT
2595 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
2596 
2604 RD_EXPORT
2606 
2623 RD_EXPORT
2624 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
2625 
2626 
2630 RD_EXPORT
2631 rd_kafka_topic_conf_t *
2632 rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf);
2633 
2638 RD_EXPORT
2639 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup(rd_kafka_t *rk);
2640 
2641 
2645 RD_EXPORT
2646 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
2647 
2648 
2657 RD_EXPORT
2658 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
2659  const char *name,
2660  const char *value,
2661  char *errstr,
2662  size_t errstr_size);
2663 
2670 RD_EXPORT
2671 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf,
2672  void *rkt_opaque);
2673 
2674 
2694 RD_EXPORT
2696  rd_kafka_topic_conf_t *topic_conf,
2697  int32_t (*partitioner)(const rd_kafka_topic_t *rkt,
2698  const void *keydata,
2699  size_t keylen,
2700  int32_t partition_cnt,
2701  void *rkt_opaque,
2702  void *msg_opaque));
2703 
2704 
2731  rd_kafka_topic_conf_t *topic_conf,
2732  int (*msg_order_cmp)(const rd_kafka_message_t *a,
2733  const rd_kafka_message_t *b));
2734 
2735 
2743 RD_EXPORT
2744 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
2745  int32_t partition);
2746 
2747 
2748 /*******************************************************************
2749  * *
2750  * Partitioners provided by rdkafka *
2751  * *
2752  *******************************************************************/
2753 
2767 RD_EXPORT
2768 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
2769  const void *key,
2770  size_t keylen,
2771  int32_t partition_cnt,
2772  void *rkt_opaque,
2773  void *msg_opaque);
2774 
2788 RD_EXPORT
2789 int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt,
2790  const void *key,
2791  size_t keylen,
2792  int32_t partition_cnt,
2793  void *rkt_opaque,
2794  void *msg_opaque);
2795 
2811 RD_EXPORT
2812 int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt,
2813  const void *key,
2814  size_t keylen,
2815  int32_t partition_cnt,
2816  void *rkt_opaque,
2817  void *msg_opaque);
2818 
2819 
2833 RD_EXPORT
2834 int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt,
2835  const void *key,
2836  size_t keylen,
2837  int32_t partition_cnt,
2838  void *rkt_opaque,
2839  void *msg_opaque);
2840 
2855 RD_EXPORT
2856 int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt,
2857  const void *key,
2858  size_t keylen,
2859  int32_t partition_cnt,
2860  void *rkt_opaque,
2861  void *msg_opaque);
2862 
2863 
2877 RD_EXPORT
2878 int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt,
2879  const void *key,
2880  size_t keylen,
2881  int32_t partition_cnt,
2882  void *rkt_opaque,
2883  void *msg_opaque);
2884 
2885 
2900 RD_EXPORT
2901 int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt,
2902  const void *key,
2903  size_t keylen,
2904  int32_t partition_cnt,
2905  void *rkt_opaque,
2906  void *msg_opaque);
2907 
2908 
2948 RD_EXPORT
2950  rd_kafka_conf_t *conf,
2951  char *errstr,
2952  size_t errstr_size);
2953 
2954 
2969 RD_EXPORT
2970 void rd_kafka_destroy(rd_kafka_t *rk);
2971 
2972 
2977 RD_EXPORT
2978 void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags);
2979 
2994 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
2995 
2996 
2997 
3001 RD_EXPORT
3002 const char *rd_kafka_name(const rd_kafka_t *rk);
3003 
3004 
3008 RD_EXPORT
3009 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
3010 
3011 
3022 RD_EXPORT
3023 char *rd_kafka_memberid(const rd_kafka_t *rk);
3024 
3025 
3026 
3045 RD_EXPORT
3046 char *rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms);
3047 
3048 
3064 RD_EXPORT
3065 int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms);
3066 
3067 
3089 RD_EXPORT
3090 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk,
3091  const char *topic,
3092  rd_kafka_topic_conf_t *conf);
3093 
3094 
3095 
3104 RD_EXPORT
3105 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
3106 
3107 
3111 RD_EXPORT
3112 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
3113 
3114 
3119 RD_EXPORT
3120 void *rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt);
3121 
3122 
3129 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
3130 
3131 
3159 RD_EXPORT
3160 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
3161 
3162 
3173 RD_EXPORT
3174 void rd_kafka_yield(rd_kafka_t *rk);
3175 
3176 
3177 
3185 RD_EXPORT rd_kafka_resp_err_t
3187  rd_kafka_topic_partition_list_t *partitions);
3188 
3189 
3190 
3198 RD_EXPORT rd_kafka_resp_err_t
3200  rd_kafka_topic_partition_list_t *partitions);
3201 
3202 
3203 
3212 RD_EXPORT rd_kafka_resp_err_t
3214  const char *topic,
3215  int32_t partition,
3216  int64_t *low,
3217  int64_t *high,
3218  int timeout_ms);
3219 
3220 
3238  const char *topic,
3239  int32_t partition,
3240  int64_t *low,
3241  int64_t *high);
3242 
3243 
3244 
3270 RD_EXPORT rd_kafka_resp_err_t
3273  int timeout_ms);
3274 
3275 
3276 
3290 RD_EXPORT
3291 void *rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size);
3292 
3293 
3294 
3308 RD_EXPORT
3309 void *rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size);
3310 
3311 
3312 
3329 RD_EXPORT
3330 void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr);
3331 
3332 
3354 RD_EXPORT
3355 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
3356 
3360 RD_EXPORT
3361 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
3362 
3363 
3370 RD_EXPORT
3371 rd_kafka_queue_t *rd_kafka_queue_get_main(rd_kafka_t *rk);
3372 
3373 
3374 
3384 RD_EXPORT
3385 rd_kafka_queue_t *rd_kafka_queue_get_sasl(rd_kafka_t *rk);
3386 
3387 
3403 RD_EXPORT
3404 rd_kafka_error_t *rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk);
3405 
3406 
3421 RD_EXPORT
3422 rd_kafka_error_t *rd_kafka_sasl_set_credentials(rd_kafka_t *rk,
3423  const char *username,
3424  const char *password);
3425 
3435 RD_EXPORT
3436 rd_kafka_queue_t *rd_kafka_queue_get_consumer(rd_kafka_t *rk);
3437 
3448 RD_EXPORT
3449 rd_kafka_queue_t *rd_kafka_queue_get_partition(rd_kafka_t *rk,
3450  const char *topic,
3451  int32_t partition);
3452 
3475 RD_EXPORT
3476 rd_kafka_queue_t *rd_kafka_queue_get_background(rd_kafka_t *rk);
3477 
3478 
3489 RD_EXPORT
3490 void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
3491 
3510 RD_EXPORT
3512  rd_kafka_queue_t *rkqu);
3513 
3514 
3518 RD_EXPORT
3519 size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu);
3520 
3521 
3539 RD_EXPORT
3540 void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu,
3541  int fd,
3542  const void *payload,
3543  size_t size);
3544 
3561 RD_EXPORT
3562 void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu,
3563  void (*event_cb)(rd_kafka_t *rk,
3564  void *qev_opaque),
3565  void *qev_opaque);
3566 
3567 
3575 RD_EXPORT
3576 void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu);
3577 
3578 
3589 #define RD_KAFKA_OFFSET_BEGINNING \
3590  -2
3592 #define RD_KAFKA_OFFSET_END \
3593  -1
3595 #define RD_KAFKA_OFFSET_STORED \
3596  -1000
3598 #define RD_KAFKA_OFFSET_INVALID -1001
3602 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
3610 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
3611 
3645 RD_EXPORT
3646 int rd_kafka_consume_start(rd_kafka_topic_t *rkt,
3647  int32_t partition,
3648  int64_t offset);
3649 
3664 RD_EXPORT
3665 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt,
3666  int32_t partition,
3667  int64_t offset,
3668  rd_kafka_queue_t *rkqu);
3669 
3683 RD_EXPORT
3684 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
3685 
3686 
3687 
3716 RD_EXPORT
3717 rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt,
3718  int32_t partition,
3719  int64_t offset,
3720  int timeout_ms);
3721 
3722 
3723 
3755 RD_EXPORT rd_kafka_error_t *
3757  rd_kafka_topic_partition_list_t *partitions,
3758  int timeout_ms);
3759 
3760 
3785 RD_EXPORT
3787 rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms);
3788 
3789 
3790 
3816 RD_EXPORT
3817 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt,
3818  int32_t partition,
3819  int timeout_ms,
3820  rd_kafka_message_t **rkmessages,
3821  size_t rkmessages_size);
3822 
3823 
3824 
3858 RD_EXPORT
3859 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt,
3860  int32_t partition,
3861  int timeout_ms,
3862  void (*consume_cb)(rd_kafka_message_t *rkmessage,
3863  void *commit_opaque),
3864  void *commit_opaque);
3865 
3866 
3885 RD_EXPORT
3887  int timeout_ms);
3888 
3894 RD_EXPORT
3895 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
3896  int timeout_ms,
3897  rd_kafka_message_t **rkmessages,
3898  size_t rkmessages_size);
3899 
3909 RD_EXPORT
3911  rd_kafka_queue_t *rkqu,
3912  int timeout_ms,
3913  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque),
3914  void *commit_opaque);
3915 
3916 
3956 RD_EXPORT
3958 rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset);
3959 
3960 
3993 RD_EXPORT rd_kafka_resp_err_t
3994 rd_kafka_offsets_store(rd_kafka_t *rk,
3996 
3997 
4018 RD_EXPORT
4020 
4074 RD_EXPORT rd_kafka_resp_err_t
4075 rd_kafka_subscribe(rd_kafka_t *rk,
4076  const rd_kafka_topic_partition_list_t *topics);
4077 
4078 
4082 RD_EXPORT
4084 
4085 
4095 RD_EXPORT rd_kafka_resp_err_t
4097 
4098 
4099 
4129 RD_EXPORT
4130 rd_kafka_message_t *rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms);
4131 
4149 RD_EXPORT
4151 
4152 
4170 RD_EXPORT
4171 rd_kafka_error_t *rd_kafka_consumer_close_queue(rd_kafka_t *rk,
4172  rd_kafka_queue_t *rkqu);
4173 
4174 
4183 RD_EXPORT
4184 int rd_kafka_consumer_closed(rd_kafka_t *rk);
4185 
4186 
4204 RD_EXPORT rd_kafka_error_t *
4206  const rd_kafka_topic_partition_list_t *partitions);
4207 
4208 
4226 RD_EXPORT rd_kafka_error_t *rd_kafka_incremental_unassign(
4227  rd_kafka_t *rk,
4228  const rd_kafka_topic_partition_list_t *partitions);
4229 
4230 
4244 RD_EXPORT
4245 const char *rd_kafka_rebalance_protocol(rd_kafka_t *rk);
4246 
4247 
4268 RD_EXPORT rd_kafka_resp_err_t
4269 rd_kafka_assign(rd_kafka_t *rk,
4270  const rd_kafka_topic_partition_list_t *partitions);
4271 
4288 RD_EXPORT rd_kafka_resp_err_t
4289 rd_kafka_assignment(rd_kafka_t *rk,
4290  rd_kafka_topic_partition_list_t **partitions);
4291 
4292 
4309 RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk);
4310 
4311 
4332 RD_EXPORT rd_kafka_resp_err_t
4333 rd_kafka_commit(rd_kafka_t *rk,
4334  const rd_kafka_topic_partition_list_t *offsets,
4335  int async);
4336 
4337 
4344 RD_EXPORT rd_kafka_resp_err_t
4346  const rd_kafka_message_t *rkmessage,
4347  int async);
4348 
4349 
4372 RD_EXPORT rd_kafka_resp_err_t
4373 rd_kafka_commit_queue(rd_kafka_t *rk,
4374  const rd_kafka_topic_partition_list_t *offsets,
4375  rd_kafka_queue_t *rkqu,
4376  void (*cb)(rd_kafka_t *rk,
4377  rd_kafka_resp_err_t err,
4379  void *commit_opaque),
4380  void *commit_opaque);
4381 
4382 
4400 RD_EXPORT rd_kafka_resp_err_t
4401 rd_kafka_committed(rd_kafka_t *rk,
4402  rd_kafka_topic_partition_list_t *partitions,
4403  int timeout_ms);
4404 
4405 
4406 
4423 RD_EXPORT rd_kafka_resp_err_t
4425 
4426 
4427 
4439 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4441 
4442 
4452 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4454 
4455 
4468 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4470  int32_t generation_id,
4471  const char *member_id,
4472  const char *group_instance_id);
4473 
4474 
4479 RD_EXPORT void
4480 rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *);
4481 
4482 
4503 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_write(
4504  const rd_kafka_consumer_group_metadata_t *cgmd,
4505  void **bufferp,
4506  size_t *sizep);
4507 
4528 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_read(
4529  rd_kafka_consumer_group_metadata_t **cgmdp,
4530  const void *buffer,
4531  size_t size);
4532 
4548 #define RD_KAFKA_MSG_F_FREE \
4549  0x1
4551 #define RD_KAFKA_MSG_F_COPY \
4552  0x2
4554 #define RD_KAFKA_MSG_F_BLOCK \
4555  0x4
4566 #define RD_KAFKA_MSG_F_PARTITION \
4567  0x8
4672 RD_EXPORT
4673 int rd_kafka_produce(rd_kafka_topic_t *rkt,
4674  int32_t partition,
4675  int msgflags,
4676  void *payload,
4677  size_t len,
4678  const void *key,
4679  size_t keylen,
4680  void *msg_opaque);
4681 
4682 
4696 RD_EXPORT
4698 
4699 
4711 RD_EXPORT
4712 rd_kafka_error_t *
4713 rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt);
4714 
4715 
4745 RD_EXPORT
4746 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt,
4747  int32_t partition,
4748  int msgflags,
4749  rd_kafka_message_t *rkmessages,
4750  int message_cnt);
4751 
4752 
4753 
4778 RD_EXPORT
4779 rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms);
4780 
4781 
4782 
4813 RD_EXPORT
4814 rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags);
4815 
4816 
4824 #define RD_KAFKA_PURGE_F_QUEUE 0x1
4825 
4833 #define RD_KAFKA_PURGE_F_INFLIGHT 0x2
4834 
4835 
4839 #define RD_KAFKA_PURGE_F_NON_BLOCKING 0x4
4840 
4841 
4856 typedef struct rd_kafka_metadata_broker {
4857  int32_t id;
4858  char *host;
4859  int port;
4861 
4865 typedef struct rd_kafka_metadata_partition {
4866  int32_t id;
4868  int32_t leader;
4870  int32_t *replicas;
4871  int isr_cnt;
4872  int32_t *isrs;
4874 
4878 typedef struct rd_kafka_metadata_topic {
4879  char *topic;
4881  struct rd_kafka_metadata_partition *partitions;
4884 
4885 
4889 typedef struct rd_kafka_metadata {
4891  struct rd_kafka_metadata_broker *brokers;
4894  struct rd_kafka_metadata_topic *topics;
4896  int32_t orig_broker_id;
4899 
4920 RD_EXPORT
4922 rd_kafka_metadata(rd_kafka_t *rk,
4923  int all_topics,
4924  rd_kafka_topic_t *only_rkt,
4925  const struct rd_kafka_metadata **metadatap,
4926  int timeout_ms);
4927 
4931 RD_EXPORT
4932 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
4933 
4937 typedef struct rd_kafka_Node_s rd_kafka_Node_t;
4938 
4946 RD_EXPORT
4948 
4959 RD_EXPORT
4960 const char *rd_kafka_Node_host(const rd_kafka_Node_t *node);
4961 
4969 RD_EXPORT
4970 uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node);
4971 
4992  char *member_id;
4993  char *client_id;
4994  char *client_host;
5001 };
5002 
5008 typedef enum {
5009  RD_KAFKA_CONSUMER_GROUP_STATE_UNKNOWN = 0,
5010  RD_KAFKA_CONSUMER_GROUP_STATE_PREPARING_REBALANCE = 1,
5011  RD_KAFKA_CONSUMER_GROUP_STATE_COMPLETING_REBALANCE = 2,
5012  RD_KAFKA_CONSUMER_GROUP_STATE_STABLE = 3,
5013  RD_KAFKA_CONSUMER_GROUP_STATE_DEAD = 4,
5014  RD_KAFKA_CONSUMER_GROUP_STATE_EMPTY = 5,
5015  RD_KAFKA_CONSUMER_GROUP_STATE__CNT
5017 
5022  struct rd_kafka_metadata_broker broker;
5023  char *group;
5025  char *state;
5027  char *protocol;
5030 };
5031 
5040 };
5041 
5042 
5073 RD_EXPORT
5075 rd_kafka_list_groups(rd_kafka_t *rk,
5076  const char *group,
5077  const struct rd_kafka_group_list **grplistp,
5078  int timeout_ms);
5079 
5087 RD_EXPORT
5088 const char *
5090 
5098 RD_EXPORT
5101 
5105 RD_EXPORT
5107 
5108 
5152 RD_EXPORT
5153 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
5154 
5155 
5156 
5169 RD_EXPORT RD_DEPRECATED void
5170 rd_kafka_set_logger(rd_kafka_t *rk,
5171  void (*func)(const rd_kafka_t *rk,
5172  int level,
5173  const char *fac,
5174  const char *buf));
5175 
5176 
5186 RD_EXPORT
5187 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
5188 
5189 
5193 RD_EXPORT
5194 void rd_kafka_log_print(const rd_kafka_t *rk,
5195  int level,
5196  const char *fac,
5197  const char *buf);
5198 
5199 
5205 RD_EXPORT
5206 void rd_kafka_log_syslog(const rd_kafka_t *rk,
5207  int level,
5208  const char *fac,
5209  const char *buf);
5210 
5211 
5234 RD_EXPORT
5235 int rd_kafka_outq_len(rd_kafka_t *rk);
5236 
5237 
5238 
5245 RD_EXPORT
5246 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
5247 
5248 
5249 
5255 RD_EXPORT
5257 
5258 
5271 
5272 
5281 RD_EXPORT
5282 int rd_kafka_wait_destroyed(int timeout_ms);
5283 
5284 
5290 RD_EXPORT
5292 
5293 
5310 RD_EXPORT
5312 
5313 
5330 #define RD_KAFKA_EVENT_NONE 0x0
5331 #define RD_KAFKA_EVENT_DR 0x1
5332 #define RD_KAFKA_EVENT_FETCH 0x2
5333 #define RD_KAFKA_EVENT_LOG 0x4
5334 #define RD_KAFKA_EVENT_ERROR 0x8
5335 #define RD_KAFKA_EVENT_REBALANCE 0x10
5336 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
5337 #define RD_KAFKA_EVENT_STATS 0x40
5338 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
5339 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
5340 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT \
5341  102
5342 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
5343 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT \
5344  104
5345 #define RD_KAFKA_EVENT_DELETERECORDS_RESULT 105
5346 #define RD_KAFKA_EVENT_DELETEGROUPS_RESULT 106
5348 #define RD_KAFKA_EVENT_DELETECONSUMERGROUPOFFSETS_RESULT 107
5350 #define RD_KAFKA_EVENT_OAUTHBEARER_TOKEN_REFRESH 0x100
5351 #define RD_KAFKA_EVENT_BACKGROUND 0x200
5352 #define RD_KAFKA_EVENT_CREATEACLS_RESULT 0x400
5353 #define RD_KAFKA_EVENT_DESCRIBEACLS_RESULT 0x800
5354 #define RD_KAFKA_EVENT_DELETEACLS_RESULT 0x1000
5356 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPS_RESULT 0x2000
5358 #define RD_KAFKA_EVENT_DESCRIBECONSUMERGROUPS_RESULT 0x4000
5360 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPOFFSETS_RESULT 0x8000
5362 #define RD_KAFKA_EVENT_ALTERCONSUMERGROUPOFFSETS_RESULT 0x10000
5363 
5364 
5371 RD_EXPORT
5372 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev);
5373 
5380 RD_EXPORT
5381 const char *rd_kafka_event_name(const rd_kafka_event_t *rkev);
5382 
5383 
5393 RD_EXPORT
5394 void rd_kafka_event_destroy(rd_kafka_event_t *rkev);
5395 
5396 
5412 RD_EXPORT
5413 const rd_kafka_message_t *rd_kafka_event_message_next(rd_kafka_event_t *rkev);
5414 
5415 
5429 RD_EXPORT
5430 size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev,
5431  const rd_kafka_message_t **rkmessages,
5432  size_t size);
5433 
5434 
5442 RD_EXPORT
5443 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev);
5444 
5445 
5457 RD_EXPORT
5458 const char *rd_kafka_event_config_string(rd_kafka_event_t *rkev);
5459 
5460 
5469 RD_EXPORT
5471 
5472 
5481 RD_EXPORT
5482 const char *rd_kafka_event_error_string(rd_kafka_event_t *rkev);
5483 
5484 
5493 RD_EXPORT
5494 int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev);
5495 
5496 
5519 RD_EXPORT
5520 void *rd_kafka_event_opaque(rd_kafka_event_t *rkev);
5521 
5522 
5531 RD_EXPORT
5532 int rd_kafka_event_log(rd_kafka_event_t *rkev,
5533  const char **fac,
5534  const char **str,
5535  int *level);
5536 
5537 
5549 RD_EXPORT
5550 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev,
5551  char *dst,
5552  size_t dstsize);
5553 
5554 
5567 RD_EXPORT
5568 const char *rd_kafka_event_stats(rd_kafka_event_t *rkev);
5569 
5570 
5582 rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev);
5583 
5584 
5595 RD_EXPORT rd_kafka_topic_partition_t *
5596 rd_kafka_event_topic_partition(rd_kafka_event_t *rkev);
5597 
5598 
5600 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
5602 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
5604 typedef rd_kafka_event_t rd_kafka_CreateAcls_result_t;
5606 typedef rd_kafka_event_t rd_kafka_DescribeAcls_result_t;
5608 typedef rd_kafka_event_t rd_kafka_DeleteAcls_result_t;
5610 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
5612 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
5614 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
5616 typedef rd_kafka_event_t rd_kafka_DeleteRecords_result_t;
5618 typedef rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t;
5622 typedef rd_kafka_event_t rd_kafka_DeleteGroups_result_t;
5629 
5639 RD_EXPORT const rd_kafka_CreateTopics_result_t *
5640 rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev);
5641 
5651 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
5652 rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev);
5653 
5663 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
5665 
5675 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
5676 rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev);
5677 
5687 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
5689 
5697 RD_EXPORT const rd_kafka_DeleteRecords_result_t *
5698 rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev);
5699 
5712 RD_EXPORT const rd_kafka_ListConsumerGroups_result_t *
5714 
5729 
5739 RD_EXPORT const rd_kafka_DeleteGroups_result_t *
5740 rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev);
5741 
5753 
5761 RD_EXPORT const rd_kafka_CreateAcls_result_t *
5762 rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev);
5763 
5771 RD_EXPORT const rd_kafka_DescribeAcls_result_t *
5772 rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev);
5773 
5781 RD_EXPORT const rd_kafka_DeleteAcls_result_t *
5782 rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev);
5783 
5798 
5813 
5823 RD_EXPORT
5824 rd_kafka_event_t *rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms);
5825 
5840 RD_EXPORT
5841 int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms);
5842 
5843 
5888  rd_kafka_conf_t *conf,
5889  void **plug_opaquep,
5890  char *errstr,
5891  size_t errstr_size);
5892 
5974  rd_kafka_conf_t *conf,
5975  const char *name,
5976  const char *val,
5977  char *errstr,
5978  size_t errstr_size,
5979  void *ic_opaque);
5980 
5981 
6004  rd_kafka_conf_t *new_conf,
6005  const rd_kafka_conf_t *old_conf,
6006  size_t filter_cnt,
6007  const char **filter,
6008  void *ic_opaque);
6009 
6010 
6018  void *ic_opaque);
6019 
6020 
6039  rd_kafka_t *rk,
6040  const rd_kafka_conf_t *conf,
6041  void *ic_opaque,
6042  char *errstr,
6043  size_t errstr_size);
6044 
6045 
6054  rd_kafka_interceptor_f_on_destroy_t)(rd_kafka_t *rk, void *ic_opaque);
6055 
6056 
6057 
6079  rd_kafka_t *rk,
6080  rd_kafka_message_t *rkmessage,
6081  void *ic_opaque);
6082 
6106  rd_kafka_t *rk,
6107  rd_kafka_message_t *rkmessage,
6108  void *ic_opaque);
6109 
6110 
6128  rd_kafka_t *rk,
6129  rd_kafka_message_t *rkmessage,
6130  void *ic_opaque);
6131 
6154  rd_kafka_t *rk,
6155  const rd_kafka_topic_partition_list_t *offsets,
6157  void *ic_opaque);
6158 
6159 
6182  rd_kafka_t *rk,
6183  int sockfd,
6184  const char *brokername,
6185  int32_t brokerid,
6186  int16_t ApiKey,
6187  int16_t ApiVersion,
6188  int32_t CorrId,
6189  size_t size,
6190  void *ic_opaque);
6191 
6192 
6219  rd_kafka_t *rk,
6220  int sockfd,
6221  const char *brokername,
6222  int32_t brokerid,
6223  int16_t ApiKey,
6224  int16_t ApiVersion,
6225  int32_t CorrId,
6226  size_t size,
6227  int64_t rtt,
6229  void *ic_opaque);
6230 
6231 
6249  rd_kafka_t *rk,
6250  rd_kafka_thread_type_t thread_type,
6251  const char *thread_name,
6252  void *ic_opaque);
6253 
6254 
6275  rd_kafka_t *rk,
6276  rd_kafka_thread_type_t thread_type,
6277  const char *thread_name,
6278  void *ic_opaque);
6279 
6280 
6296  rd_kafka_t *rk,
6297  int32_t broker_id,
6298  const char *secproto,
6299  const char *name,
6300  int port,
6301  const char *state,
6302  void *ic_opaque);
6303 
6304 
6318  rd_kafka_conf_t *conf,
6319  const char *ic_name,
6321  void *ic_opaque);
6322 
6323 
6337  rd_kafka_conf_t *conf,
6338  const char *ic_name,
6340  void *ic_opaque);
6341 
6356  rd_kafka_conf_t *conf,
6357  const char *ic_name,
6359  void *ic_opaque);
6360 
6361 
6383 RD_EXPORT rd_kafka_resp_err_t
6385  const char *ic_name,
6387  void *ic_opaque);
6388 
6389 
6390 
6404  rd_kafka_t *rk,
6405  const char *ic_name,
6407  void *ic_opaque);
6408 
6409 
6422 RD_EXPORT rd_kafka_resp_err_t
6424  const char *ic_name,
6426  void *ic_opaque);
6427 
6441  rd_kafka_t *rk,
6442  const char *ic_name,
6443  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
6444  void *ic_opaque);
6445 
6446 
6460  rd_kafka_t *rk,
6461  const char *ic_name,
6463  void *ic_opaque);
6464 
6465 
6479  rd_kafka_t *rk,
6480  const char *ic_name,
6482  void *ic_opaque);
6483 
6484 
6498  rd_kafka_t *rk,
6499  const char *ic_name,
6501  void *ic_opaque);
6502 
6503 
6517  rd_kafka_t *rk,
6518  const char *ic_name,
6519  rd_kafka_interceptor_f_on_response_received_t *on_response_received,
6520  void *ic_opaque);
6521 
6522 
6536  rd_kafka_t *rk,
6537  const char *ic_name,
6539  void *ic_opaque);
6540 
6541 
6555  rd_kafka_t *rk,
6556  const char *ic_name,
6558  void *ic_opaque);
6559 
6560 
6573 RD_EXPORT
6575  rd_kafka_t *rk,
6576  const char *ic_name,
6577  rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change,
6578  void *ic_opaque);
6579 
6580 
6581 
6602 RD_EXPORT rd_kafka_resp_err_t
6603 rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres);
6604 
6611 RD_EXPORT const char *
6612 rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres);
6613 
6619 RD_EXPORT const char *
6620 rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres);
6621 
6631 RD_EXPORT const rd_kafka_error_t *
6632 rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres);
6633 
6639 RD_EXPORT const char *
6640 rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres);
6641 
6647 RD_EXPORT const rd_kafka_topic_partition_list_t *
6648 rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres);
6649 
6650 
6697 typedef enum rd_kafka_admin_op_t {
6719 
6732 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
6733 
6753 RD_EXPORT rd_kafka_AdminOptions_t *
6755 
6756 
6761 
6762 
6781 RD_EXPORT rd_kafka_resp_err_t
6783  int timeout_ms,
6784  char *errstr,
6785  size_t errstr_size);
6786 
6787 
6814 RD_EXPORT rd_kafka_resp_err_t
6816  int timeout_ms,
6817  char *errstr,
6818  size_t errstr_size);
6819 
6820 
6839 RD_EXPORT rd_kafka_resp_err_t
6841  int true_or_false,
6842  char *errstr,
6843  size_t errstr_size);
6844 
6845 
6871 RD_EXPORT rd_kafka_resp_err_t
6873  int32_t broker_id,
6874  char *errstr,
6875  size_t errstr_size);
6876 
6877 
6890 RD_EXPORT
6892  rd_kafka_AdminOptions_t *options,
6893  int true_or_false);
6894 
6907 RD_EXPORT
6909  rd_kafka_AdminOptions_t *options,
6910  const rd_kafka_consumer_group_state_t *consumer_group_states,
6911  size_t consumer_group_states_cnt);
6912 
6917 RD_EXPORT void
6919  void *ev_opaque);
6920 
6932 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
6933 
6955 RD_EXPORT rd_kafka_NewTopic_t *rd_kafka_NewTopic_new(const char *topic,
6956  int num_partitions,
6957  int replication_factor,
6958  char *errstr,
6959  size_t errstr_size);
6960 
6966 
6967 
6974  size_t new_topic_cnt);
6975 
6976 
6998 RD_EXPORT rd_kafka_resp_err_t
7000  int32_t partition,
7001  int32_t *broker_ids,
7002  size_t broker_id_cnt,
7003  char *errstr,
7004  size_t errstr_size);
7005 
7018 RD_EXPORT rd_kafka_resp_err_t
7020  const char *name,
7021  const char *value);
7022 
7023 
7042 RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk,
7043  rd_kafka_NewTopic_t **new_topics,
7044  size_t new_topic_cnt,
7045  const rd_kafka_AdminOptions_t *options,
7046  rd_kafka_queue_t *rkqu);
7047 
7048 
7049 /*
7050  * CreateTopics result type and methods
7051  */
7052 
7061 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_CreateTopics_result_topics(
7062  const rd_kafka_CreateTopics_result_t *result,
7063  size_t *cntp);
7064 
7065 
7066 
7067 /*
7068  * DeleteTopics - delete topics from cluster
7069  *
7070  */
7071 
7073 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
7074 
7084 RD_EXPORT rd_kafka_DeleteTopic_t *rd_kafka_DeleteTopic_new(const char *topic);
7085 
7091 
7097 RD_EXPORT void
7099  size_t del_topic_cnt);
7100 
7114 RD_EXPORT
7115 void rd_kafka_DeleteTopics(rd_kafka_t *rk,
7116  rd_kafka_DeleteTopic_t **del_topics,
7117  size_t del_topic_cnt,
7118  const rd_kafka_AdminOptions_t *options,
7119  rd_kafka_queue_t *rkqu);
7120 
7121 
7122 
7123 /*
7124  * DeleteTopics result type and methods
7125  */
7126 
7135 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_DeleteTopics_result_topics(
7136  const rd_kafka_DeleteTopics_result_t *result,
7137  size_t *cntp);
7138 
7139 
7150 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
7151 
7168 RD_EXPORT rd_kafka_NewPartitions_t *
7169 rd_kafka_NewPartitions_new(const char *topic,
7170  size_t new_total_cnt,
7171  char *errstr,
7172  size_t errstr_size);
7173 
7178 RD_EXPORT void
7180 
7186 RD_EXPORT void
7188  size_t new_parts_cnt);
7189 
7213  rd_kafka_NewPartitions_t *new_parts,
7214  int32_t new_partition_idx,
7215  int32_t *broker_ids,
7216  size_t broker_id_cnt,
7217  char *errstr,
7218  size_t errstr_size);
7219 
7220 
7239 RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk,
7240  rd_kafka_NewPartitions_t **new_parts,
7241  size_t new_parts_cnt,
7242  const rd_kafka_AdminOptions_t *options,
7243  rd_kafka_queue_t *rkqu);
7244 
7245 
7246 
7247 /*
7248  * CreatePartitions result type and methods
7249  */
7250 
7259 RD_EXPORT const rd_kafka_topic_result_t **
7261  const rd_kafka_CreatePartitions_result_t *result,
7262  size_t *cntp);
7263 
7299 
7303 
7304 
7308 RD_EXPORT const char *
7310 
7311 
7313 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
7314 
7318 RD_EXPORT const char *
7320 
7325 RD_EXPORT const char *
7327 
7331 RD_EXPORT rd_kafka_ConfigSource_t
7333 
7338 RD_EXPORT int
7340 
7346 RD_EXPORT int
7348 
7356 RD_EXPORT int
7358 
7362 RD_EXPORT int
7364 
7365 
7376 RD_EXPORT const rd_kafka_ConfigEntry_t **
7378  size_t *cntp);
7379 
7380 
7381 
7394 
7410  RD_KAFKA_RESOURCE_PATTERN_TYPE__CNT,
7412 
7417  rd_kafka_ResourcePatternType_t resource_pattern_type);
7418 
7422 RD_EXPORT const char *
7424 
7426 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
7427 
7428 
7437 RD_EXPORT rd_kafka_ConfigResource_t *
7439  const char *resname);
7440 
7445 RD_EXPORT void
7447 
7448 
7454 RD_EXPORT void
7456  size_t config_cnt);
7457 
7458 
7472 RD_EXPORT rd_kafka_resp_err_t
7474  const char *name,
7475  const char *value);
7476 
7477 
7486 RD_EXPORT const rd_kafka_ConfigEntry_t **
7488  size_t *cntp);
7489 
7490 
7491 
7495 RD_EXPORT rd_kafka_ResourceType_t
7497 
7501 RD_EXPORT const char *
7503 
7507 RD_EXPORT rd_kafka_resp_err_t
7509 
7514 RD_EXPORT const char *
7516 
7517 
7518 /*
7519  * AlterConfigs - alter cluster configuration.
7520  *
7521  */
7522 
7523 
7544 RD_EXPORT
7545 void rd_kafka_AlterConfigs(rd_kafka_t *rk,
7546  rd_kafka_ConfigResource_t **configs,
7547  size_t config_cnt,
7548  const rd_kafka_AdminOptions_t *options,
7549  rd_kafka_queue_t *rkqu);
7550 
7551 
7552 /*
7553  * AlterConfigs result type and methods
7554  */
7555 
7570 RD_EXPORT const rd_kafka_ConfigResource_t **
7572  const rd_kafka_AlterConfigs_result_t *result,
7573  size_t *cntp);
7574 
7575 
7576 
7577 /*
7578  * DescribeConfigs - retrieve cluster configuration.
7579  *
7580  */
7581 
7582 
7607 RD_EXPORT
7608 void rd_kafka_DescribeConfigs(rd_kafka_t *rk,
7609  rd_kafka_ConfigResource_t **configs,
7610  size_t config_cnt,
7611  const rd_kafka_AdminOptions_t *options,
7612  rd_kafka_queue_t *rkqu);
7613 
7614 
7615 
7616 /*
7617  * DescribeConfigs result type and methods
7618  */
7619 
7628 RD_EXPORT const rd_kafka_ConfigResource_t **
7630  const rd_kafka_DescribeConfigs_result_t *result,
7631  size_t *cntp);
7632 
7633 
7644 typedef struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t;
7645 
7663  const rd_kafka_topic_partition_list_t *before_offsets);
7664 
7669 RD_EXPORT void
7671 
7677 RD_EXPORT void
7679  size_t del_record_cnt);
7680 
7702 RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk,
7703  rd_kafka_DeleteRecords_t **del_records,
7704  size_t del_record_cnt,
7705  const rd_kafka_AdminOptions_t *options,
7706  rd_kafka_queue_t *rkqu);
7707 
7708 
7709 /*
7710  * DeleteRecords result type and methods
7711  */
7712 
7722 RD_EXPORT const rd_kafka_topic_partition_list_t *
7724  const rd_kafka_DeleteRecords_result_t *result);
7725 
7739 typedef struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t;
7740 
7742 typedef struct rd_kafka_ListConsumerGroupsResult_s
7744 
7755 RD_EXPORT
7756 void rd_kafka_ListConsumerGroups(rd_kafka_t *rk,
7757  const rd_kafka_AdminOptions_t *options,
7758  rd_kafka_queue_t *rkqu);
7759 
7770 RD_EXPORT
7772  const rd_kafka_ConsumerGroupListing_t *grplist);
7773 
7782 RD_EXPORT
7784  const rd_kafka_ConsumerGroupListing_t *grplist);
7785 
7793 RD_EXPORT
7795  const rd_kafka_ConsumerGroupListing_t *grplist);
7796 
7808 RD_EXPORT
7812  size_t *cntp);
7813 
7827 RD_EXPORT
7830  size_t *cntp);
7831 
7843 typedef struct rd_kafka_ConsumerGroupDescription_s
7845 
7850 typedef struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t;
7851 
7856 typedef struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t;
7857 
7871 RD_EXPORT
7873  const char **groups,
7874  size_t groups_cnt,
7875  const rd_kafka_AdminOptions_t *options,
7876  rd_kafka_queue_t *rkqu);
7877 
7889 RD_EXPORT
7893  size_t *cntp);
7894 
7895 
7906 RD_EXPORT
7908  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
7909 
7920 RD_EXPORT
7922  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
7923 
7931 RD_EXPORT
7933  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
7934 
7935 
7946 RD_EXPORT
7948  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
7949 
7950 
7958 RD_EXPORT
7960  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
7961 
7972 RD_EXPORT
7974  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
7975 
7983 RD_EXPORT
7985  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
7986 
7999 RD_EXPORT
8001  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8002  size_t idx);
8003 
8014 RD_EXPORT
8016  const rd_kafka_MemberDescription_t *member);
8017 
8028 RD_EXPORT
8030  const rd_kafka_MemberDescription_t *member);
8031 
8042 RD_EXPORT
8044  const rd_kafka_MemberDescription_t *member);
8045 
8056 RD_EXPORT
8057 const char *
8059 
8070 RD_EXPORT
8072  const rd_kafka_MemberDescription_t *member);
8073 
8084 RD_EXPORT
8086  const rd_kafka_MemberAssignment_t *assignment);
8087 
8099 typedef struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t;
8100 
8110 RD_EXPORT
8112 
8117 RD_EXPORT
8119 
8125 RD_EXPORT void
8127  size_t del_group_cnt);
8128 
8144 RD_EXPORT
8145 void rd_kafka_DeleteGroups(rd_kafka_t *rk,
8146  rd_kafka_DeleteGroup_t **del_groups,
8147  size_t del_group_cnt,
8148  const rd_kafka_AdminOptions_t *options,
8149  rd_kafka_queue_t *rkqu);
8150 
8151 
8152 
8153 /*
8154  * DeleteGroups result type and methods
8155  */
8156 
8165 RD_EXPORT const rd_kafka_group_result_t **rd_kafka_DeleteGroups_result_groups(
8166  const rd_kafka_DeleteGroups_result_t *result,
8167  size_t *cntp);
8168 
8179 typedef struct rd_kafka_ListConsumerGroupOffsets_s
8181 
8196  const char *group_id,
8197  const rd_kafka_topic_partition_list_t *partitions);
8198 
8204  rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets);
8205 
8212  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8213  size_t list_grpoffset_cnt);
8214 
8232 RD_EXPORT
8234  rd_kafka_t *rk,
8235  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8236  size_t list_grpoffsets_cnt,
8237  const rd_kafka_AdminOptions_t *options,
8238  rd_kafka_queue_t *rkqu);
8239 
8240 
8241 
8242 /*
8243  * ListConsumerGroupOffsets result type and methods
8244  */
8245 
8257 RD_EXPORT const rd_kafka_group_result_t **
8260  size_t *cntp);
8261 
8262 
8263 
8274 typedef struct rd_kafka_AlterConsumerGroupOffsets_s
8276 
8291  const char *group_id,
8292  const rd_kafka_topic_partition_list_t *partitions);
8293 
8299  rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets);
8300 
8307  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
8308  size_t alter_grpoffset_cnt);
8309 
8328 RD_EXPORT
8330  rd_kafka_t *rk,
8331  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
8332  size_t alter_grpoffsets_cnt,
8333  const rd_kafka_AdminOptions_t *options,
8334  rd_kafka_queue_t *rkqu);
8335 
8336 
8337 
8338 /*
8339  * AlterConsumerGroupOffsets result type and methods
8340  */
8341 
8353 RD_EXPORT const rd_kafka_group_result_t **
8356  size_t *cntp);
8357 
8358 
8359 
8370 typedef struct rd_kafka_DeleteConsumerGroupOffsets_s
8372 
8387  const char *group,
8388  const rd_kafka_topic_partition_list_t *partitions);
8389 
8395  rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets);
8396 
8403  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
8404  size_t del_grpoffset_cnt);
8405 
8424 RD_EXPORT
8426  rd_kafka_t *rk,
8427  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
8428  size_t del_grpoffsets_cnt,
8429  const rd_kafka_AdminOptions_t *options,
8430  rd_kafka_queue_t *rkqu);
8431 
8432 
8433 
8434 /*
8435  * DeleteConsumerGroupOffsets result type and methods
8436  */
8437 
8446 RD_EXPORT const rd_kafka_group_result_t **
8449  size_t *cntp);
8450 
8463 typedef struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t;
8464 
8470 
8474 RD_EXPORT const rd_kafka_error_t *
8475 rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres);
8476 
8477 
8485  1,
8494  9,
8496  10,
8498  11,
8500  12,
8501  RD_KAFKA_ACL_OPERATION__CNT
8503 
8507 RD_EXPORT const char *
8509 
8517  1,
8520  RD_KAFKA_ACL_PERMISSION_TYPE__CNT
8522 
8526 RD_EXPORT const char *rd_kafka_AclPermissionType_name(
8527  rd_kafka_AclPermissionType_t acl_permission_type);
8528 
8547 RD_EXPORT rd_kafka_AclBinding_t *
8549  const char *name,
8550  rd_kafka_ResourcePatternType_t resource_pattern_type,
8551  const char *principal,
8552  const char *host,
8553  rd_kafka_AclOperation_t operation,
8554  rd_kafka_AclPermissionType_t permission_type,
8555  char *errstr,
8556  size_t errstr_size);
8557 
8584  rd_kafka_ResourceType_t restype,
8585  const char *name,
8586  rd_kafka_ResourcePatternType_t resource_pattern_type,
8587  const char *principal,
8588  const char *host,
8589  rd_kafka_AclOperation_t operation,
8590  rd_kafka_AclPermissionType_t permission_type,
8591  char *errstr,
8592  size_t errstr_size);
8593 
8597 RD_EXPORT rd_kafka_ResourceType_t
8599 
8605 RD_EXPORT const char *
8607 
8613 RD_EXPORT const char *
8615 
8621 RD_EXPORT const char *
8623 
8627 RD_EXPORT rd_kafka_AclOperation_t
8629 
8635 
8641 
8645 RD_EXPORT const rd_kafka_error_t *
8647 
8648 
8654 
8655 
8661 RD_EXPORT void
8663  size_t acl_bindings_cnt);
8664 
8672 RD_EXPORT const rd_kafka_acl_result_t **
8674  size_t *cntp);
8675 
8692 RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk,
8693  rd_kafka_AclBinding_t **new_acls,
8694  size_t new_acls_cnt,
8695  const rd_kafka_AdminOptions_t *options,
8696  rd_kafka_queue_t *rkqu);
8697 
8711 RD_EXPORT const rd_kafka_AclBinding_t **
8713  size_t *cntp);
8714 
8729 RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk,
8730  rd_kafka_AclBindingFilter_t *acl_filter,
8731  const rd_kafka_AdminOptions_t *options,
8732  rd_kafka_queue_t *rkqu);
8733 
8740 typedef struct rd_kafka_DeleteAcls_result_response_s
8742 
8750 RD_EXPORT const rd_kafka_DeleteAcls_result_response_t **
8752  size_t *cntp);
8753 
8758 RD_EXPORT const rd_kafka_error_t *rd_kafka_DeleteAcls_result_response_error(
8759  const rd_kafka_DeleteAcls_result_response_t *result_response);
8760 
8761 
8768 RD_EXPORT const rd_kafka_AclBinding_t **
8770  const rd_kafka_DeleteAcls_result_response_t *result_response,
8771  size_t *matching_acls_cntp);
8772 
8789 RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk,
8790  rd_kafka_AclBindingFilter_t **del_acls,
8791  size_t del_acls_cnt,
8792  const rd_kafka_AdminOptions_t *options,
8793  rd_kafka_queue_t *rkqu);
8794 
8847 RD_EXPORT
8850  const char *token_value,
8851  int64_t md_lifetime_ms,
8852  const char *md_principal_name,
8853  const char **extensions,
8854  size_t extension_size,
8855  char *errstr,
8856  size_t errstr_size);
8857 
8878 RD_EXPORT
8880  const char *errstr);
8881 
9068 RD_EXPORT
9069 rd_kafka_error_t *rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms);
9070 
9071 
9072 
9117 RD_EXPORT
9118 rd_kafka_error_t *rd_kafka_begin_transaction(rd_kafka_t *rk);
9119 
9120 
9192 RD_EXPORT
9194  rd_kafka_t *rk,
9195  const rd_kafka_topic_partition_list_t *offsets,
9196  const rd_kafka_consumer_group_metadata_t *cgmetadata,
9197  int timeout_ms);
9198 
9199 
9267 RD_EXPORT
9268 rd_kafka_error_t *rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms);
9269 
9270 
9329 RD_EXPORT
9330 rd_kafka_error_t *rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms);
9331 
9332 
9335 /* @cond NO_DOC */
9336 #ifdef __cplusplus
9337 }
9338 #endif
9339 #endif /* _RDKAFKA_H_ */
9340 /* @endcond NO_DOC */
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_new_t *on_new, void *ic_opaque)
Append an on_new() interceptor.
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Murmur2 partitioner (Java compatible).
struct rd_kafka_Node_s rd_kafka_Node_t
Node (broker) information.
Definition: rdkafka.h:4937
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_DeleteConsumerGroupOffsets_result_groups(const rd_kafka_DeleteConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a DeleteConsumerGroupOffsets result.
RD_EXPORT void * rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt)
Get the rkt_opaque pointer that was set in the topic configuration with rd_kafka_topic_conf_set_opaqu...
RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk, rd_kafka_AclBinding_t **new_acls, size_t new_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create acls as specified by the new_acls array of size new_topic_cnt elements.
rd_kafka_resp_err_t
Error codes.
Definition: rdkafka.h:279
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_EXPIRED
Definition: rdkafka.h:565
@ RD_KAFKA_RESP_ERR_POLICY_VIOLATION
Definition: rdkafka.h:515
@ RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafka.h:536
@ RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
Definition: rdkafka.h:405
@ RD_KAFKA_RESP_ERR_RESOURCE_NOT_FOUND
Definition: rdkafka.h:618
@ RD_KAFKA_RESP_ERR__PURGE_QUEUE
Definition: rdkafka.h:381
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafka.h:513
@ RD_KAFKA_RESP_ERR_OPERATION_NOT_ATTEMPTED
Definition: rdkafka.h:542
@ RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafka.h:529
@ RD_KAFKA_RESP_ERR_INVALID_PRINCIPAL_TYPE
Definition: rdkafka.h:567
@ RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:593
@ RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION
Definition: rdkafka.h:305
@ RD_KAFKA_RESP_ERR__FS
Definition: rdkafka.h:307
@ RD_KAFKA_RESP_ERR__TIMED_OUT
Definition: rdkafka.h:315
@ RD_KAFKA_RESP_ERR_REQUEST_TIMED_OUT
Definition: rdkafka.h:432
@ RD_KAFKA_RESP_ERR_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:426
@ RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
Definition: rdkafka.h:469
@ RD_KAFKA_RESP_ERR__FATAL
Definition: rdkafka.h:385
@ RD_KAFKA_RESP_ERR__UNDERFLOW
Definition: rdkafka.h:375
@ RD_KAFKA_RESP_ERR__BEGIN
Definition: rdkafka.h:282
@ RD_KAFKA_RESP_ERR__NOENT
Definition: rdkafka.h:373
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_OR_PART
Definition: rdkafka.h:422
@ RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafka.h:463
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafka.h:521
@ RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
Definition: rdkafka.h:444
@ RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
Definition: rdkafka.h:544
@ RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
Definition: rdkafka.h:571
@ RD_KAFKA_RESP_ERR_TOPIC_ALREADY_EXISTS
Definition: rdkafka.h:499
@ RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
Definition: rdkafka.h:451
@ RD_KAFKA_RESP_ERR_MSG_SIZE_TOO_LARGE
Definition: rdkafka.h:438
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTH_DISABLED
Definition: rdkafka.h:555
@ RD_KAFKA_RESP_ERR_DUPLICATE_RESOURCE
Definition: rdkafka.h:620
@ RD_KAFKA_RESP_ERR_INVALID_CONFIG
Definition: rdkafka.h:507
@ RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
Definition: rdkafka.h:629
@ RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
Definition: rdkafka.h:391
@ RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
Definition: rdkafka.h:627
@ RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
Definition: rdkafka.h:327
@ RD_KAFKA_RESP_ERR__ISR_INSUFF
Definition: rdkafka.h:319
@ RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafka.h:549
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_ID_MAPPING
Definition: rdkafka.h:526
@ RD_KAFKA_RESP_ERR__IN_PROGRESS
Definition: rdkafka.h:329
@ RD_KAFKA_RESP_ERR__READ_ONLY
Definition: rdkafka.h:371
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafka.h:467
@ RD_KAFKA_RESP_ERR__UNKNOWN_BROKER
Definition: rdkafka.h:393
@ RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafka.h:473
@ RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
Definition: rdkafka.h:569
@ RD_KAFKA_RESP_ERR_SECURITY_DISABLED
Definition: rdkafka.h:540
@ RD_KAFKA_RESP_ERR_TOPIC_DELETION_DISABLED
Definition: rdkafka.h:579
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_OWNER_MISMATCH
Definition: rdkafka.h:559
@ RD_KAFKA_RESP_ERR__INVALID_ARG
Definition: rdkafka.h:313
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_VERSION
Definition: rdkafka.h:497
@ RD_KAFKA_RESP_ERR__OUTDATED
Definition: rdkafka.h:351
@ RD_KAFKA_RESP_ERR__FAIL
Definition: rdkafka.h:290
@ RD_KAFKA_RESP_ERR_INVALID_MSG_SIZE
Definition: rdkafka.h:424
@ RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
Definition: rdkafka.h:587
@ RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
Definition: rdkafka.h:446
@ RD_KAFKA_RESP_ERR__ASSIGN_PARTITIONS
Definition: rdkafka.h:335
@ RD_KAFKA_RESP_ERR__VALUE_SERIALIZATION
Definition: rdkafka.h:363
@ RD_KAFKA_RESP_ERR_FETCH_SESSION_ID_NOT_FOUND
Definition: rdkafka.h:573
@ RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
Definition: rdkafka.h:337
@ RD_KAFKA_RESP_ERR__APPLICATION
Definition: rdkafka.h:399
@ RD_KAFKA_RESP_ERR_REPLICA_NOT_AVAILABLE
Definition: rdkafka.h:436
@ RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
Definition: rdkafka.h:333
@ RD_KAFKA_RESP_ERR_NOT_CONTROLLER
Definition: rdkafka.h:509
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_NOT_FOUND
Definition: rdkafka.h:557
@ RD_KAFKA_RESP_ERR_LISTENER_NOT_FOUND
Definition: rdkafka.h:577
@ RD_KAFKA_RESP_ERR_INVALID_GROUP_ID
Definition: rdkafka.h:475
@ RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafka.h:611
@ RD_KAFKA_RESP_ERR__GAPLESS_GUARANTEE
Definition: rdkafka.h:389
@ RD_KAFKA_RESP_ERR__INVALID_TYPE
Definition: rdkafka.h:377
@ RD_KAFKA_RESP_ERR_OFFSET_NOT_AVAILABLE
Definition: rdkafka.h:589
@ RD_KAFKA_RESP_ERR__PURGE_INFLIGHT
Definition: rdkafka.h:383
@ RD_KAFKA_RESP_ERR__STATE
Definition: rdkafka.h:341
@ RD_KAFKA_RESP_ERR_UNKNOWN_PRODUCER_ID
Definition: rdkafka.h:551
@ RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafka.h:583
@ RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafka.h:622
@ RD_KAFKA_RESP_ERR_BROKER_NOT_AVAILABLE
Definition: rdkafka.h:434
@ RD_KAFKA_RESP_ERR_INCONSISTENT_VOTER_SET
Definition: rdkafka.h:625
@ RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafka.h:600
@ RD_KAFKA_RESP_ERR__RETRY
Definition: rdkafka.h:379
@ RD_KAFKA_RESP_ERR_INVALID_REPLICATION_FACTOR
Definition: rdkafka.h:503
@ RD_KAFKA_RESP_ERR_NOT_COORDINATOR
Definition: rdkafka.h:456
@ RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
Definition: rdkafka.h:331
@ RD_KAFKA_RESP_ERR_ILLEGAL_GENERATION
Definition: rdkafka.h:471
@ RD_KAFKA_RESP_ERR__BAD_MSG
Definition: rdkafka.h:284
@ RD_KAFKA_RESP_ERR__BAD_COMPRESSION
Definition: rdkafka.h:286
@ RD_KAFKA_RESP_ERR__QUEUE_FULL
Definition: rdkafka.h:317
@ RD_KAFKA_RESP_ERR__WAIT_CACHE
Definition: rdkafka.h:357
@ RD_KAFKA_RESP_ERR__RESOLVE
Definition: rdkafka.h:296
@ RD_KAFKA_RESP_ERR_PRODUCER_FENCED
Definition: rdkafka.h:616
@ RD_KAFKA_RESP_ERR__AUTHENTICATION
Definition: rdkafka.h:347
@ RD_KAFKA_RESP_ERR__NOT_IMPLEMENTED
Definition: rdkafka.h:345
@ RD_KAFKA_RESP_ERR_INVALID_RECORD
Definition: rdkafka.h:609
@ RD_KAFKA_RESP_ERR__UNKNOWN_PROTOCOL
Definition: rdkafka.h:343
@ RD_KAFKA_RESP_ERR__PARTIAL
Definition: rdkafka.h:369
@ RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
Definition: rdkafka.h:440
@ RD_KAFKA_RESP_ERR_CLUSTER_AUTHORIZATION_FAILED
Definition: rdkafka.h:489
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafka.h:563
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS
Definition: rdkafka.h:465
@ RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafka.h:595
@ RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
Definition: rdkafka.h:523
@ RD_KAFKA_RESP_ERR__DESTROY
Definition: rdkafka.h:288
@ RD_KAFKA_RESP_ERR__INCONSISTENT
Definition: rdkafka.h:387
@ RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
Definition: rdkafka.h:361
@ RD_KAFKA_RESP_ERR_UNKNOWN_MEMBER_ID
Definition: rdkafka.h:477
@ RD_KAFKA_RESP_ERR_MEMBER_ID_REQUIRED
Definition: rdkafka.h:591
@ RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafka.h:532
@ RD_KAFKA_RESP_ERR__TRANSPORT
Definition: rdkafka.h:292
@ RD_KAFKA_RESP_ERR_INVALID_TIMESTAMP
Definition: rdkafka.h:491
@ RD_KAFKA_RESP_ERR_INVALID_REQUEST
Definition: rdkafka.h:511
@ RD_KAFKA_RESP_ERR_OFFSET_OUT_OF_RANGE
Definition: rdkafka.h:418
@ RD_KAFKA_RESP_ERR__LOG_TRUNCATION
Definition: rdkafka.h:407
@ RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
Definition: rdkafka.h:598
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafka.h:561
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafka.h:493
@ RD_KAFKA_RESP_ERR__CONFLICT
Definition: rdkafka.h:339
@ RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:553
@ RD_KAFKA_RESP_ERR_INVALID_COMMIT_OFFSET_SIZE
Definition: rdkafka.h:483
@ RD_KAFKA_RESP_ERR__NOOP
Definition: rdkafka.h:403
@ RD_KAFKA_RESP_ERR_THROTTLING_QUOTA_EXCEEDED
Definition: rdkafka.h:613
@ RD_KAFKA_RESP_ERR__TIMED_OUT_QUEUE
Definition: rdkafka.h:353
@ RD_KAFKA_RESP_ERR_DUPLICATE_SEQUENCE_NUMBER
Definition: rdkafka.h:519
@ RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
Definition: rdkafka.h:602
@ RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafka.h:485
@ RD_KAFKA_RESP_ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
Definition: rdkafka.h:538
@ RD_KAFKA_RESP_ERR_GROUP_SUBSCRIBED_TO_TOPIC
Definition: rdkafka.h:607
@ RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
Definition: rdkafka.h:401
@ RD_KAFKA_RESP_ERR__MSG_TIMED_OUT
Definition: rdkafka.h:298
@ RD_KAFKA_RESP_ERR_INVALID_SESSION_TIMEOUT
Definition: rdkafka.h:479
@ RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC
Definition: rdkafka.h:309
@ RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafka.h:631
@ RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafka.h:442
@ RD_KAFKA_RESP_ERR__NODE_UPDATE
Definition: rdkafka.h:321
@ RD_KAFKA_RESP_ERR_NO_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:604
@ RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
Definition: rdkafka.h:294
@ RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafka.h:575
@ RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
Definition: rdkafka.h:367
@ RD_KAFKA_RESP_ERR__NO_OFFSET
Definition: rdkafka.h:349
@ RD_KAFKA_RESP_ERR_INVALID_MSG
Definition: rdkafka.h:420
@ RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
Definition: rdkafka.h:547
@ RD_KAFKA_RESP_ERR__ALL_BROKERS_DOWN
Definition: rdkafka.h:311
@ RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafka.h:505
@ RD_KAFKA_RESP_ERR__FENCED
Definition: rdkafka.h:397
@ RD_KAFKA_RESP_ERR_INVALID_PARTITIONS
Definition: rdkafka.h:501
@ RD_KAFKA_RESP_ERR__NOT_CONFIGURED
Definition: rdkafka.h:395
@ RD_KAFKA_RESP_ERR__UNSUPPORTED_FEATURE
Definition: rdkafka.h:355
@ RD_KAFKA_RESP_ERR_NO_ERROR
Definition: rdkafka.h:416
@ RD_KAFKA_RESP_ERR__END
Definition: rdkafka.h:410
@ RD_KAFKA_RESP_ERR__INTR
Definition: rdkafka.h:359
@ RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
Definition: rdkafka.h:495
@ RD_KAFKA_RESP_ERR_GROUP_AUTHORIZATION_FAILED
Definition: rdkafka.h:487
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_COMPRESSION_TYPE
Definition: rdkafka.h:585
@ RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafka.h:517
@ RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
Definition: rdkafka.h:365
@ RD_KAFKA_RESP_ERR__SSL
Definition: rdkafka.h:323
@ RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
Definition: rdkafka.h:481
@ RD_KAFKA_RESP_ERR_UNKNOWN
Definition: rdkafka.h:414
@ RD_KAFKA_RESP_ERR_FENCED_LEADER_EPOCH
Definition: rdkafka.h:581
@ RD_KAFKA_RESP_ERR__WAIT_COORD
Definition: rdkafka.h:325
@ RD_KAFKA_RESP_ERR__PARTITION_EOF
Definition: rdkafka.h:303
@ RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
Definition: rdkafka.h:461
RD_EXPORT void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf, void(*throttle_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int throttle_time_ms, void *opaque))
Set throttle callback.
RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic)
Destroy and free a NewTopic object previously created with rd_kafka_NewTopic_new()
RD_EXPORT void rd_kafka_conf_set_socket_cb(rd_kafka_conf_t *conf, int(*socket_cb)(int domain, int type, int protocol, void *opaque))
Set socket callback.
RD_EXPORT void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs, size_t *cntp)
Returns the full list of error codes.
RD_EXPORT const rd_kafka_error_t * rd_kafka_AclBinding_error(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_store(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets)
Store offsets for next auto-commit for one or more partitions.
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupListing_state(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets state for the grplist group.
RD_EXPORT void rd_kafka_DeleteRecords_destroy_array(rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt)
Helper function to destroy all DeleteRecords objects in the del_groups array (of del_group_cnt elemen...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Atomic assignment of partitions to consume.
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:6932
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:7150
RD_EXPORT void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf, void(*log_cb)(const rd_kafka_t *rk, int level, const char *fac, const char *buf))
Set logger callback.
RD_EXPORT void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf, void(*dr_cb)(rd_kafka_t *rk, void *payload, size_t len, rd_kafka_resp_err_t err, void *opaque, void *msg_opaque))
RD_EXPORT const char * rd_kafka_ConfigResource_error_string(const rd_kafka_ConfigResource_t *config)
RD_EXPORT void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf, void *callback_data)
Set callback_data for OpenSSL engine.
RD_EXPORT void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque)
Sets the application's opaque pointer that will be passed to callbacks.
struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t
Definition: rdkafka.h:7644
RD_EXPORT int rd_kafka_thread_cnt(void)
Retrieve the current number of threads in use by librdkafka.
RD_EXPORT const rd_kafka_CreateTopics_result_t * rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev)
Get CreateTopics result.
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_ListConsumerGroupOffsets_result_groups(const rd_kafka_ListConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a ListConsumerGroupOffsets result.
RD_EXPORT void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata)
Release metadata memory.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_request_sent_t(rd_kafka_t *rk, int sockfd, const char *brokername, int32_t brokerid, int16_t ApiKey, int16_t ApiVersion, int32_t CorrId, size_t size, void *ic_opaque)
on_request_sent() is called when a request has been fully written to a broker TCP connections socket.
Definition: rdkafka.h:6181
RD_EXPORT void rd_kafka_topic_partition_list_destroy(rd_kafka_topic_partition_list_t *rkparlist)
Free all resources used by the list and the list itself.
RD_EXPORT const char * rd_kafka_version_str(void)
Returns the librdkafka version as string.
RD_EXPORT const rd_kafka_error_t ** rd_kafka_ListConsumerGroups_result_errors(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of errors from a ListConsumerGroups call result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_commit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_commit_t *on_commit, void *ic_opaque)
Append an on_commit() interceptor.
rd_kafka_event_t rd_kafka_DeleteRecords_result_t
Definition: rdkafka.h:5616
RD_EXPORT const rd_kafka_DescribeConfigs_result_t * rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev)
Get DescribeConfigs result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_consume(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_consume_t *on_consume, void *ic_opaque)
Append an on_consume() interceptor.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscribe(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *topics)
Subscribe to topic set using balanced consumer groups.
struct rd_kafka_DeleteAcls_result_response_s rd_kafka_DeleteAcls_result_response_t
Definition: rdkafka.h:8740
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_partition_list_set_offset(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition, int64_t offset)
Set offset to offset for topic and partition.
RD_EXPORT rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assignment(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **partitions)
Returns the current partition assignment as set by rd_kafka_assign() or rd_kafka_incremental_assign()...
RD_EXPORT void rd_kafka_conf_set_rebalance_cb(rd_kafka_conf_t *conf, void(*rebalance_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *partitions, void *opaque))
Consumer: Set rebalance callback for use with coordinated consumer group balancing.
RD_EXPORT void rd_kafka_yield(rd_kafka_t *rk)
Cancels the current callback dispatcher (rd_kafka_poll(), rd_kafka_consume_callback(),...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_destroy(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_destroy_t *on_destroy, void *ic_opaque)
Append an on_destroy() interceptor.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_find(const rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Find element by topic and partition.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_add(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Add topic+partition to list.
RD_EXPORT void rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf, int(*resolve_cb)(const char *node, const char *service, const struct addrinfo *hints, struct addrinfo **res, void *opaque))
Set address resolution callback.
RD_EXPORT void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf, rd_kafka_topic_conf_t *tconf)
Sets the default topic configuration to use for automatically subscribed topics (e....
RD_EXPORT int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for events served through callbacks for max timeout_ms.
RD_EXPORT const rd_kafka_AclBinding_t ** rd_kafka_DeleteAcls_result_response_matching_acls(const rd_kafka_DeleteAcls_result_response_t *result_response, size_t *matching_acls_cntp)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf)
Creates a copy/duplicate of topic configuration object conf.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_dup_t(rd_kafka_conf_t *new_conf, const rd_kafka_conf_t *old_conf, size_t filter_cnt, const char **filter, void *ic_opaque)
on_conf_dup() is called from rd_kafka_conf_dup() in the order the interceptors were added and is used...
Definition: rdkafka.h:6003
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata(rd_kafka_t *rk)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token(rd_kafka_t *rk, const char *token_value, int64_t md_lifetime_ms, const char *md_principal_name, const char **extensions, size_t extension_size, char *errstr, size_t errstr_size)
Set SASL/OAUTHBEARER token and metadata.
RD_EXPORT size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev, const rd_kafka_message_t **rkmessages, size_t size)
Extacts size message(s) from the event into the pre-allocated array rkmessages.
RD_EXPORT int rd_kafka_ConfigEntry_is_synonym(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_new(void)
Create topic configuration object.
RD_EXPORT void rd_kafka_conf_set_offset_commit_cb(rd_kafka_conf_t *conf, void(*offset_commit_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *opaque))
Consumer: Set offset commit callback for use with consumer groups.
RD_EXPORT const char * rd_kafka_ConsumerGroupDescription_group_id(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the group id for the grpdesc group.
RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk, rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create additional partitions for the given topics, as specified by the new_parts array of size new_pa...
RD_EXPORT const rd_kafka_DescribeConsumerGroups_result_t * rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev)
Get DescribeConsumerGroups result.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_new_t(rd_kafka_t *rk, const rd_kafka_conf_t *conf, void *ic_opaque, char *errstr, size_t errstr_size)
on_new() is called from rd_kafka_new() prior toreturning the newly created client instance to the app...
Definition: rdkafka.h:6038
RD_EXPORT rd_kafka_DeleteTopic_t * rd_kafka_DeleteTopic_new(const char *topic)
Create a new DeleteTopic object. This object is later passed to rd_kafka_DeleteTopics().
RD_EXPORT const char * rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres)
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_DescribeConfigs_result_resources(const rd_kafka_DescribeConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeConfigs result.
RD_EXPORT int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_AclBindingFilter_t * rd_kafka_AclBindingFilter_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBindingFilter object. This object is later passed to rd_kafka_DescribeAcls() or rd_ka...
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_DeleteGroups_result_groups(const rd_kafka_DeleteGroups_result_t *result, size_t *cntp)
Get an array of group results from a DeleteGroups result.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy_array(rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffset_cnt)
Helper function to destroy all AlterConsumerGroupOffsets objects in the alter_grpoffsets array (of al...
rd_kafka_event_t rd_kafka_CreatePartitions_result_t
Definition: rdkafka.h:5610
RD_EXPORT void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf, void *rkt_opaque)
Sets the application's opaque pointer that will be passed to all topic callbacks as the rkt_opaque ar...
RD_EXPORT char * rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms)
Returns the ClusterId as reported in broker metadata.
RD_EXPORT rd_kafka_error_t * rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt)
Produce and send a single message to broker.
RD_EXPORT void rd_kafka_dump(FILE *fp, rd_kafka_t *rk)
Dumps rdkafka's internal state for handle rk to stream fp.
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_unassign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally remove partitions from the current assignment.
RD_EXPORT rd_kafka_error_t * rd_kafka_send_offsets_to_transaction(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, const rd_kafka_consumer_group_metadata_t *cgmetadata, int timeout_ms)
Sends a list of topic partition offsets to the consumer group coordinator for cgmetadata,...
RD_EXPORT const char * rd_kafka_ConsumerGroupDescription_partition_assignor(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the partition assignor for the grpdesc group.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_pause_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Pause producing or consumption for the provided list of partitions.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_config(rd_kafka_NewTopic_t *new_topic, const char *name, const char *value)
Set (broker-side) topic configuration name/value pair.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_broker_state_change(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change, void *ic_opaque)
Append an on_broker_state_change() interceptor.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_new(rd_kafka_t *rk)
Create a new message queue.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_background(rd_kafka_t *rk)
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf, size_t filter_cnt, const char **filter)
Same as rd_kafka_conf_dup() but with an array of property name prefixes to filter out (ignore) when c...
rd_kafka_event_t rd_kafka_DescribeConsumerGroups_result_t
Definition: rdkafka.h:5620
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreatePartitions_result_topics(const rd_kafka_CreatePartitions_result_t *result, size_t *cntp)
Get an array of topic results from a CreatePartitions result.
RD_EXPORT rd_kafka_error_t * rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms)
Aborts the ongoing transaction.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk, rd_kafka_resp_err_t err, const char *reason)
Trigger a fatal error for testing purposes.
RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding)
Destroy and free an AclBinding object previously created with rd_kafka_AclBinding_new()
RD_EXPORT rd_kafka_event_t * rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for an event for max timeout_ms.
RD_EXPORT void rd_kafka_error_destroy(rd_kafka_error_t *error)
Free and destroy an error object.
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new(const char *group_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
rd_kafka_event_t rd_kafka_DeleteTopics_result_t
Definition: rdkafka.h:5602
RD_EXPORT rd_kafka_AclBinding_t * rd_kafka_AclBinding_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBinding object. This object is later passed to rd_kafka_CreateAcls().
RD_EXPORT void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage)
Frees resources for rkmessage and hands ownership back to rdkafka.
RD_EXPORT const rd_kafka_DeleteConsumerGroupOffsets_result_t * rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get DeleteConsumerGroupOffsets result.
rd_kafka_event_t rd_kafka_ListConsumerGroupOffsets_result_t
Definition: rdkafka.h:5628
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_dup(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_dup_t *on_conf_dup, void *ic_opaque)
Append an on_conf_dup() interceptor.
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_require_stable_offsets(rd_kafka_AdminOptions_t *options, int true_or_false)
Whether broker should return stable offsets (transaction-committed).
RD_EXPORT rd_kafka_resp_err_t rd_kafka_consumer_close(rd_kafka_t *rk)
Close the consumer.
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk)
Enable SASL OAUTHBEARER refresh callbacks on the librdkafka background thread.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_committed(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Retrieve committed offsets for topics+partitions.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_send_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_send() is called from rd_kafka_produce*() (et.al) prior to the partitioner being called.
Definition: rdkafka.h:6078
RD_EXPORT const rd_kafka_ConsumerGroupListing_t ** rd_kafka_ListConsumerGroups_result_valid(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of valid list groups from a ListConsumerGroups result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs, size_t idx, const char **namep, const void **valuep, size_t *sizep)
Iterator for all headers.
RD_EXPORT const rd_kafka_message_t * rd_kafka_event_message_next(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_error_name(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally add partitions to the current assignment.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Alter committed offsets for a set of partitions in a consumer group. This will succeed at the partiti...
RD_EXPORT void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk, const char **groups, size_t groups_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe groups from cluster as specified by the groups array of size groups_cnt elements.
RD_EXPORT rd_kafka_error_t * rd_kafka_begin_transaction(rd_kafka_t *rk)
Begin a new transaction.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_unsubscribe(rd_kafka_t *rk)
Unsubscribe from the current subscription set.
RD_EXPORT void rd_kafka_log_print(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin (default) log sink: print to stderr.
RD_EXPORT ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume batch of messages from queue.
RD_EXPORT const char * rd_kafka_topic_name(const rd_kafka_topic_t *rkt)
Returns the topic name.
RD_EXPORT const char * rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres)
RD_EXPORT const char ** rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf, size_t *cntp)
Dump the topic configuration properties and values of conf to an array with "key",...
RD_EXPORT void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu, void(*event_cb)(rd_kafka_t *rk, void *qev_opaque), void *qev_opaque)
Enable callback event triggering for queue.
RD_EXPORT int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage, rd_kafka_timestamp_type_t *tstype)
Returns the message timestamp for a consumed message.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_request_sent(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_request_sent_t *on_request_sent, void *ic_opaque)
Append an on_request_sent() interceptor.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size)
Returns the first fatal error set on this client instance, or RD_KAFKA_RESP_ERR_NO_ERROR if no fatal ...
rd_kafka_event_t rd_kafka_CreateAcls_result_t
Definition: rdkafka.h:5604
RD_EXPORT rd_kafka_resp_err_t rd_kafka_query_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high, int timeout_ms)
Query broker for low (oldest/beginning) and high (newest/end) offsets for partition.
rd_kafka_event_t rd_kafka_CreateTopics_result_t
Definition: rdkafka.h:5600
RD_EXPORT int rd_kafka_consume_callback_queue(rd_kafka_queue_t *rkqu, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consume multiple messages from queue with callback.
RD_EXPORT void rd_kafka_conf_set_open_cb(rd_kafka_conf_t *conf, int(*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque))
Set open callback.
rd_kafka_event_t rd_kafka_DeleteAcls_result_t
Definition: rdkafka.h:5608
RD_EXPORT const char * rd_kafka_ResourcePatternType_name(rd_kafka_ResourcePatternType_t resource_pattern_type)
RD_EXPORT const char * rd_kafka_Node_host(const rd_kafka_Node_t *node)
Get the host of node.
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreateTopics_result_topics(const rd_kafka_CreateTopics_result_t *result, size_t *cntp)
Get an array of topic results from a CreateTopics result.
RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt)
Helper function to destroy all NewTopic objects in the new_topics array (of new_topic_cnt elements)....
rd_kafka_event_t rd_kafka_DescribeAcls_result_t
Definition: rdkafka.h:5606
RD_EXPORT const rd_kafka_DeleteTopics_result_t * rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev)
Get DeleteTopics result.
RD_EXPORT void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr)
Free pointer returned by librdkafka.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_broker_state_change_t(rd_kafka_t *rk, int32_t broker_id, const char *secproto, const char *name, int port, const char *state, void *ic_opaque)
on_broker_state_change() is called just after a broker has been created or its state has been changed...
Definition: rdkafka.h:6295
RD_EXPORT const rd_kafka_error_t * rd_kafka_DeleteAcls_result_response_error(const rd_kafka_DeleteAcls_result_response_t *result_response)
rd_kafka_AclOperation_t
Apache Kafka ACL operation types.
Definition: rdkafka.h:8482
@ RD_KAFKA_ACL_OPERATION_ALL
Definition: rdkafka.h:8486
@ RD_KAFKA_ACL_OPERATION_ALTER
Definition: rdkafka.h:8491
@ RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
Definition: rdkafka.h:8493
@ RD_KAFKA_ACL_OPERATION_IDEMPOTENT_WRITE
Definition: rdkafka.h:8499
@ RD_KAFKA_ACL_OPERATION_DELETE
Definition: rdkafka.h:8490
@ RD_KAFKA_ACL_OPERATION_WRITE
Definition: rdkafka.h:8488
@ RD_KAFKA_ACL_OPERATION_ANY
Definition: rdkafka.h:8484
@ RD_KAFKA_ACL_OPERATION_UNKNOWN
Definition: rdkafka.h:8483
@ RD_KAFKA_ACL_OPERATION_CREATE
Definition: rdkafka.h:8489
@ RD_KAFKA_ACL_OPERATION_ALTER_CONFIGS
Definition: rdkafka.h:8497
@ RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
Definition: rdkafka.h:8495
@ RD_KAFKA_ACL_OPERATION_READ
Definition: rdkafka.h:8487
@ RD_KAFKA_ACL_OPERATION_DESCRIBE
Definition: rdkafka.h:8492
RD_EXPORT ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume up to rkmessages_size from topic rkt and partition putting a pointer to each message in the a...
RD_EXPORT int rd_kafka_event_log(rd_kafka_event_t *rkev, const char **fac, const char **str, int *level)
Extract log message from the event.
RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk)
Check whether the consumer considers the current assignment to have been lost involuntarily....
RD_EXPORT void rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf, int(*connect_cb)(int sockfd, const struct sockaddr *addr, int addrlen, const char *id, void *opaque))
Set connect callback.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf, rd_kafka_cert_type_t cert_type, rd_kafka_cert_enc_t cert_enc, const void *buffer, size_t size, char *errstr, size_t errstr_size)
Set certificate/key cert_type from the cert_enc encoded memory at buffer of size bytes.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve topic configuration value for property name.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_sasl(rd_kafka_t *rk)
RD_EXPORT int rd_kafka_consume_callback(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consumes messages from topic rkt and partition, calling the provided callback for each consumed messs...
RD_EXPORT const char * rd_kafka_rebalance_protocol(rd_kafka_t *rk)
The rebalance protocol currently in use. This will be "NONE" if the consumer has not (yet) joined a g...
RD_EXPORT int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Random partitioner.
RD_EXPORT const char * rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage)
Returns the error string for an errored rd_kafka_message_t or NULL if there was no error.
RD_EXPORT void rd_kafka_conf_set_stats_cb(rd_kafka_conf_t *conf, int(*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque))
Set statistics callback in provided conf object.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags)
Purge messages currently handled by the producer instance.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token_failure(rd_kafka_t *rk, const char *errstr)
SASL/OAUTHBEARER token refresh failure indicator.
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres)
RD_EXPORT int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev, char *dst, size_t dstsize)
Extract log debug context from event.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_set_log_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Forward librdkafka logs (and debug) to the specified queue for serving with one of the ....
RD_EXPORT void rd_kafka_conf_set_background_event_cb(rd_kafka_conf_t *conf, void(*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque))
Generic event callback to be used with the event API to trigger callbacks for rd_kafka_event_t object...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get(const rd_kafka_headers_t *hdrs, size_t idx, const char *name, const void **valuep, size_t *sizep)
Iterator for headers matching name.
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_consumer_group_state_code(const char *name)
Returns a code for a state name.
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_MemberAssignment_partitions(const rd_kafka_MemberAssignment_t *assignment)
Gets assigned partitions of a member assignment.
RD_EXPORT void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst)
Forward/re-route queue src to dst. If dst is NULL the forwarding is removed.
RD_EXPORT const rd_kafka_ConfigEntry_t ** rd_kafka_ConfigResource_configs(const rd_kafka_ConfigResource_t *config, size_t *cntp)
Get an array of config entries from a ConfigResource object.
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy_array(rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffset_cnt)
Helper function to destroy all DeleteConsumerGroupOffsets objects in the del_grpoffsets array (of del...
rd_kafka_cert_type_t
SSL certificate type.
Definition: rdkafka.h:2385
@ RD_KAFKA_CERT_PUBLIC_KEY
Definition: rdkafka.h:2386
@ RD_KAFKA_CERT_PRIVATE_KEY
Definition: rdkafka.h:2387
@ RD_KAFKA_CERT_CA
Definition: rdkafka.h:2388
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_copy(const rd_kafka_headers_t *src)
Make a copy of headers list src.
RD_EXPORT const char * rd_kafka_event_name(const rd_kafka_event_t *rkev)
RD_EXPORT void * rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size)
Allocate memory using the same allocator librdkafka uses.
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy(rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets)
Destroy and free a DeleteConsumerGroupOffsets object previously created with rd_kafka_DeleteConsumerG...
RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t **del_acls, size_t del_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete acls matching the filteres provided in del_acls array of size del_acls_cnt.
RD_EXPORT size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf)
Gets the default topic configuration as previously set with rd_kafka_conf_set_default_topic_conf() or...
RD_EXPORT void rd_kafka_topic_partition_list_add_range(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t start, int32_t stop)
Add range of partitions from start to stop inclusive.
RD_EXPORT rd_kafka_error_t * rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms)
Commit the current transaction (as started with rd_kafka_begin_transaction()).
RD_EXPORT const rd_kafka_Node_t * rd_kafka_ConsumerGroupDescription_coordinator(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the coordinator for the grpdesc group.
RD_EXPORT rd_kafka_t * rd_kafka_new(rd_kafka_type_t type, rd_kafka_conf_t *conf, char *errstr, size_t errstr_size)
Creates a new Kafka handle and starts its operation according to the specified type (RD_KAFKA_CONSUME...
struct rd_kafka_ListConsumerGroupsResult_s rd_kafka_ListConsumerGroupsResult_t
Definition: rdkafka.h:7742
RD_EXPORT const char * rd_kafka_ConsumerGroupListing_group_id(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets the group id for the grplist group.
RD_EXPORT rd_kafka_message_t * rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms)
Poll the consumer for messages or events.
RD_EXPORT void rd_kafka_conf_set_consume_cb(rd_kafka_conf_t *conf, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque))
Consumer: Set consume callback for use with rd_kafka_consumer_poll()
RD_EXPORT rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, int timeout_ms)
Seek consumer for topic+partition to offset which is either an absolute or logical offset.
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_copy(const rd_kafka_topic_partition_list_t *src)
Make a copy of an existing list.
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new_with_genid(const char *group_id, int32_t generation_id, const char *member_id, const char *group_instance_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
RD_EXPORT void rd_kafka_DeleteRecords_destroy(rd_kafka_DeleteRecords_t *del_records)
Destroy and free a DeleteRecords object previously created with rd_kafka_DeleteRecords_new()
RD_EXPORT rd_kafka_AdminOptions_t * rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api)
Create a new AdminOptions object.
RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk, rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create topics in cluster as specified by the new_topics array of size new_topic_cnt elements.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_error(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewPartitions_set_replica_assignment(rd_kafka_NewPartitions_t *new_parts, int32_t new_partition_idx, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker id) assignment for new_partition_idx to the replica set in broker_ids (of bro...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error)
rd_kafka_conf_res_t() rd_kafka_interceptor_f_on_conf_set_t(rd_kafka_conf_t *conf, const char *name, const char *val, char *errstr, size_t errstr_size, void *ic_opaque)
on_conf_set() is called from rd_kafka_*_conf_set() in the order the interceptors were added.
Definition: rdkafka.h:5973
RD_EXPORT rd_kafka_DeleteGroup_t * rd_kafka_DeleteGroup_new(const char *group)
Create a new DeleteGroup object. This object is later passed to rd_kafka_DeleteGroups().
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs, const char *name, const void **valuep, size_t *sizep)
Find last header in list hdrs matching name.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_list_groups(rd_kafka_t *rk, const char *group, const struct rd_kafka_group_list **grplistp, int timeout_ms)
List and describe client groups in cluster.
rd_kafka_resp_err_t() rd_kafka_plugin_f_conf_init_t(rd_kafka_conf_t *conf, void **plug_opaquep, char *errstr, size_t errstr_size)
Plugin's configuration initializer method called each time the library is referenced from configurati...
Definition: rdkafka.h:5887
RD_EXPORT rd_kafka_resp_err_t rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Retrieve current positions (offsets) for topics+partitions.
RD_EXPORT const char ** rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp)
Dump the configuration properties and values of conf to an array with "key", "value" pairs.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_operation_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the broker's operation timeout, such as the timeout for CreateTopics to complete the creation of...
RD_EXPORT const rd_kafka_ConsumerGroupDescription_t ** rd_kafka_DescribeConsumerGroups_result_groups(const rd_kafka_DescribeConsumerGroups_result_t *result, size_t *cntp)
Get an array of group results from a DescribeConsumerGroups result.
RD_EXPORT const rd_kafka_conf_t * rd_kafka_conf(rd_kafka_t *rk)
RD_EXPORT void rd_kafka_DeleteGroup_destroy(rd_kafka_DeleteGroup_t *del_group)
Destroy and free a DeleteGroup object previously created with rd_kafka_DeleteGroup_new()
RD_EXPORT void * rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size)
Allocate and zero memory using the same allocator librdkafka uses.
RD_EXPORT const char * rd_kafka_ConfigEntry_value(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT const char * rd_kafka_MemberDescription_host(const rd_kafka_MemberDescription_t *member)
Gets host of member.
RD_EXPORT void rd_kafka_log_syslog(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin log sink: print to syslog.
struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t
Definition: rdkafka.h:7073
RD_EXPORT void rd_kafka_topic_conf_set_msg_order_cmp(rd_kafka_topic_conf_t *topic_conf, int(*msg_order_cmp)(const rd_kafka_message_t *a, const rd_kafka_message_t *b))
Producer: Set message queueing order comparator callback.
RD_EXPORT rd_kafka_error_t * rd_kafka_seek_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Seek consumer for partitions in partitions to the per-partition offset in the .offset field of partit...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Store offset offset + 1 for topic rkt partition partition.
RD_EXPORT void rd_kafka_DeleteGroup_destroy_array(rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt)
Helper function to destroy all DeleteGroup objects in the del_groups array (of del_group_cnt elements...
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_AlterConfigs_result_resources(const rd_kafka_AlterConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a AlterConfigs result.
RD_EXPORT int rd_kafka_produce_batch(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, rd_kafka_message_t *rkmessages, int message_cnt)
Produce multiple messages.
rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t
Definition: rdkafka.h:5618
RD_EXPORT const rd_kafka_MemberDescription_t * rd_kafka_ConsumerGroupDescription_member(const rd_kafka_ConsumerGroupDescription_t *grpdesc, size_t idx)
Gets a member of grpdesc group.
RD_EXPORT rd_kafka_ResourcePatternType_t rd_kafka_AclBinding_resource_pattern_type(const rd_kafka_AclBinding_t *acl)
RD_EXPORT int rd_kafka_ConfigEntry_is_sensitive(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy(rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets)
Destroy and free a ListConsumerGroupOffsets object previously created with rd_kafka_ListConsumerGroup...
RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t *acl_filter, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe acls matching the filter provided in acl_filter.
RD_EXPORT const char * rd_kafka_event_config_string(rd_kafka_event_t *rkev)
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_AlterConsumerGroupOffsets_result_groups(const rd_kafka_AlterConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a AlterConsumerGroupOffsets result.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_main(rd_kafka_t *rk)
RD_EXPORT const char * rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres)
rd_kafka_admin_op_t
Admin operation enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:6697
@ RD_KAFKA_ADMIN_OP_DELETERECORDS
Definition: rdkafka.h:6704
@ RD_KAFKA_ADMIN_OP_CREATETOPICS
Definition: rdkafka.h:6699
@ RD_KAFKA_ADMIN_OP_CREATEACLS
Definition: rdkafka.h:6708
@ RD_KAFKA_ADMIN_OP_DESCRIBEACLS
Definition: rdkafka.h:6709
@ RD_KAFKA_ADMIN_OP_DESCRIBECONSUMERGROUPS
Definition: rdkafka.h:6712
@ RD_KAFKA_ADMIN_OP_DELETEGROUPS
Definition: rdkafka.h:6705
@ RD_KAFKA_ADMIN_OP_ANY
Definition: rdkafka.h:6698
@ RD_KAFKA_ADMIN_OP_DELETEACLS
Definition: rdkafka.h:6710
@ RD_KAFKA_ADMIN_OP_DELETECONSUMERGROUPOFFSETS
Definition: rdkafka.h:6707
@ RD_KAFKA_ADMIN_OP_DELETETOPICS
Definition: rdkafka.h:6700
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6714
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPS
Definition: rdkafka.h:6711
@ RD_KAFKA_ADMIN_OP__CNT
Definition: rdkafka.h:6717
@ RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6716
@ RD_KAFKA_ADMIN_OP_ALTERCONFIGS
Definition: rdkafka.h:6702
@ RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
Definition: rdkafka.h:6701
@ RD_KAFKA_ADMIN_OP_DESCRIBECONFIGS
Definition: rdkafka.h:6703
struct rd_kafka_AlterConsumerGroupOffsets_s rd_kafka_AlterConsumerGroupOffsets_t
Definition: rdkafka.h:8274
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_response_received(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_response_received_t *on_response_received, void *ic_opaque)
Append an on_response_received() interceptor.
RD_EXPORT const rd_kafka_error_t * rd_kafka_ConsumerGroupDescription_error(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the error for the grpdesc group.
RD_EXPORT void rd_kafka_NewPartitions_destroy_array(rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt)
Helper function to destroy all NewPartitions objects in the new_parts array (of new_parts_cnt element...
RD_EXPORT int rd_kafka_version(void)
Returns the librdkafka version as integer.
RD_EXPORT int rd_kafka_topic_partition_list_del_by_idx(rd_kafka_topic_partition_list_t *rktparlist, int idx)
Delete partition from list by elems[] index.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_metadata(rd_kafka_t *rk, int all_topics, rd_kafka_topic_t *only_rkt, const struct rd_kafka_metadata **metadatap, int timeout_ms)
Request Metadata from broker.
RD_EXPORT char * rd_kafka_memberid(const rd_kafka_t *rk)
Returns this client's broker-assigned group member id.
RD_EXPORT rd_kafka_error_t * rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage)
Store offset +1 for the consumed message.
RD_EXPORT void * rd_kafka_event_opaque(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_MemberDescription_consumer_id(const rd_kafka_MemberDescription_t *member)
Gets consumer id of member.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_exit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_exit_t *on_thread_exit, void *ic_opaque)
Append an on_thread_exit() interceptor.
RD_EXPORT int rd_kafka_consumer_closed(rd_kafka_t *rk)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_commit_t(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, rd_kafka_resp_err_t err, void *ic_opaque)
on_commit() is called on completed or failed offset commit. It is called from internal librdkafka thr...
Definition: rdkafka.h:6153
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs, const char *name)
Remove all headers for the given key (if any).
RD_EXPORT const rd_kafka_ConfigEntry_t ** rd_kafka_ConfigEntry_synonyms(const rd_kafka_ConfigEntry_t *entry, size_t *cntp)
RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void)
Returns the thread-local system errno.
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_default_topic_conf_dup(rd_kafka_t *rk)
Creates a copy/duplicate of rk 's default topic configuration object.
RD_EXPORT rd_kafka_DeleteConsumerGroupOffsets_t * rd_kafka_DeleteConsumerGroupOffsets_new(const char *group, const rd_kafka_topic_partition_list_t *partitions)
Create a new DeleteConsumerGroupOffsets object. This object is later passed to rd_kafka_DeleteConsume...
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_set_credentials(rd_kafka_t *rk, const char *username, const char *password)
Sets SASL credentials used for SASL PLAIN and SCRAM mechanisms by this Kafka client.
RD_EXPORT const char * rd_kafka_AclBinding_host(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup(const rd_kafka_conf_t *conf)
Creates a copy/duplicate of configuration object conf.
RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic)
Destroy and free a DeleteTopic object previously created with rd_kafka_DeleteTopic_new()
RD_EXPORT const char * rd_kafka_err2name(rd_kafka_resp_err_t err)
Returns the error code name (enum name).
rd_kafka_consumer_group_state_t
Consumer group state.
Definition: rdkafka.h:5008
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_response_received_t(rd_kafka_t *rk, int sockfd, const char *brokername, int32_t brokerid, int16_t ApiKey, int16_t ApiVersion, int32_t CorrId, size_t size, int64_t rtt, rd_kafka_resp_err_t err, void *ic_opaque)
on_response_received() is called when a protocol response has been fully received from a broker TCP c...
Definition: rdkafka.h:6218
RD_EXPORT int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, rd_kafka_queue_t *rkqu)
Same as rd_kafka_consume_start() but re-routes incoming messages to the provided queue rkqu (which mu...
int rd_kafka_event_type_t
Event types.
Definition: rdkafka.h:5329
RD_EXPORT size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu)
struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t
ListConsumerGroups result for a single group.
Definition: rdkafka.h:7739
RD_EXPORT void rd_kafka_conf_dump_free(const char **arr, size_t cnt)
Frees a configuration dump returned from rd_kafka_conf_dump() or `rd_kafka_topic_conf_dump().
RD_EXPORT rd_kafka_message_t * rd_kafka_consume_queue(rd_kafka_queue_t *rkqu, int timeout_ms)
Consume from queue.
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_match_consumer_group_states(rd_kafka_AdminOptions_t *options, const rd_kafka_consumer_group_state_t *consumer_group_states, size_t consumer_group_states_cnt)
Set consumer groups states to query for.
struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t
Definition: rdkafka.h:7313
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_destroy(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_destroy_t *on_conf_destroy, void *ic_opaque)
Append an on_conf_destroy() interceptor.
RD_EXPORT void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt)
Loose application's topic handle refcount as previously created with rd_kafka_topic_new().
RD_EXPORT void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu)
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy_array(rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffset_cnt)
Helper function to destroy all ListConsumerGroupOffsets objects in the list_grpoffsets array (of list...
struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t
AdminOptions provides a generic mechanism for setting optional parameters for the Admin API requests.
Definition: rdkafka.h:6732
RD_EXPORT int rd_kafka_topic_partition_list_del(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Delete partition from list.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres)
Topic result provides per-topic operation result information.
RD_EXPORT int rd_kafka_error_is_fatal(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_set(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_set_t *on_conf_set, void *ic_opaque)
Append an on_conf_set() interceptor.
RD_EXPORT const rd_kafka_error_t * rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres)
Group result provides per-group operation result information.
RD_EXPORT const rd_kafka_AlterConfigs_result_t * rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev)
Get AlterConfigs result.
RD_EXPORT void rd_kafka_conf_set_oauthbearer_token_refresh_cb(rd_kafka_conf_t *conf, void(*oauthbearer_token_refresh_cb)(rd_kafka_t *rk, const char *oauthbearer_config, void *opaque))
Set SASL/OAUTHBEARER token refresh callback in provided conf object.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy(rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets)
Destroy and free a AlterConsumerGroupOffsets object previously created with rd_kafka_AlterConsumerGro...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_request_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the overall request timeout, including broker lookup, request transmission, operation time on br...
RD_EXPORT void rd_kafka_message_set_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t *hdrs)
Replace the message's current headers with a new list.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_message(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, int async)
Commit message's offset on broker for the message's partition. The committed offset is the message's ...
rd_kafka_vtype_t
Var-arg tag types.
Definition: rdkafka.h:1118
@ RD_KAFKA_VTYPE_PARTITION
Definition: rdkafka.h:1122
@ RD_KAFKA_VTYPE_HEADERS
Definition: rdkafka.h:1134
@ RD_KAFKA_VTYPE_TOPIC
Definition: rdkafka.h:1120
@ RD_KAFKA_VTYPE_MSGFLAGS
Definition: rdkafka.h:1130
@ RD_KAFKA_VTYPE_RKT
Definition: rdkafka.h:1121
@ RD_KAFKA_VTYPE_HEADER
Definition: rdkafka.h:1132
@ RD_KAFKA_VTYPE_KEY
Definition: rdkafka.h:1124
@ RD_KAFKA_VTYPE_END
Definition: rdkafka.h:1119
@ RD_KAFKA_VTYPE_OPAQUE
Definition: rdkafka.h:1125
@ RD_KAFKA_VTYPE_VALUE
Definition: rdkafka.h:1123
@ RD_KAFKA_VTYPE_TIMESTAMP
Definition: rdkafka.h:1131
struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t
Definition: rdkafka.h:7426
RD_EXPORT const char * rd_kafka_name(const rd_kafka_t *rk)
Returns Kafka handle name.
RD_EXPORT const rd_kafka_DeleteAcls_result_t * rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev)
struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t
Definition: rdkafka.h:8099
RD_EXPORT rd_kafka_resp_err_t rd_kafka_poll_set_consumer(rd_kafka_t *rk)
Redirect the main (rd_kafka_poll()) queue to the KafkaConsumer's queue (rd_kafka_consumer_poll()).
RD_EXPORT const rd_kafka_MemberAssignment_t * rd_kafka_MemberDescription_assignment(const rd_kafka_MemberDescription_t *member)
Gets assignment of member.
RD_EXPORT void rd_kafka_AdminOptions_set_opaque(rd_kafka_AdminOptions_t *options, void *ev_opaque)
Set application opaque value that can be extracted from the result event using rd_kafka_event_opaque(...
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_close_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Asynchronously close the consumer.
RD_EXPORT const char * rd_kafka_AclPermissionType_name(rd_kafka_AclPermissionType_t acl_permission_type)
RD_EXPORT RD_DEPRECATED void rd_kafka_set_logger(rd_kafka_t *rk, void(*func)(const rd_kafka_t *rk, int level, const char *fac, const char *buf))
Set logger function.
RD_EXPORT rd_kafka_NewTopic_t * rd_kafka_NewTopic_new(const char *topic, int num_partitions, int replication_factor, char *errstr, size_t errstr_size)
Create a new NewTopic object. This object is later passed to rd_kafka_CreateTopics().
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_new(size_t initial_count)
Create a new headers list.
RD_EXPORT rd_kafka_AlterConsumerGroupOffsets_t * rd_kafka_AlterConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new AlterConsumerGroupOffsets object. This object is later passed to rd_kafka_AlterConsumerG...
RD_EXPORT int rd_kafka_ConsumerGroupDescription_is_simple_consumer_group(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Is the grpdesc group a simple consumer group.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev)
RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs)
Destroy the headers list. The object and any returned value pointers are not usable after this call.
RD_EXPORT int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage)
Returns the broker id of the broker the message was produced to or fetched from.
rd_kafka_event_t rd_kafka_AlterConsumerGroupOffsets_result_t
Definition: rdkafka.h:5626
RD_EXPORT void rd_kafka_ConfigResource_destroy_array(rd_kafka_ConfigResource_t **config, size_t config_cnt)
Helper function to destroy all ConfigResource objects in the configs array (of config_cnt elements)....
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_start(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_start_t *on_thread_start, void *ic_opaque)
Append an on_thread_start() interceptor.
RD_EXPORT rd_kafka_message_t * rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms)
Consume a single message from topic rkt and partition.
RD_EXPORT size_t rd_kafka_ConsumerGroupDescription_member_count(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the members count of grpdesc group.
RD_EXPORT void rd_kafka_conf_properties_show(FILE *fp)
Prints a table to fp of all supported configuration properties, their default values as well as a des...
RD_EXPORT const rd_kafka_AclBinding_t ** rd_kafka_DescribeAcls_result_acls(const rd_kafka_DescribeAcls_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeAcls result.
RD_EXPORT const rd_kafka_CreateAcls_result_t * rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_acknowledgement_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_acknowledgement() is called to inform interceptors that a message was succesfully delivered or per...
Definition: rdkafka.h:6105
struct rd_kafka_DeleteConsumerGroupOffsets_s rd_kafka_DeleteConsumerGroupOffsets_t
Definition: rdkafka.h:8370
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_new(void)
Create configuration object.
RD_EXPORT int rd_kafka_ConfigEntry_is_read_only(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_thread_start_t(rd_kafka_t *rk, rd_kafka_thread_type_t thread_type, const char *thread_name, void *ic_opaque)
on_thread_start() is called from a newly created librdkafka-managed thread.
Definition: rdkafka.h:6248
RD_EXPORT int rd_kafka_wait_destroyed(int timeout_ms)
Wait for all rd_kafka_t objects to be destroyed.
RD_EXPORT const char * rd_kafka_consumer_group_state_name(rd_kafka_consumer_group_state_t state)
Returns a name for a state code.
rd_kafka_ConfigSource_t
Apache Kafka config sources.
Definition: rdkafka.h:7282
@ RD_KAFKA_CONFIG_SOURCE_UNKNOWN_CONFIG
Definition: rdkafka.h:7285
@ RD_KAFKA_CONFIG_SOURCE__CNT
Definition: rdkafka.h:7301
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
Definition: rdkafka.h:7289
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_TOPIC_CONFIG
Definition: rdkafka.h:7287
@ RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
Definition: rdkafka.h:7298
@ RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
Definition: rdkafka.h:7295
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_DEFAULT_BROKER_CONFIG
Definition: rdkafka.h:7292
RD_EXPORT const rd_kafka_AlterConsumerGroupOffsets_result_t * rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get AlterConsumerGroupOffsets result.
RD_EXPORT int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent partitioner.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_headers(const rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list.
RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk, rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete records (messages) in topic partitions older than the offsets provided.
RD_EXPORT rd_kafka_AclPermissionType_t rd_kafka_AclBinding_permission_type(const rd_kafka_AclBinding_t *acl)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_destroy_t(rd_kafka_t *rk, void *ic_opaque)
on_destroy() is called from rd_kafka_destroy() or (rd_kafka_new() if rd_kafka_new() fails during init...
Definition: rdkafka.h:6054
RD_EXPORT rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms)
Wait until all outstanding produce requests, et.al, are completed. This should typically be done prio...
RD_EXPORT int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_topic_t * rd_kafka_topic_new(rd_kafka_t *rk, const char *topic, rd_kafka_topic_conf_t *conf)
Creates a new topic handle for topic named topic.
RD_EXPORT void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist)
Release list memory.
rd_kafka_AclPermissionType_t
Apache Kafka ACL permission types.
Definition: rdkafka.h:8514
@ RD_KAFKA_ACL_PERMISSION_TYPE_DENY
Definition: rdkafka.h:8518
@ RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
Definition: rdkafka.h:8519
@ RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
Definition: rdkafka.h:8515
@ RD_KAFKA_ACL_PERMISSION_TYPE_ANY
Definition: rdkafka.h:8516
rd_kafka_thread_type_t
librdkafka internal thread type.
Definition: rdkafka.h:5266
@ RD_KAFKA_THREAD_MAIN
Definition: rdkafka.h:5267
@ RD_KAFKA_THREAD_BACKGROUND
Definition: rdkafka.h:5268
@ RD_KAFKA_THREAD_BROKER
Definition: rdkafka.h:5269
RD_EXPORT int rd_kafka_ConsumerGroupListing_is_simple_consumer_group(const rd_kafka_ConsumerGroupListing_t *grplist)
Is the grplist group a simple consumer group.
RD_EXPORT void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags)
Destroy Kafka handle according to specified destroy flags.
RD_EXPORT rd_kafka_AclOperation_t rd_kafka_AclBinding_operation(const rd_kafka_AclBinding_t *acl)
RD_EXPORT const rd_kafka_CreatePartitions_result_t * rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev)
Get CreatePartitions result.
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List committed offsets for a set of partitions in a consumer group.
RD_EXPORT void rd_kafka_conf_set_closesocket_cb(rd_kafka_conf_t *conf, int(*closesocket_cb)(int sockfd, void *opaque))
Set close socket callback.
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random Murmur2 partitioner (Java compatible).
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics)
Returns the current topic subscription.
RD_EXPORT rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk)
Returns Kafka handle type.
RD_EXPORT void rd_kafka_topic_partition_set_leader_epoch(rd_kafka_topic_partition_t *rktpar, int32_t leader_epoch)
Sets the offset leader epoch (use -1 to clear).
rd_kafka_cert_enc_t
SSL certificate encoding.
Definition: rdkafka.h:2399
@ RD_KAFKA_CERT_ENC_PEM
Definition: rdkafka.h:2402
@ RD_KAFKA_CERT_ENC_PKCS12
Definition: rdkafka.h:2400
@ RD_KAFKA_CERT_ENC_DER
Definition: rdkafka.h:2401
RD_EXPORT const char * rd_kafka_err2str(rd_kafka_resp_err_t err)
Returns a human readable representation of a kafka error.
RD_EXPORT int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist)
Adds one or more brokers to the kafka handle's list of initial bootstrap brokers.
RD_EXPORT int rd_kafka_unittest(void)
Run librdkafka's built-in unit-tests.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, int async)
Commit offsets on broker for the provided list of partitions.
RD_EXPORT int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage)
Returns the latency for a produced message measured from the produce() call.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_acknowledgement(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement, void *ic_opaque)
Append an on_acknowledgement() interceptor.
rd_kafka_event_t rd_kafka_DeleteGroups_result_t
Definition: rdkafka.h:5622
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a configuration property.
rd_kafka_event_t rd_kafka_DescribeConfigs_result_t
Definition: rdkafka.h:5614
RD_EXPORT uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node)
Get the port of node.
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev)
RD_EXPORT int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms)
Returns the current ControllerId as reported in broker metadata.
RD_EXPORT int rd_kafka_ConfigEntry_is_default(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT rd_kafka_NewPartitions_t * rd_kafka_NewPartitions_new(const char *topic, size_t new_total_cnt, char *errstr, size_t errstr_size)
Create a new NewPartitions. This object is later passed to rd_kafka_CreatePartitions() to increase th...
RD_EXPORT void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events)
Enable event sourcing. events is a bitmask of RD_KAFKA_EVENT_* of events to enable for consumption by...
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve configuration value for property name.
RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox)
Converts the system errno value errnox to a rd_kafka_resp_err_t error code upon failure from the foll...
RD_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
FNV-1a partitioner.
RD_EXPORT rd_kafka_error_t * rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms)
Initialize transactions for the producer instance.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_event_topic_partition(rd_kafka_event_t *rkev)
rd_kafka_event_t rd_kafka_DeleteConsumerGroupOffsets_result_t
Definition: rdkafka.h:5624
RD_EXPORT void rd_kafka_topic_conf_set_partitioner_cb(rd_kafka_topic_conf_t *topic_conf, int32_t(*partitioner)(const rd_kafka_topic_t *rkt, const void *keydata, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque))
Producer: Set partitioner callback in provided topic conf object.
RD_EXPORT const rd_kafka_acl_result_t ** rd_kafka_CreateAcls_result_acls(const rd_kafka_CreateAcls_result_t *result, size_t *cntp)
Get an array of acl results from a CreateAcls result.
RD_EXPORT const char * rd_kafka_ResourceType_name(rd_kafka_ResourceType_t restype)
RD_EXPORT void rd_kafka_destroy(rd_kafka_t *rk)
Destroy Kafka handle.
RD_EXPORT int rd_kafka_Node_id(const rd_kafka_Node_t *node)
Get the id of node.
RD_EXPORT void rd_kafka_conf_set_dr_msg_cb(rd_kafka_conf_t *conf, void(*dr_msg_cb)(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, void *opaque))
Producer: Set delivery report callback in provided conf object.
RD_EXPORT const rd_kafka_ListConsumerGroupOffsets_result_t * rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get ListConsumerGroupOffsets result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk,...)
Produce and send a single message to broker.
RD_EXPORT void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf)
Destroys a topic conf object.
RD_EXPORT const char * rd_kafka_MemberDescription_group_instance_id(const rd_kafka_MemberDescription_t *member)
Gets group instance id of member.
RD_EXPORT void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar)
Destroy a rd_kafka_topic_partition_t.
RD_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random FNV-1a partitioner.
RD_EXPORT void rd_kafka_DescribeConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Get configuration for the specified resources in configs.
rd_kafka_type_t
rd_kafka_t handle type.
Definition: rdkafka.h:208
@ RD_KAFKA_CONSUMER
Definition: rdkafka.h:210
@ RD_KAFKA_PRODUCER
Definition: rdkafka.h:209
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_DeleteRecords_result_offsets(const rd_kafka_DeleteRecords_result_t *result)
Get a list of topic and partition results from a DeleteRecords result. The returned objects will cont...
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a single rd_kafka_topic_conf_t value by property name.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_consumer(rd_kafka_t *rk)
RD_EXPORT void rd_kafka_set_log_level(rd_kafka_t *rk, int level)
Specifies the maximum logging level emitted by internal kafka logging and debugging.
RD_EXPORT const char * rd_kafka_AclBinding_name(const rd_kafka_AclBinding_t *acl)
RD_EXPORT void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu, int fd, const void *payload, size_t size)
Enable IO event triggering for queue.
rd_kafka_ResourcePatternType_t
Apache Kafka pattern types.
Definition: rdkafka.h:7399
@ RD_KAFKA_RESOURCE_PATTERN_LITERAL
Definition: rdkafka.h:7407
@ RD_KAFKA_RESOURCE_PATTERN_ANY
Definition: rdkafka.h:7403
@ RD_KAFKA_RESOURCE_PATTERN_MATCH
Definition: rdkafka.h:7405
@ RD_KAFKA_RESOURCE_PATTERN_PREFIXED
Definition: rdkafka.h:7409
@ RD_KAFKA_RESOURCE_PATTERN_UNKNOWN
Definition: rdkafka.h:7401
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupDescription_state(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets state for the grpdesc group.
RD_EXPORT int rd_kafka_error_is_retriable(const rd_kafka_error_t *error)
rd_kafka_ResourceType_t
Apache Kafka resource types.
Definition: rdkafka.h:7386
@ RD_KAFKA_RESOURCE_BROKER
Definition: rdkafka.h:7391
@ RD_KAFKA_RESOURCE__CNT
Definition: rdkafka.h:7392
@ RD_KAFKA_RESOURCE_TOPIC
Definition: rdkafka.h:7389
@ RD_KAFKA_RESOURCE_GROUP
Definition: rdkafka.h:7390
@ RD_KAFKA_RESOURCE_UNKNOWN
Definition: rdkafka.h:7387
@ RD_KAFKA_RESOURCE_ANY
Definition: rdkafka.h:7388
struct rd_kafka_ConsumerGroupDescription_s rd_kafka_ConsumerGroupDescription_t
DescribeConsumerGroups result type.
Definition: rdkafka.h:7843
RD_EXPORT void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf, void(*error_cb)(rd_kafka_t *rk, int err, const char *reason, void *opaque))
Set error callback in provided conf object.
RD_EXPORT int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition)
Stop consuming messages for topic rkt and partition, purging all messages currently in the local queu...
RD_EXPORT void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable)
Enable/disable creation of a queue specific to SASL events and callbacks.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_resume_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Resume producing consumption for the provided list of partitions.
RD_EXPORT void rd_kafka_ConfigResource_destroy(rd_kafka_ConfigResource_t *config)
Destroy and free a ConfigResource object previously created with rd_kafka_ConfigResource_new()
RD_EXPORT void rd_kafka_DeleteTopic_destroy_array(rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt)
Helper function to destroy all DeleteTopic objects in the del_topics array (of del_topic_cnt elements...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high)
Get last known low (oldest/beginning) and high (newest/end) offsets for partition.
RD_EXPORT int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt, int32_t partition)
Check if partition is available (has a leader broker).
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_read(rd_kafka_consumer_group_metadata_t **cgmdp, const void *buffer, size_t size)
Reads serialized consumer group metadata and returns a consumer group metadata object....
RD_EXPORT void rd_kafka_ListConsumerGroups(rd_kafka_t *rk, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List the consumer groups available in the cluster.
RD_EXPORT const rd_kafka_DeleteGroups_result_t * rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev)
Get DeleteGroups result.
rd_kafka_msg_status_t
Message persistence status can be used by the application to find out if a produced message was persi...
Definition: rdkafka.h:1594
@ RD_KAFKA_MSG_STATUS_POSSIBLY_PERSISTED
Definition: rdkafka.h:1603
@ RD_KAFKA_MSG_STATUS_PERSISTED
Definition: rdkafka.h:1608
@ RD_KAFKA_MSG_STATUS_NOT_PERSISTED
Definition: rdkafka.h:1598
RD_EXPORT int rd_kafka_outq_len(rd_kafka_t *rk)
Returns the current out queue length.
RD_EXPORT int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms)
Polls the provided kafka handle for events.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_partition(rd_kafka_t *rk, const char *topic, int32_t partition)
RD_EXPORT const char * rd_kafka_ConfigEntry_name(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_event_t rd_kafka_AlterConfigs_result_t
Definition: rdkafka.h:5612
RD_EXPORT const char * rd_kafka_event_error_string(rd_kafka_event_t *rkev)
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1648
@ RD_KAFKA_CONF_OK
Definition: rdkafka.h:1653
@ RD_KAFKA_CONF_INVALID
Definition: rdkafka.h:1650
@ RD_KAFKA_CONF_UNKNOWN
Definition: rdkafka.h:1649
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs, const char *name, ssize_t name_size, const void *value, ssize_t value_size)
Add header with name name and value val (copied) of size size (not including null-terminator).
RD_EXPORT int32_t rd_kafka_topic_partition_get_leader_epoch(const rd_kafka_topic_partition_t *rktpar)
RD_EXPORT const char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
RD_EXPORT const rd_kafka_DeleteRecords_result_t * rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_AclBinding_principal(const rd_kafka_AclBinding_t *acl)
RD_EXPORT int32_t rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage)
RD_EXPORT const rd_kafka_DescribeAcls_result_t * rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_DeleteRecords_t * rd_kafka_DeleteRecords_new(const rd_kafka_topic_partition_list_t *before_offsets)
Create a new DeleteRecords object. This object is later passed to rd_kafka_DeleteRecords().
RD_EXPORT void rd_kafka_NewPartitions_destroy(rd_kafka_NewPartitions_t *new_parts)
Destroy and free a NewPartitions object previously created with rd_kafka_NewPartitions_new()
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_replica_assignment(rd_kafka_NewTopic_t *new_topic, int32_t partition, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker) assignment for partition to the replica set in broker_ids (of broker_id_cnt ...
RD_EXPORT void rd_kafka_AlterConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Update the configuration for the specified resources. Updates are not transactional so they may succe...
RD_EXPORT const char * rd_kafka_get_debug_contexts(void)
Retrieve supported debug contexts for use with the "debug" configuration property....
RD_EXPORT rd_kafka_ConfigResource_t * rd_kafka_ConfigResource_new(rd_kafka_ResourceType_t restype, const char *resname)
Create new ConfigResource object.
RD_EXPORT void rd_kafka_AdminOptions_destroy(rd_kafka_AdminOptions_t *options)
Destroy a AdminOptions object.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert_verify_cb(rd_kafka_conf_t *conf, int(*ssl_cert_verify_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int *x509_error, int depth, const char *buf, size_t size, char *errstr, size_t errstr_size, void *opaque))
Sets the verification callback of the broker certificate.
RD_EXPORT int rd_kafka_consume_start(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Start consuming messages for topic rkt and partition at offset offset which may either be an absolute...
RD_EXPORT int rd_kafka_produce(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, void *payload, size_t len, const void *key, size_t keylen, void *msg_opaque)
Produce and send a single message to broker.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_for_times(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets, int timeout_ms)
Look up the offsets for the given partitions by timestamp.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_consume_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_consume() is called just prior to passing the message to the application in rd_kafka_consumer_poll...
Definition: rdkafka.h:6127
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_broker(rd_kafka_AdminOptions_t *options, int32_t broker_id, char *errstr, size_t errstr_size)
Override what broker the Admin request will be sent to.
RD_EXPORT void rd_kafka_DeleteTopics(rd_kafka_t *rk, rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete topics from cluster as specified by the topics array of size topic_cnt elements.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_last_error(void)
Returns the last error code generated by a legacy API call in the current thread.
RD_EXPORT const char * rd_kafka_event_stats(rd_kafka_event_t *rkev)
Extract stats from the event.
struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t
ACL Binding is used to create access control lists.
Definition: rdkafka.h:8463
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_thread_exit_t(rd_kafka_t *rk, rd_kafka_thread_type_t thread_type, const char *thread_name, void *ic_opaque)
on_thread_exit() is called just prior to a librdkafka-managed thread exiting from the exiting thread ...
Definition: rdkafka.h:6274
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_validate_only(rd_kafka_AdminOptions_t *options, int true_or_false, char *errstr, size_t errstr_size)
Tell broker to only validate the request, without performing the requested operation (create topics,...
RD_EXPORT const rd_kafka_ListConsumerGroups_result_t * rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev)
Get ListConsumerGroups result.
RD_EXPORT const char * rd_kafka_error_string(const rd_kafka_error_t *error)
RD_EXPORT const rd_kafka_error_t * rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres)
RD_EXPORT int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random partitioner.
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_write(const rd_kafka_consumer_group_metadata_t *cgmd, void **bufferp, size_t *sizep)
Serialize the consumer group metadata to a binary format. This is mainly for client binding use and n...
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_DeleteTopics_result_topics(const rd_kafka_DeleteTopics_result_t *result, size_t *cntp)
Get an array of topic results from a DeleteTopics result.
RD_EXPORT void rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *)
Frees the consumer group metadata object as returned by rd_kafka_consumer_group_metadata().
struct rd_kafka_ListConsumerGroupOffsets_s rd_kafka_ListConsumerGroupOffsets_t
Definition: rdkafka.h:8179
RD_EXPORT void * rd_kafka_opaque(const rd_kafka_t *rk)
Retrieves the opaque pointer previously set with rd_kafka_conf_set_opaque()
RD_EXPORT void rd_kafka_event_destroy(rd_kafka_event_t *rkev)
Destroy an event.
RD_EXPORT const rd_kafka_DeleteAcls_result_response_t ** rd_kafka_DeleteAcls_result_responses(const rd_kafka_DeleteAcls_result_t *result, size_t *cntp)
Get an array of DeleteAcls result responses from a DeleteAcls result.
struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t
Member description included in ConsumerGroupDescription.
Definition: rdkafka.h:7850
RD_EXPORT rd_kafka_msg_status_t rd_kafka_message_status(const rd_kafka_message_t *rkmessage)
Returns the message's persistence status in the topic log.
RD_EXPORT void rd_kafka_AclBinding_destroy_array(rd_kafka_AclBinding_t **acl_bindings, size_t acl_bindings_cnt)
Helper function to destroy all AclBinding objects in the acl_bindings array (of acl_bindings_cnt elem...
RD_EXPORT void rd_kafka_conf_destroy(rd_kafka_conf_t *conf)
Destroys a conf object.
RD_EXPORT void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu)
Cancels the current rd_kafka_queue_poll() on rkqu.
RD_EXPORT const char * rd_kafka_MemberDescription_client_id(const rd_kafka_MemberDescription_t *member)
Gets client id of member.
RD_EXPORT void rd_kafka_topic_partition_list_sort(rd_kafka_topic_partition_list_t *rktparlist, int(*cmp)(const void *a, const void *b, void *cmp_opaque), void *cmp_opaque)
Sort list using comparator cmp.
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_AclBinding_restype(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_send(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_send_t *on_send, void *ic_opaque)
Append an on_send() interceptor.
RD_EXPORT const char * rd_kafka_ConfigResource_name(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_queue(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, rd_kafka_queue_t *rkqu, void(*cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *commit_opaque), void *commit_opaque)
Commit offsets on broker for the provided list of partitions.
rd_kafka_timestamp_type_t
Definition: rdkafka.h:219
@ RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafka.h:220
@ RD_KAFKA_TIMESTAMP_CREATE_TIME
Definition: rdkafka.h:221
@ RD_KAFKA_TIMESTAMP_LOG_APPEND_TIME
Definition: rdkafka.h:222
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_detach_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list and detach the list from the message making the application the owner of ...
RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs)
Returns the number of header key/value pairs.
RD_EXPORT void rd_kafka_DeleteGroups(rd_kafka_t *rk, rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete groups from cluster as specified by the del_groups array of size del_group_cnt elements.
RD_EXPORT rd_kafka_ListConsumerGroupOffsets_t * rd_kafka_ListConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new ListConsumerGroupOffsets object. This object is later passed to rd_kafka_ListConsumerGro...
rd_kafka_AclBinding_t rd_kafka_AclBindingFilter_t
ACL Binding filter is used to filter access control lists.
Definition: rdkafka.h:8469
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete committed offsets for a set of partitions in a consumer group. This will succeed at the partit...
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_ConfigResource_type(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_new(int size)
Create a new list/vector Topic+Partition container.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_set_config(rd_kafka_ConfigResource_t *config, const char *name, const char *value)
Set configuration name value pair.
RD_EXPORT rd_kafka_error_t * rd_kafka_error_new(rd_kafka_resp_err_t code, const char *fmt,...) RD_FORMAT(printf
Create a new error object with error code and optional human readable error string in fmt.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_destroy_t(void *ic_opaque)
on_conf_destroy() is called from rd_kafka_*_conf_destroy() in the order the interceptors were added.
Definition: rdkafka.h:6017
struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t
Member assignment included in MemberDescription.
Definition: rdkafka.h:7856
RD_EXPORT const char * rd_kafka_AclOperation_name(rd_kafka_AclOperation_t acl_operation)
RD_EXPORT rd_kafka_ConfigSource_t rd_kafka_ConfigEntry_source(const rd_kafka_ConfigEntry_t *entry)
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:642
const char * name
Definition: rdkafka.h:644
rd_kafka_resp_err_t code
Definition: rdkafka.h:643
const char * desc
Definition: rdkafka.h:645
Group information.
Definition: rdkafka.h:5021
struct rd_kafka_metadata_broker broker
Definition: rdkafka.h:5022
int member_cnt
Definition: rdkafka.h:5029
char * state
Definition: rdkafka.h:5025
char * group
Definition: rdkafka.h:5023
struct rd_kafka_group_member_info * members
Definition: rdkafka.h:5028
char * protocol
Definition: rdkafka.h:5027
char * protocol_type
Definition: rdkafka.h:5026
rd_kafka_resp_err_t err
Definition: rdkafka.h:5024
List of groups.
Definition: rdkafka.h:5037
int group_cnt
Definition: rdkafka.h:5039
struct rd_kafka_group_info * groups
Definition: rdkafka.h:5038
Group member information.
Definition: rdkafka.h:4991
char * member_id
Definition: rdkafka.h:4992
int member_assignment_size
Definition: rdkafka.h:5000
int member_metadata_size
Definition: rdkafka.h:4997
void * member_metadata
Definition: rdkafka.h:4995
char * client_host
Definition: rdkafka.h:4994
void * member_assignment
Definition: rdkafka.h:4998
char * client_id
Definition: rdkafka.h:4993
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1440
size_t key_len
Definition: rdkafka.h:1453
size_t len
Definition: rdkafka.h:1448
void * _private
Definition: rdkafka.h:1463
void * key
Definition: rdkafka.h:1451
int64_t offset
Definition: rdkafka.h:1455
void * payload
Definition: rdkafka.h:1444
int32_t partition
Definition: rdkafka.h:1443
rd_kafka_topic_t * rkt
Definition: rdkafka.h:1442
rd_kafka_resp_err_t err
Definition: rdkafka.h:1441
Broker information.
Definition: rdkafka.h:4856
int32_t id
Definition: rdkafka.h:4857
int port
Definition: rdkafka.h:4859
char * host
Definition: rdkafka.h:4858
Partition information.
Definition: rdkafka.h:4865
int32_t leader
Definition: rdkafka.h:4868
int32_t * isrs
Definition: rdkafka.h:4872
int replica_cnt
Definition: rdkafka.h:4869
rd_kafka_resp_err_t err
Definition: rdkafka.h:4867
int isr_cnt
Definition: rdkafka.h:4871
int32_t id
Definition: rdkafka.h:4866
int32_t * replicas
Definition: rdkafka.h:4870
Metadata container.
Definition: rdkafka.h:4889
int32_t orig_broker_id
Definition: rdkafka.h:4896
char * orig_broker_name
Definition: rdkafka.h:4897
int broker_cnt
Definition: rdkafka.h:4890
struct rd_kafka_metadata_topic * topics
Definition: rdkafka.h:4894
int topic_cnt
Definition: rdkafka.h:4893
struct rd_kafka_metadata_broker * brokers
Definition: rdkafka.h:4891
Topic information.
Definition: rdkafka.h:4878
char * topic
Definition: rdkafka.h:4879
struct rd_kafka_metadata_partition * partitions
Definition: rdkafka.h:4881
rd_kafka_resp_err_t err
Definition: rdkafka.h:4882
int partition_cnt
Definition: rdkafka.h:4880
A growable list of Topic+Partitions.
Definition: rdkafka.h:949
int cnt
Definition: rdkafka.h:950
int size
Definition: rdkafka.h:951
rd_kafka_topic_partition_t * elems
Definition: rdkafka.h:952
Topic+Partition place holder.
Definition: rdkafka.h:897
int64_t offset
Definition: rdkafka.h:900
size_t metadata_size
Definition: rdkafka.h:902
void * opaque
Definition: rdkafka.h:903
char * topic
Definition: rdkafka.h:898
void * metadata
Definition: rdkafka.h:901
int32_t partition
Definition: rdkafka.h:899
rd_kafka_resp_err_t err
Definition: rdkafka.h:904
void * _private
Definition: rdkafka.h:905
VTYPE + argument container for use with rd_kafka_produce_va()
Definition: rdkafka.h:1144
rd_kafka_vtype_t vtype
Definition: rdkafka.h:1145