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  * 2023, Confluent Inc.
6  * All rights reserved.
7  *
8  * Redistribution and use in source and binary forms, with or without
9  * modification, are permitted provided that the following conditions are met:
10  *
11  * 1. Redistributions of source code must retain the above copyright notice,
12  * this list of conditions and the following disclaimer.
13  * 2. Redistributions in binary form must reproduce the above copyright notice,
14  * this list of conditions and the following disclaimer in the documentation
15  * and/or other materials provided with the distribution.
16  *
17  * THIS SOFTWARE IS PROVIDED BY THE COPYRIGHT HOLDERS AND CONTRIBUTORS "AS IS"
18  * AND ANY EXPRESS OR IMPLIED WARRANTIES, INCLUDING, BUT NOT LIMITED TO, THE
19  * IMPLIED WARRANTIES OF MERCHANTABILITY AND FITNESS FOR A PARTICULAR PURPOSE
20  * ARE DISCLAIMED. IN NO EVENT SHALL THE COPYRIGHT OWNER OR CONTRIBUTORS BE
21  * LIABLE FOR ANY DIRECT, INDIRECT, INCIDENTAL, SPECIAL, EXEMPLARY, OR
22  * CONSEQUENTIAL DAMAGES (INCLUDING, BUT NOT LIMITED TO, PROCUREMENT OF
23  * SUBSTITUTE GOODS OR SERVICES; LOSS OF USE, DATA, OR PROFITS; OR BUSINESS
24  * INTERRUPTION) HOWEVER CAUSED AND ON ANY THEORY OF LIABILITY, WHETHER IN
25  * CONTRACT, STRICT LIABILITY, OR TORT (INCLUDING NEGLIGENCE OR OTHERWISE)
26  * ARISING IN ANY WAY OUT OF THE USE OF THIS SOFTWARE, EVEN IF ADVISED OF THE
27  * POSSIBILITY OF SUCH DAMAGE.
28  */
29 
44 /* @cond NO_DOC */
45 #ifndef _RDKAFKA_H_
46 #define _RDKAFKA_H_
47 
48 #include <stdio.h>
49 #include <inttypes.h>
50 #include <sys/types.h>
51 
52 #ifdef __cplusplus
53 extern "C" {
54 #if 0
55 } /* Restore indent */
56 #endif
57 #endif
58 
59 #ifdef _WIN32
60 #include <basetsd.h>
61 #ifndef WIN32_MEAN_AND_LEAN
62 #define WIN32_MEAN_AND_LEAN
63 #endif
64 #include <winsock2.h> /* for sockaddr, .. */
65 #ifndef _SSIZE_T_DEFINED
66 #define _SSIZE_T_DEFINED
67 typedef SSIZE_T ssize_t;
68 #endif
69 #define RD_UNUSED
70 #define RD_INLINE __inline
71 #define RD_DEPRECATED __declspec(deprecated)
72 #define RD_FORMAT(...)
73 #undef RD_EXPORT
74 #ifdef LIBRDKAFKA_STATICLIB
75 #define RD_EXPORT
76 #else
77 #ifdef LIBRDKAFKA_EXPORTS
78 #define RD_EXPORT __declspec(dllexport)
79 #else
80 #define RD_EXPORT __declspec(dllimport)
81 #endif
82 #ifndef LIBRDKAFKA_TYPECHECKS
83 #define LIBRDKAFKA_TYPECHECKS 0
84 #endif
85 #endif
86 
87 #else
88 #include <sys/socket.h> /* for sockaddr, .. */
89 
90 #define RD_UNUSED __attribute__((unused))
91 #define RD_INLINE inline
92 #define RD_EXPORT
93 #define RD_DEPRECATED __attribute__((deprecated))
94 
95 #if defined(__clang__) || defined(__GNUC__) || defined(__GNUG__)
96 #define RD_HAS_STATEMENT_EXPRESSIONS
97 #define RD_FORMAT(...) __attribute__((format(__VA_ARGS__)))
98 #else
99 #define RD_FORMAT(...)
100 #endif
101 
102 #ifndef LIBRDKAFKA_TYPECHECKS
103 #define LIBRDKAFKA_TYPECHECKS 1
104 #endif
105 #endif
106 
107 
113 #if LIBRDKAFKA_TYPECHECKS
114 #define _LRK_TYPECHECK(RET, TYPE, ARG) \
115  ({ \
116  if (0) { \
117  TYPE __t RD_UNUSED = (ARG); \
118  } \
119  RET; \
120  })
121 
122 #define _LRK_TYPECHECK2(RET, TYPE, ARG, TYPE2, ARG2) \
123  ({ \
124  if (0) { \
125  TYPE __t RD_UNUSED = (ARG); \
126  TYPE2 __t2 RD_UNUSED = (ARG2); \
127  } \
128  RET; \
129  })
130 
131 #define _LRK_TYPECHECK3(RET, TYPE, ARG, TYPE2, ARG2, TYPE3, ARG3) \
132  ({ \
133  if (0) { \
134  TYPE __t RD_UNUSED = (ARG); \
135  TYPE2 __t2 RD_UNUSED = (ARG2); \
136  TYPE3 __t3 RD_UNUSED = (ARG3); \
137  } \
138  RET; \
139  })
140 #else
141 #define _LRK_TYPECHECK(RET, TYPE, ARG) (RET)
142 #define _LRK_TYPECHECK2(RET, TYPE, ARG, TYPE2, ARG2) (RET)
143 #define _LRK_TYPECHECK3(RET, TYPE, ARG, TYPE2, ARG2, TYPE3, ARG3) (RET)
144 #endif
145 
146 /* @endcond */
147 
148 
170 #define RD_KAFKA_VERSION 0x020400ff
171 
180 RD_EXPORT
181 int rd_kafka_version(void);
182 
188 RD_EXPORT
189 const char *rd_kafka_version_str(void);
190 
209 typedef enum rd_kafka_type_t {
213 
214 
225 
226 
227 
234 RD_EXPORT
235 const char *rd_kafka_get_debug_contexts(void);
236 
244 #define RD_KAFKA_DEBUG_CONTEXTS \
245  "all,generic,broker,topic,metadata,feature,queue,msg,protocol,cgrp," \
246  "security,fetch,interceptor,plugin,consumer,admin,eos,mock,assignor," \
247  "conf"
248 
249 
250 /* @cond NO_DOC */
251 /* Private types to provide ABI compatibility */
252 typedef struct rd_kafka_s rd_kafka_t;
253 typedef struct rd_kafka_topic_s rd_kafka_topic_t;
254 typedef struct rd_kafka_conf_s rd_kafka_conf_t;
255 typedef struct rd_kafka_topic_conf_s rd_kafka_topic_conf_t;
256 typedef struct rd_kafka_queue_s rd_kafka_queue_t;
257 typedef struct rd_kafka_op_s rd_kafka_event_t;
258 typedef struct rd_kafka_topic_result_s rd_kafka_topic_result_t;
259 typedef struct rd_kafka_consumer_group_metadata_s
260  rd_kafka_consumer_group_metadata_t;
261 typedef struct rd_kafka_error_s rd_kafka_error_t;
262 typedef struct rd_kafka_headers_s rd_kafka_headers_t;
263 typedef struct rd_kafka_group_result_s rd_kafka_group_result_t;
264 typedef struct rd_kafka_acl_result_s rd_kafka_acl_result_t;
265 typedef struct rd_kafka_Uuid_s rd_kafka_Uuid_t;
266 /* @endcond */
267 
268 
281 typedef enum {
282  /* Internal errors to rdkafka: */
413 
416 
417  /* Kafka broker errors: */
433 #define RD_KAFKA_RESP_ERR_NOT_LEADER_OR_FOLLOWER \
434  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION
435  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION = 6,
453 #define RD_KAFKA_RESP_ERR_GROUP_LOAD_IN_PROGRESS \
454  RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
455 
458 #define RD_KAFKA_RESP_ERR_GROUP_COORDINATOR_NOT_AVAILABLE \
459  RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
460 
463 #define RD_KAFKA_RESP_ERR_NOT_COORDINATOR_FOR_GROUP \
464  RD_KAFKA_RESP_ERR_NOT_COORDINATOR
465 
649  RD_KAFKA_RESP_ERR_END_ALL,
651 
652 
660  const char *name;
661  const char *desc;
662 };
663 
664 
668 RD_EXPORT
669 void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs,
670  size_t *cntp);
671 
672 
673 
679 RD_EXPORT
680 const char *rd_kafka_err2str(rd_kafka_resp_err_t err);
681 
682 
683 
689 RD_EXPORT
690 const char *rd_kafka_err2name(rd_kafka_resp_err_t err);
691 
692 
718 RD_EXPORT
720 
721 
746 RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox);
747 
748 
761 RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void);
762 
763 
764 
791 RD_EXPORT
793 rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size);
794 
795 
813 RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk,
815  const char *reason);
816 
817 
822 RD_EXPORT
823 rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error);
824 
833 RD_EXPORT
834 const char *rd_kafka_error_name(const rd_kafka_error_t *error);
835 
842 RD_EXPORT
843 const char *rd_kafka_error_string(const rd_kafka_error_t *error);
844 
845 
850 RD_EXPORT
851 int rd_kafka_error_is_fatal(const rd_kafka_error_t *error);
852 
853 
858 RD_EXPORT
859 int rd_kafka_error_is_retriable(const rd_kafka_error_t *error);
860 
861 
872 RD_EXPORT
873 int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error);
874 
880 RD_EXPORT
881 void rd_kafka_error_destroy(rd_kafka_error_t *error);
882 
883 
892 RD_EXPORT
893 rd_kafka_error_t *rd_kafka_error_new(rd_kafka_resp_err_t code,
894  const char *fmt,
895  ...) RD_FORMAT(printf, 2, 3);
896 
897 
914  char *topic;
915  int32_t partition;
916  int64_t offset;
917  void *metadata;
918  size_t metadata_size;
919  void *opaque;
921  void *_private;
925 } rd_kafka_topic_partition_t;
926 
931 RD_EXPORT
932 void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar);
933 
934 
943 RD_EXPORT
945  rd_kafka_topic_partition_t *rktpar,
946  int32_t leader_epoch);
947 
956 RD_EXPORT
958  const rd_kafka_topic_partition_t *rktpar);
959 
965  int cnt;
966  int size;
967  rd_kafka_topic_partition_t *elems;
968 } rd_kafka_topic_partition_list_t;
969 
984 RD_EXPORT
985 rd_kafka_topic_partition_list_t *rd_kafka_topic_partition_list_new(int size);
986 
990 RD_EXPORT
992  rd_kafka_topic_partition_list_t *rkparlist);
993 
1003 RD_EXPORT
1004 rd_kafka_topic_partition_t *
1005 rd_kafka_topic_partition_list_add(rd_kafka_topic_partition_list_t *rktparlist,
1006  const char *topic,
1007  int32_t partition);
1008 
1009 
1018 RD_EXPORT
1020  rd_kafka_topic_partition_list_t *rktparlist,
1021  const char *topic,
1022  int32_t start,
1023  int32_t stop);
1024 
1025 
1026 
1038 RD_EXPORT
1040  rd_kafka_topic_partition_list_t *rktparlist,
1041  const char *topic,
1042  int32_t partition);
1043 
1044 
1052 RD_EXPORT
1054  rd_kafka_topic_partition_list_t *rktparlist,
1055  int idx);
1056 
1057 
1065 RD_EXPORT
1066 rd_kafka_topic_partition_list_t *
1067 rd_kafka_topic_partition_list_copy(const rd_kafka_topic_partition_list_t *src);
1068 
1069 
1070 
1078 RD_EXPORT
1080  rd_kafka_topic_partition_list_t *rktparlist,
1081  const char *topic,
1082  int32_t partition,
1083  int64_t offset);
1084 
1085 
1086 
1092 RD_EXPORT
1093 rd_kafka_topic_partition_t *rd_kafka_topic_partition_list_find(
1094  const rd_kafka_topic_partition_list_t *rktparlist,
1095  const char *topic,
1096  int32_t partition);
1097 
1098 
1108 RD_EXPORT void rd_kafka_topic_partition_list_sort(
1109  rd_kafka_topic_partition_list_t *rktparlist,
1110  int (*cmp)(const void *a, const void *b, void *cmp_opaque),
1111  void *cmp_opaque);
1112 
1113 
1131 typedef enum rd_kafka_vtype_t {
1149 
1150 
1157 typedef struct rd_kafka_vu_s {
1160  union {
1161  const char *cstr;
1162  rd_kafka_topic_t *rkt;
1163  int i;
1164  int32_t i32;
1165  int64_t i64;
1166  struct {
1167  void *ptr;
1168  size_t size;
1169  } mem;
1170  struct {
1171  const char *name;
1172  const void *val;
1173  ssize_t size;
1174  } header;
1175  rd_kafka_headers_t *headers;
1176  void *ptr;
1177  char _pad[64];
1178  } u;
1179 } rd_kafka_vu_t;
1180 
1189 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
1190 
1196 #define RD_KAFKA_V_TOPIC(topic) \
1197  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
1198  (const char *)topic
1199 
1204 #define RD_KAFKA_V_RKT(rkt) \
1205  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
1206  (rd_kafka_topic_t *)rkt
1207 
1212 #define RD_KAFKA_V_PARTITION(partition) \
1213  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
1214  (int32_t)partition
1215 
1220 #define RD_KAFKA_V_VALUE(VALUE, LEN) \
1221  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
1222  (void *)VALUE, (size_t)LEN
1223 
1228 #define RD_KAFKA_V_KEY(KEY, LEN) \
1229  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
1230  (void *)KEY, (size_t)LEN
1231 
1238 #define RD_KAFKA_V_OPAQUE(msg_opaque) \
1239  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, msg_opaque), \
1240  (void *)msg_opaque
1241 
1247 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
1248  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), (int)msgflags
1249 
1255 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
1256  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
1257  (int64_t)timestamp
1258 
1266 #define RD_KAFKA_V_HEADER(NAME, VALUE, LEN) \
1267  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
1268  const void *, VALUE, ssize_t, LEN), \
1269  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
1270 
1282 #define RD_KAFKA_V_HEADERS(HDRS) \
1283  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
1284  (rd_kafka_headers_t *)HDRS
1285 
1286 
1317 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new(size_t initial_count);
1318 
1323 RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs);
1324 
1328 RD_EXPORT rd_kafka_headers_t *
1329 rd_kafka_headers_copy(const rd_kafka_headers_t *src);
1330 
1348 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs,
1349  const char *name,
1350  ssize_t name_size,
1351  const void *value,
1352  ssize_t value_size);
1353 
1361 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs,
1362  const char *name);
1363 
1364 
1382 RD_EXPORT rd_kafka_resp_err_t
1383 rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs,
1384  const char *name,
1385  const void **valuep,
1386  size_t *sizep);
1387 
1401 RD_EXPORT rd_kafka_resp_err_t
1402 rd_kafka_header_get(const rd_kafka_headers_t *hdrs,
1403  size_t idx,
1404  const char *name,
1405  const void **valuep,
1406  size_t *sizep);
1407 
1408 
1416 RD_EXPORT rd_kafka_resp_err_t
1417 rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs,
1418  size_t idx,
1419  const char **namep,
1420  const void **valuep,
1421  size_t *sizep);
1422 
1423 
1424 
1437 // FIXME: This doesn't show up in docs for some reason
1438 // "Compound rd_kafka_message_t is not documented."
1439 
1453 typedef struct rd_kafka_message_s {
1455  rd_kafka_topic_t *rkt;
1456  int32_t partition;
1457  void *payload;
1461  size_t len;
1464  void *key;
1466  size_t key_len;
1468  int64_t offset;
1476  void *_private;
1483 } rd_kafka_message_t;
1484 
1485 
1489 RD_EXPORT
1490 void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage);
1491 
1492 
1493 
1500 RD_EXPORT
1501 const char *rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage);
1502 
1509 RD_EXPORT
1510 const char *
1511 rd_kafka_message_produce_errstr(const rd_kafka_message_t *rkmessage);
1512 
1513 
1525 RD_EXPORT
1526 int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage,
1527  rd_kafka_timestamp_type_t *tstype);
1528 
1529 
1530 
1537 RD_EXPORT
1538 int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage);
1539 
1540 
1547 RD_EXPORT
1548 int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage);
1549 
1550 
1567 RD_EXPORT rd_kafka_resp_err_t
1568 rd_kafka_message_headers(const rd_kafka_message_t *rkmessage,
1569  rd_kafka_headers_t **hdrsp);
1570 
1582 RD_EXPORT rd_kafka_resp_err_t
1583 rd_kafka_message_detach_headers(rd_kafka_message_t *rkmessage,
1584  rd_kafka_headers_t **hdrsp);
1585 
1586 
1599 RD_EXPORT
1600 void rd_kafka_message_set_headers(rd_kafka_message_t *rkmessage,
1601  rd_kafka_headers_t *hdrs);
1602 
1603 
1609 RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs);
1610 
1611 
1617 typedef enum {
1622 
1627 
1633 
1634 
1641 RD_EXPORT rd_kafka_msg_status_t
1642 rd_kafka_message_status(const rd_kafka_message_t *rkmessage);
1643 
1644 
1652 RD_EXPORT int32_t
1653 rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage);
1654 
1655 
1672 RD_EXPORT const char *rd_kafka_Uuid_base64str(const rd_kafka_Uuid_t *uuid);
1673 
1681 RD_EXPORT int64_t
1682 rd_kafka_Uuid_least_significant_bits(const rd_kafka_Uuid_t *uuid);
1683 
1684 
1692 RD_EXPORT int64_t
1693 rd_kafka_Uuid_most_significant_bits(const rd_kafka_Uuid_t *uuid);
1694 
1695 
1705 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_new(int64_t most_significant_bits,
1706  int64_t least_significant_bits);
1707 
1716 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_copy(const rd_kafka_Uuid_t *uuid);
1717 
1723 RD_EXPORT void rd_kafka_Uuid_destroy(rd_kafka_Uuid_t *uuid);
1724 
1740 typedef enum {
1747 
1748 
1782 RD_EXPORT
1783 rd_kafka_conf_t *rd_kafka_conf_new(void);
1784 
1785 
1789 RD_EXPORT
1790 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1791 
1792 
1799 RD_EXPORT
1800 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1801 
1802 
1807 RD_EXPORT
1808 rd_kafka_conf_t *rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf,
1809  size_t filter_cnt,
1810  const char **filter);
1811 
1812 
1813 
1822 RD_EXPORT
1823 const rd_kafka_conf_t *rd_kafka_conf(rd_kafka_t *rk);
1824 
1825 
1845 RD_EXPORT
1846 rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf,
1847  const char *name,
1848  const char *value,
1849  char *errstr,
1850  size_t errstr_size);
1851 
1852 
1858 RD_EXPORT
1859 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1860 
1861 
1898  rd_kafka_conf_t *conf,
1899  void (*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque));
1900 
1901 
1905 RD_EXPORT
1906 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1907  void (*dr_cb)(rd_kafka_t *rk,
1908  void *payload,
1909  size_t len,
1910  rd_kafka_resp_err_t err,
1911  void *opaque,
1912  void *msg_opaque));
1913 
1944 RD_EXPORT
1946  rd_kafka_conf_t *conf,
1947  void (*dr_msg_cb)(rd_kafka_t *rk,
1948  const rd_kafka_message_t *rkmessage,
1949  void *opaque));
1950 
1951 
1959 RD_EXPORT
1961  rd_kafka_conf_t *conf,
1962  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque));
1963 
2068 RD_EXPORT
2070  rd_kafka_conf_t *conf,
2071  void (*rebalance_cb)(rd_kafka_t *rk,
2072  rd_kafka_resp_err_t err,
2073  rd_kafka_topic_partition_list_t *partitions,
2074  void *opaque));
2075 
2076 
2077 
2095 RD_EXPORT
2097  rd_kafka_conf_t *conf,
2098  void (*offset_commit_cb)(rd_kafka_t *rk,
2099  rd_kafka_resp_err_t err,
2100  rd_kafka_topic_partition_list_t *offsets,
2101  void *opaque));
2102 
2103 
2126 RD_EXPORT
2127 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
2128  void (*error_cb)(rd_kafka_t *rk,
2129  int err,
2130  const char *reason,
2131  void *opaque));
2132 
2150 RD_EXPORT
2151 void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf,
2152  void (*throttle_cb)(rd_kafka_t *rk,
2153  const char *broker_name,
2154  int32_t broker_id,
2155  int throttle_time_ms,
2156  void *opaque));
2157 
2158 
2175 RD_EXPORT
2176 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
2177  void (*log_cb)(const rd_kafka_t *rk,
2178  int level,
2179  const char *fac,
2180  const char *buf));
2181 
2182 
2205 RD_EXPORT
2207  rd_kafka_conf_t *conf,
2208  int (*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque));
2209 
2259 RD_EXPORT
2261  rd_kafka_conf_t *conf,
2262  void (*oauthbearer_token_refresh_cb)(rd_kafka_t *rk,
2263  const char *oauthbearer_config,
2264  void *opaque));
2265 
2293 RD_EXPORT
2294 void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable);
2295 
2296 
2314 RD_EXPORT
2316  rd_kafka_conf_t *conf,
2317  int (*socket_cb)(int domain, int type, int protocol, void *opaque));
2318 
2319 
2320 
2336 RD_EXPORT void
2337 rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf,
2338  int (*connect_cb)(int sockfd,
2339  const struct sockaddr *addr,
2340  int addrlen,
2341  const char *id,
2342  void *opaque));
2343 
2354 RD_EXPORT void rd_kafka_conf_set_closesocket_cb(
2355  rd_kafka_conf_t *conf,
2356  int (*closesocket_cb)(int sockfd, void *opaque));
2357 
2358 
2359 
2360 #ifndef _WIN32
2361 
2378 RD_EXPORT
2380  rd_kafka_conf_t *conf,
2381  int (*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque));
2382 #endif
2383 
2385 struct addrinfo;
2386 
2406 RD_EXPORT void
2407 rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf,
2408  int (*resolve_cb)(const char *node,
2409  const char *service,
2410  const struct addrinfo *hints,
2411  struct addrinfo **res,
2412  void *opaque));
2413 
2455 RD_EXPORT
2457  rd_kafka_conf_t *conf,
2458  int (*ssl_cert_verify_cb)(rd_kafka_t *rk,
2459  const char *broker_name,
2460  int32_t broker_id,
2461  int *x509_error,
2462  int depth,
2463  const char *buf,
2464  size_t size,
2465  char *errstr,
2466  size_t errstr_size,
2467  void *opaque));
2468 
2469 
2477 typedef enum rd_kafka_cert_type_t {
2481  RD_KAFKA_CERT__CNT,
2483 
2491 typedef enum rd_kafka_cert_enc_t {
2495  RD_KAFKA_CERT_ENC__CNT,
2497 
2498 
2541 RD_EXPORT rd_kafka_conf_res_t
2542 rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf,
2543  rd_kafka_cert_type_t cert_type,
2544  rd_kafka_cert_enc_t cert_enc,
2545  const void *buffer,
2546  size_t size,
2547  char *errstr,
2548  size_t errstr_size);
2549 
2550 
2565 RD_EXPORT
2566 void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf,
2567  void *callback_data);
2568 
2569 
2575 RD_EXPORT
2576 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
2577 
2582 RD_EXPORT
2583 void *rd_kafka_opaque(const rd_kafka_t *rk);
2584 
2585 
2586 
2600 RD_EXPORT
2601 void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf,
2602  rd_kafka_topic_conf_t *tconf);
2603 
2616 RD_EXPORT rd_kafka_topic_conf_t *
2617 rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf);
2618 
2619 
2639 RD_EXPORT
2640 rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf,
2641  const char *name,
2642  char *dest,
2643  size_t *dest_size);
2644 
2645 
2651 RD_EXPORT
2652 rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf,
2653  const char *name,
2654  char *dest,
2655  size_t *dest_size);
2656 
2657 
2666 RD_EXPORT
2667 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
2668 
2669 
2678 RD_EXPORT
2679 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
2680  size_t *cntp);
2681 
2686 RD_EXPORT
2687 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
2688 
2696 RD_EXPORT
2697 void rd_kafka_conf_properties_show(FILE *fp);
2698 
2715 RD_EXPORT
2716 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
2717 
2718 
2722 RD_EXPORT
2723 rd_kafka_topic_conf_t *
2724 rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf);
2725 
2730 RD_EXPORT
2731 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup(rd_kafka_t *rk);
2732 
2733 
2737 RD_EXPORT
2738 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
2739 
2740 
2749 RD_EXPORT
2750 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
2751  const char *name,
2752  const char *value,
2753  char *errstr,
2754  size_t errstr_size);
2755 
2762 RD_EXPORT
2763 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf,
2764  void *rkt_opaque);
2765 
2766 
2786 RD_EXPORT
2788  rd_kafka_topic_conf_t *topic_conf,
2789  int32_t (*partitioner)(const rd_kafka_topic_t *rkt,
2790  const void *keydata,
2791  size_t keylen,
2792  int32_t partition_cnt,
2793  void *rkt_opaque,
2794  void *msg_opaque));
2795 
2796 
2823  rd_kafka_topic_conf_t *topic_conf,
2824  int (*msg_order_cmp)(const rd_kafka_message_t *a,
2825  const rd_kafka_message_t *b));
2826 
2827 
2835 RD_EXPORT
2836 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
2837  int32_t partition);
2838 
2839 
2840 /*******************************************************************
2841  * *
2842  * Partitioners provided by rdkafka *
2843  * *
2844  *******************************************************************/
2845 
2859 RD_EXPORT
2860 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
2861  const void *key,
2862  size_t keylen,
2863  int32_t partition_cnt,
2864  void *rkt_opaque,
2865  void *msg_opaque);
2866 
2880 RD_EXPORT
2881 int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt,
2882  const void *key,
2883  size_t keylen,
2884  int32_t partition_cnt,
2885  void *rkt_opaque,
2886  void *msg_opaque);
2887 
2903 RD_EXPORT
2904 int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt,
2905  const void *key,
2906  size_t keylen,
2907  int32_t partition_cnt,
2908  void *rkt_opaque,
2909  void *msg_opaque);
2910 
2911 
2925 RD_EXPORT
2926 int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt,
2927  const void *key,
2928  size_t keylen,
2929  int32_t partition_cnt,
2930  void *rkt_opaque,
2931  void *msg_opaque);
2932 
2947 RD_EXPORT
2948 int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt,
2949  const void *key,
2950  size_t keylen,
2951  int32_t partition_cnt,
2952  void *rkt_opaque,
2953  void *msg_opaque);
2954 
2955 
2969 RD_EXPORT
2970 int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt,
2971  const void *key,
2972  size_t keylen,
2973  int32_t partition_cnt,
2974  void *rkt_opaque,
2975  void *msg_opaque);
2976 
2977 
2992 RD_EXPORT
2993 int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt,
2994  const void *key,
2995  size_t keylen,
2996  int32_t partition_cnt,
2997  void *rkt_opaque,
2998  void *msg_opaque);
2999 
3000 
3040 RD_EXPORT
3041 rd_kafka_t *rd_kafka_new(rd_kafka_type_t type,
3042  rd_kafka_conf_t *conf,
3043  char *errstr,
3044  size_t errstr_size);
3045 
3046 
3061 RD_EXPORT
3062 void rd_kafka_destroy(rd_kafka_t *rk);
3063 
3064 
3069 RD_EXPORT
3070 void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags);
3071 
3086 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
3087 
3088 
3089 
3093 RD_EXPORT
3094 const char *rd_kafka_name(const rd_kafka_t *rk);
3095 
3096 
3100 RD_EXPORT
3101 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
3102 
3103 
3114 RD_EXPORT
3115 char *rd_kafka_memberid(const rd_kafka_t *rk);
3116 
3117 
3118 
3137 RD_EXPORT
3138 char *rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms);
3139 
3140 
3156 RD_EXPORT
3157 int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms);
3158 
3159 
3181 RD_EXPORT
3182 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk,
3183  const char *topic,
3184  rd_kafka_topic_conf_t *conf);
3185 
3186 
3187 
3196 RD_EXPORT
3197 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
3198 
3199 
3203 RD_EXPORT
3204 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
3205 
3206 
3211 RD_EXPORT
3212 void *rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt);
3213 
3214 
3221 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
3222 
3223 
3251 RD_EXPORT
3252 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
3253 
3254 
3265 RD_EXPORT
3266 void rd_kafka_yield(rd_kafka_t *rk);
3267 
3268 
3269 
3277 RD_EXPORT rd_kafka_resp_err_t
3278 rd_kafka_pause_partitions(rd_kafka_t *rk,
3279  rd_kafka_topic_partition_list_t *partitions);
3280 
3281 
3282 
3290 RD_EXPORT rd_kafka_resp_err_t
3291 rd_kafka_resume_partitions(rd_kafka_t *rk,
3292  rd_kafka_topic_partition_list_t *partitions);
3293 
3294 
3295 
3304 RD_EXPORT rd_kafka_resp_err_t
3305 rd_kafka_query_watermark_offsets(rd_kafka_t *rk,
3306  const char *topic,
3307  int32_t partition,
3308  int64_t *low,
3309  int64_t *high,
3310  int timeout_ms);
3311 
3312 
3329 RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk,
3330  const char *topic,
3331  int32_t partition,
3332  int64_t *low,
3333  int64_t *high);
3334 
3335 
3336 
3362 RD_EXPORT rd_kafka_resp_err_t
3363 rd_kafka_offsets_for_times(rd_kafka_t *rk,
3364  rd_kafka_topic_partition_list_t *offsets,
3365  int timeout_ms);
3366 
3367 
3368 
3382 RD_EXPORT
3383 void *rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size);
3384 
3385 
3386 
3400 RD_EXPORT
3401 void *rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size);
3402 
3403 
3404 
3421 RD_EXPORT
3422 void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr);
3423 
3424 
3446 RD_EXPORT
3447 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
3448 
3452 RD_EXPORT
3453 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
3454 
3455 
3462 RD_EXPORT
3463 rd_kafka_queue_t *rd_kafka_queue_get_main(rd_kafka_t *rk);
3464 
3465 
3466 
3476 RD_EXPORT
3477 rd_kafka_queue_t *rd_kafka_queue_get_sasl(rd_kafka_t *rk);
3478 
3479 
3495 RD_EXPORT
3496 rd_kafka_error_t *rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk);
3497 
3498 
3513 RD_EXPORT
3514 rd_kafka_error_t *rd_kafka_sasl_set_credentials(rd_kafka_t *rk,
3515  const char *username,
3516  const char *password);
3517 
3533 RD_EXPORT
3534 rd_kafka_queue_t *rd_kafka_queue_get_consumer(rd_kafka_t *rk);
3535 
3546 RD_EXPORT
3547 rd_kafka_queue_t *rd_kafka_queue_get_partition(rd_kafka_t *rk,
3548  const char *topic,
3549  int32_t partition);
3550 
3573 RD_EXPORT
3574 rd_kafka_queue_t *rd_kafka_queue_get_background(rd_kafka_t *rk);
3575 
3576 
3587 RD_EXPORT
3588 void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
3589 
3608 RD_EXPORT
3610  rd_kafka_queue_t *rkqu);
3611 
3612 
3616 RD_EXPORT
3617 size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu);
3618 
3619 
3637 RD_EXPORT
3638 void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu,
3639  int fd,
3640  const void *payload,
3641  size_t size);
3642 
3659 RD_EXPORT
3660 void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu,
3661  void (*event_cb)(rd_kafka_t *rk,
3662  void *qev_opaque),
3663  void *qev_opaque);
3664 
3665 
3673 RD_EXPORT
3674 void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu);
3675 
3676 
3687 #define RD_KAFKA_OFFSET_BEGINNING \
3688  -2
3690 #define RD_KAFKA_OFFSET_END \
3691  -1
3693 #define RD_KAFKA_OFFSET_STORED \
3694  -1000
3696 #define RD_KAFKA_OFFSET_INVALID -1001
3700 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
3701 
3708 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
3709 
3743 RD_EXPORT
3744 int rd_kafka_consume_start(rd_kafka_topic_t *rkt,
3745  int32_t partition,
3746  int64_t offset);
3747 
3762 RD_EXPORT
3763 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt,
3764  int32_t partition,
3765  int64_t offset,
3766  rd_kafka_queue_t *rkqu);
3767 
3781 RD_EXPORT
3782 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
3783 
3784 
3785 
3814 RD_EXPORT
3815 rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt,
3816  int32_t partition,
3817  int64_t offset,
3818  int timeout_ms);
3819 
3820 
3821 
3853 RD_EXPORT rd_kafka_error_t *
3854 rd_kafka_seek_partitions(rd_kafka_t *rk,
3855  rd_kafka_topic_partition_list_t *partitions,
3856  int timeout_ms);
3857 
3858 
3883 RD_EXPORT
3884 rd_kafka_message_t *
3885 rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms);
3886 
3887 
3888 
3914 RD_EXPORT
3915 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt,
3916  int32_t partition,
3917  int timeout_ms,
3918  rd_kafka_message_t **rkmessages,
3919  size_t rkmessages_size);
3920 
3921 
3922 
3956 RD_EXPORT
3957 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt,
3958  int32_t partition,
3959  int timeout_ms,
3960  void (*consume_cb)(rd_kafka_message_t *rkmessage,
3961  void *commit_opaque),
3962  void *commit_opaque);
3963 
3964 
3983 RD_EXPORT
3984 rd_kafka_message_t *rd_kafka_consume_queue(rd_kafka_queue_t *rkqu,
3985  int timeout_ms);
3986 
3992 RD_EXPORT
3993 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
3994  int timeout_ms,
3995  rd_kafka_message_t **rkmessages,
3996  size_t rkmessages_size);
3997 
4007 RD_EXPORT
4009  rd_kafka_queue_t *rkqu,
4010  int timeout_ms,
4011  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque),
4012  void *commit_opaque);
4013 
4014 
4054 RD_EXPORT
4056 rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset);
4057 
4058 
4091 RD_EXPORT rd_kafka_resp_err_t
4092 rd_kafka_offsets_store(rd_kafka_t *rk,
4093  rd_kafka_topic_partition_list_t *offsets);
4094 
4095 
4116 RD_EXPORT
4117 rd_kafka_error_t *rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage);
4118 
4172 RD_EXPORT rd_kafka_resp_err_t
4173 rd_kafka_subscribe(rd_kafka_t *rk,
4174  const rd_kafka_topic_partition_list_t *topics);
4175 
4176 
4180 RD_EXPORT
4182 
4183 
4193 RD_EXPORT rd_kafka_resp_err_t
4194 rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics);
4195 
4196 
4197 
4227 RD_EXPORT
4228 rd_kafka_message_t *rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms);
4229 
4247 RD_EXPORT
4249 
4250 
4268 RD_EXPORT
4269 rd_kafka_error_t *rd_kafka_consumer_close_queue(rd_kafka_t *rk,
4270  rd_kafka_queue_t *rkqu);
4271 
4272 
4281 RD_EXPORT
4282 int rd_kafka_consumer_closed(rd_kafka_t *rk);
4283 
4284 
4302 RD_EXPORT rd_kafka_error_t *
4303 rd_kafka_incremental_assign(rd_kafka_t *rk,
4304  const rd_kafka_topic_partition_list_t *partitions);
4305 
4306 
4324 RD_EXPORT rd_kafka_error_t *rd_kafka_incremental_unassign(
4325  rd_kafka_t *rk,
4326  const rd_kafka_topic_partition_list_t *partitions);
4327 
4328 
4342 RD_EXPORT
4343 const char *rd_kafka_rebalance_protocol(rd_kafka_t *rk);
4344 
4345 
4366 RD_EXPORT rd_kafka_resp_err_t
4367 rd_kafka_assign(rd_kafka_t *rk,
4368  const rd_kafka_topic_partition_list_t *partitions);
4369 
4386 RD_EXPORT rd_kafka_resp_err_t
4387 rd_kafka_assignment(rd_kafka_t *rk,
4388  rd_kafka_topic_partition_list_t **partitions);
4389 
4390 
4407 RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk);
4408 
4409 
4445 RD_EXPORT rd_kafka_resp_err_t
4446 rd_kafka_commit(rd_kafka_t *rk,
4447  const rd_kafka_topic_partition_list_t *offsets,
4448  int async);
4449 
4450 
4457 RD_EXPORT rd_kafka_resp_err_t
4458 rd_kafka_commit_message(rd_kafka_t *rk,
4459  const rd_kafka_message_t *rkmessage,
4460  int async);
4461 
4462 
4485 RD_EXPORT rd_kafka_resp_err_t
4486 rd_kafka_commit_queue(rd_kafka_t *rk,
4487  const rd_kafka_topic_partition_list_t *offsets,
4488  rd_kafka_queue_t *rkqu,
4489  void (*cb)(rd_kafka_t *rk,
4490  rd_kafka_resp_err_t err,
4491  rd_kafka_topic_partition_list_t *offsets,
4492  void *commit_opaque),
4493  void *commit_opaque);
4494 
4495 
4513 RD_EXPORT rd_kafka_resp_err_t
4514 rd_kafka_committed(rd_kafka_t *rk,
4515  rd_kafka_topic_partition_list_t *partitions,
4516  int timeout_ms);
4517 
4518 
4519 
4536 RD_EXPORT rd_kafka_resp_err_t
4537 rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions);
4538 
4539 
4540 
4552 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4553 rd_kafka_consumer_group_metadata(rd_kafka_t *rk);
4554 
4555 
4565 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4566 rd_kafka_consumer_group_metadata_new(const char *group_id);
4567 
4568 
4581 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4583  int32_t generation_id,
4584  const char *member_id,
4585  const char *group_instance_id);
4586 
4587 
4597 RD_EXPORT
4599  const rd_kafka_consumer_group_metadata_t *group_metadata);
4600 
4601 
4606 RD_EXPORT void
4607 rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *);
4608 
4609 
4630 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_write(
4631  const rd_kafka_consumer_group_metadata_t *cgmd,
4632  void **bufferp,
4633  size_t *sizep);
4634 
4655 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_read(
4656  rd_kafka_consumer_group_metadata_t **cgmdp,
4657  const void *buffer,
4658  size_t size);
4659 
4675 #define RD_KAFKA_MSG_F_FREE \
4676  0x1
4678 #define RD_KAFKA_MSG_F_COPY \
4679  0x2
4681 #define RD_KAFKA_MSG_F_BLOCK \
4682  0x4
4693 #define RD_KAFKA_MSG_F_PARTITION \
4694  0x8
4799 RD_EXPORT
4800 int rd_kafka_produce(rd_kafka_topic_t *rkt,
4801  int32_t partition,
4802  int msgflags,
4803  void *payload,
4804  size_t len,
4805  const void *key,
4806  size_t keylen,
4807  void *msg_opaque);
4808 
4809 
4823 RD_EXPORT
4824 rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk, ...);
4825 
4826 
4838 RD_EXPORT
4839 rd_kafka_error_t *
4840 rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt);
4841 
4842 
4872 RD_EXPORT
4873 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt,
4874  int32_t partition,
4875  int msgflags,
4876  rd_kafka_message_t *rkmessages,
4877  int message_cnt);
4878 
4879 
4880 
4905 RD_EXPORT
4906 rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms);
4907 
4908 
4909 
4940 RD_EXPORT
4941 rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags);
4942 
4943 
4951 #define RD_KAFKA_PURGE_F_QUEUE 0x1
4952 
4960 #define RD_KAFKA_PURGE_F_INFLIGHT 0x2
4961 
4962 
4966 #define RD_KAFKA_PURGE_F_NON_BLOCKING 0x4
4967 
4968 
4984  int32_t id;
4985  char *host;
4986  int port;
4987 } rd_kafka_metadata_broker_t;
4988 
4993  int32_t id;
4995  int32_t leader;
4997  int32_t *replicas;
4998  int isr_cnt;
4999  int32_t *isrs;
5000 } rd_kafka_metadata_partition_t;
5001 
5005 typedef struct rd_kafka_metadata_topic {
5006  char *topic;
5010 } rd_kafka_metadata_topic_t;
5011 
5012 
5016 typedef struct rd_kafka_metadata {
5023  int32_t orig_broker_id;
5025 } rd_kafka_metadata_t;
5026 
5047 RD_EXPORT
5049 rd_kafka_metadata(rd_kafka_t *rk,
5050  int all_topics,
5051  rd_kafka_topic_t *only_rkt,
5052  const struct rd_kafka_metadata **metadatap,
5053  int timeout_ms);
5054 
5058 RD_EXPORT
5059 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
5060 
5064 typedef struct rd_kafka_Node_s rd_kafka_Node_t;
5065 
5073 RD_EXPORT
5074 int rd_kafka_Node_id(const rd_kafka_Node_t *node);
5075 
5086 RD_EXPORT
5087 const char *rd_kafka_Node_host(const rd_kafka_Node_t *node);
5088 
5096 RD_EXPORT
5097 uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node);
5098 
5106 RD_EXPORT
5107 const char *rd_kafka_Node_rack(const rd_kafka_Node_t *node);
5108 
5129  char *member_id;
5130  char *client_id;
5131  char *client_host;
5138 };
5139 
5145 typedef enum {
5146  RD_KAFKA_CONSUMER_GROUP_STATE_UNKNOWN = 0,
5147  RD_KAFKA_CONSUMER_GROUP_STATE_PREPARING_REBALANCE = 1,
5148  RD_KAFKA_CONSUMER_GROUP_STATE_COMPLETING_REBALANCE = 2,
5149  RD_KAFKA_CONSUMER_GROUP_STATE_STABLE = 3,
5150  RD_KAFKA_CONSUMER_GROUP_STATE_DEAD = 4,
5151  RD_KAFKA_CONSUMER_GROUP_STATE_EMPTY = 5,
5152  RD_KAFKA_CONSUMER_GROUP_STATE__CNT
5154 
5160  char *group;
5162  char *state;
5164  char *protocol;
5167 };
5168 
5177 };
5178 
5179 
5210 RD_EXPORT
5212 rd_kafka_list_groups(rd_kafka_t *rk,
5213  const char *group,
5214  const struct rd_kafka_group_list **grplistp,
5215  int timeout_ms);
5216 
5224 RD_EXPORT
5225 const char *
5227 
5235 RD_EXPORT
5237 rd_kafka_consumer_group_state_code(const char *name);
5238 
5242 RD_EXPORT
5243 void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist);
5244 
5245 
5289 RD_EXPORT
5290 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
5291 
5292 
5293 
5306 RD_EXPORT RD_DEPRECATED void
5307 rd_kafka_set_logger(rd_kafka_t *rk,
5308  void (*func)(const rd_kafka_t *rk,
5309  int level,
5310  const char *fac,
5311  const char *buf));
5312 
5313 
5323 RD_EXPORT
5324 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
5325 
5326 
5330 RD_EXPORT
5331 void rd_kafka_log_print(const rd_kafka_t *rk,
5332  int level,
5333  const char *fac,
5334  const char *buf);
5335 
5336 
5342 RD_EXPORT
5343 void rd_kafka_log_syslog(const rd_kafka_t *rk,
5344  int level,
5345  const char *fac,
5346  const char *buf);
5347 
5348 
5371 RD_EXPORT
5372 int rd_kafka_outq_len(rd_kafka_t *rk);
5373 
5374 
5375 
5382 RD_EXPORT
5383 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
5384 
5385 
5386 
5392 RD_EXPORT
5393 int rd_kafka_thread_cnt(void);
5394 
5395 
5408 
5409 
5418 RD_EXPORT
5419 int rd_kafka_wait_destroyed(int timeout_ms);
5420 
5421 
5427 RD_EXPORT
5428 int rd_kafka_unittest(void);
5429 
5430 
5447 RD_EXPORT
5449 
5450 
5467 #define RD_KAFKA_EVENT_NONE 0x0
5468 #define RD_KAFKA_EVENT_DR 0x1
5469 #define RD_KAFKA_EVENT_FETCH 0x2
5470 #define RD_KAFKA_EVENT_LOG 0x4
5471 #define RD_KAFKA_EVENT_ERROR 0x8
5472 #define RD_KAFKA_EVENT_REBALANCE 0x10
5473 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
5474 #define RD_KAFKA_EVENT_STATS 0x40
5475 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
5476 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
5477 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT \
5478  102
5479 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
5480 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT \
5481  104
5482 #define RD_KAFKA_EVENT_DELETERECORDS_RESULT 105
5483 #define RD_KAFKA_EVENT_DELETEGROUPS_RESULT 106
5485 #define RD_KAFKA_EVENT_DELETECONSUMERGROUPOFFSETS_RESULT 107
5486 
5487 #define RD_KAFKA_EVENT_OAUTHBEARER_TOKEN_REFRESH 0x100
5488 #define RD_KAFKA_EVENT_BACKGROUND 0x200
5489 #define RD_KAFKA_EVENT_CREATEACLS_RESULT 0x400
5490 #define RD_KAFKA_EVENT_DESCRIBEACLS_RESULT 0x800
5491 #define RD_KAFKA_EVENT_DELETEACLS_RESULT 0x1000
5493 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPS_RESULT 0x2000
5494 
5495 #define RD_KAFKA_EVENT_DESCRIBECONSUMERGROUPS_RESULT 0x4000
5496 
5497 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPOFFSETS_RESULT 0x8000
5498 
5499 #define RD_KAFKA_EVENT_ALTERCONSUMERGROUPOFFSETS_RESULT 0x10000
5500 
5501 #define RD_KAFKA_EVENT_INCREMENTALALTERCONFIGS_RESULT 0x20000
5502 
5503 #define RD_KAFKA_EVENT_DESCRIBEUSERSCRAMCREDENTIALS_RESULT 0x40000
5504 
5505 #define RD_KAFKA_EVENT_ALTERUSERSCRAMCREDENTIALS_RESULT 0x80000
5506 
5507 #define RD_KAFKA_EVENT_DESCRIBETOPICS_RESULT 0x100000
5508 
5509 #define RD_KAFKA_EVENT_DESCRIBECLUSTER_RESULT 0x200000
5510 
5511 #define RD_KAFKA_EVENT_LISTOFFSETS_RESULT 0x400000
5512 
5519 RD_EXPORT
5520 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev);
5521 
5528 RD_EXPORT
5529 const char *rd_kafka_event_name(const rd_kafka_event_t *rkev);
5530 
5531 
5541 RD_EXPORT
5542 void rd_kafka_event_destroy(rd_kafka_event_t *rkev);
5543 
5544 
5560 RD_EXPORT
5561 const rd_kafka_message_t *rd_kafka_event_message_next(rd_kafka_event_t *rkev);
5562 
5563 
5577 RD_EXPORT
5578 size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev,
5579  const rd_kafka_message_t **rkmessages,
5580  size_t size);
5581 
5582 
5590 RD_EXPORT
5591 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev);
5592 
5593 
5605 RD_EXPORT
5606 const char *rd_kafka_event_config_string(rd_kafka_event_t *rkev);
5607 
5608 
5617 RD_EXPORT
5618 rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev);
5619 
5620 
5629 RD_EXPORT
5630 const char *rd_kafka_event_error_string(rd_kafka_event_t *rkev);
5631 
5632 
5641 RD_EXPORT
5642 int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev);
5643 
5644 
5671 RD_EXPORT
5672 void *rd_kafka_event_opaque(rd_kafka_event_t *rkev);
5673 
5674 
5683 RD_EXPORT
5684 int rd_kafka_event_log(rd_kafka_event_t *rkev,
5685  const char **fac,
5686  const char **str,
5687  int *level);
5688 
5689 
5701 RD_EXPORT
5702 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev,
5703  char *dst,
5704  size_t dstsize);
5705 
5706 
5719 RD_EXPORT
5720 const char *rd_kafka_event_stats(rd_kafka_event_t *rkev);
5721 
5722 
5733 RD_EXPORT rd_kafka_topic_partition_list_t *
5734 rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev);
5735 
5736 
5747 RD_EXPORT rd_kafka_topic_partition_t *
5748 rd_kafka_event_topic_partition(rd_kafka_event_t *rkev);
5749 
5750 
5752 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
5754 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
5756 typedef rd_kafka_event_t rd_kafka_CreateAcls_result_t;
5758 typedef rd_kafka_event_t rd_kafka_DescribeAcls_result_t;
5760 typedef rd_kafka_event_t rd_kafka_DeleteAcls_result_t;
5762 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
5764 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
5768 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
5770 typedef rd_kafka_event_t rd_kafka_DeleteRecords_result_t;
5772 typedef rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t;
5776 typedef rd_kafka_event_t rd_kafka_DeleteGroups_result_t;
5784 typedef rd_kafka_event_t rd_kafka_DescribeTopics_result_t;
5786 typedef rd_kafka_event_t rd_kafka_DescribeCluster_result_t;
5792 typedef rd_kafka_event_t rd_kafka_ListOffsets_result_t;
5793 
5803 RD_EXPORT const rd_kafka_CreateTopics_result_t *
5804 rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev);
5805 
5815 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
5816 rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev);
5817 
5827 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
5828 rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev);
5829 
5839 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
5840 rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev);
5841 
5852 rd_kafka_event_IncrementalAlterConfigs_result(rd_kafka_event_t *rkev);
5853 
5863 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
5864 rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev);
5865 
5873 RD_EXPORT const rd_kafka_DeleteRecords_result_t *
5874 rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev);
5875 
5888 RD_EXPORT const rd_kafka_ListConsumerGroups_result_t *
5889 rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev);
5890 
5904 rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev);
5905 
5918 RD_EXPORT const rd_kafka_DescribeTopics_result_t *
5919 rd_kafka_event_DescribeTopics_result(rd_kafka_event_t *rkev);
5920 
5933 RD_EXPORT const rd_kafka_DescribeCluster_result_t *
5934 rd_kafka_event_DescribeCluster_result(rd_kafka_event_t *rkev);
5944 RD_EXPORT const rd_kafka_DeleteGroups_result_t *
5945 rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev);
5946 
5957 rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
5958 
5966 RD_EXPORT const rd_kafka_CreateAcls_result_t *
5967 rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev);
5968 
5976 RD_EXPORT const rd_kafka_DescribeAcls_result_t *
5977 rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev);
5978 
5986 RD_EXPORT const rd_kafka_DeleteAcls_result_t *
5987 rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev);
5988 
6002 rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
6003 
6017 rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
6018 
6031 RD_EXPORT const rd_kafka_ListOffsets_result_t *
6032 rd_kafka_event_ListOffsets_result(rd_kafka_event_t *rkev);
6033 
6034 
6049 
6063 rd_kafka_event_AlterUserScramCredentials_result(rd_kafka_event_t *rkev);
6064 
6074 RD_EXPORT
6075 rd_kafka_event_t *rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms);
6076 
6091 RD_EXPORT
6092 int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms);
6093 
6094 
6139  rd_kafka_conf_t *conf,
6140  void **plug_opaquep,
6141  char *errstr,
6142  size_t errstr_size);
6143 
6225  rd_kafka_conf_t *conf,
6226  const char *name,
6227  const char *val,
6228  char *errstr,
6229  size_t errstr_size,
6230  void *ic_opaque);
6231 
6232 
6255  rd_kafka_conf_t *new_conf,
6256  const rd_kafka_conf_t *old_conf,
6257  size_t filter_cnt,
6258  const char **filter,
6259  void *ic_opaque);
6260 
6261 
6269  void *ic_opaque);
6270 
6271 
6290  rd_kafka_t *rk,
6291  const rd_kafka_conf_t *conf,
6292  void *ic_opaque,
6293  char *errstr,
6294  size_t errstr_size);
6295 
6296 
6304 typedef rd_kafka_resp_err_t(
6305  rd_kafka_interceptor_f_on_destroy_t)(rd_kafka_t *rk, void *ic_opaque);
6306 
6307 
6308 
6330  rd_kafka_t *rk,
6331  rd_kafka_message_t *rkmessage,
6332  void *ic_opaque);
6333 
6357  rd_kafka_t *rk,
6358  rd_kafka_message_t *rkmessage,
6359  void *ic_opaque);
6360 
6361 
6379  rd_kafka_t *rk,
6380  rd_kafka_message_t *rkmessage,
6381  void *ic_opaque);
6382 
6405  rd_kafka_t *rk,
6406  const rd_kafka_topic_partition_list_t *offsets,
6408  void *ic_opaque);
6409 
6410 
6433  rd_kafka_t *rk,
6434  int sockfd,
6435  const char *brokername,
6436  int32_t brokerid,
6437  int16_t ApiKey,
6438  int16_t ApiVersion,
6439  int32_t CorrId,
6440  size_t size,
6441  void *ic_opaque);
6442 
6443 
6470  rd_kafka_t *rk,
6471  int sockfd,
6472  const char *brokername,
6473  int32_t brokerid,
6474  int16_t ApiKey,
6475  int16_t ApiVersion,
6476  int32_t CorrId,
6477  size_t size,
6478  int64_t rtt,
6480  void *ic_opaque);
6481 
6482 
6500  rd_kafka_t *rk,
6501  rd_kafka_thread_type_t thread_type,
6502  const char *thread_name,
6503  void *ic_opaque);
6504 
6505 
6526  rd_kafka_t *rk,
6527  rd_kafka_thread_type_t thread_type,
6528  const char *thread_name,
6529  void *ic_opaque);
6530 
6531 
6547  rd_kafka_t *rk,
6548  int32_t broker_id,
6549  const char *secproto,
6550  const char *name,
6551  int port,
6552  const char *state,
6553  void *ic_opaque);
6554 
6555 
6569  rd_kafka_conf_t *conf,
6570  const char *ic_name,
6572  void *ic_opaque);
6573 
6574 
6588  rd_kafka_conf_t *conf,
6589  const char *ic_name,
6591  void *ic_opaque);
6592 
6607  rd_kafka_conf_t *conf,
6608  const char *ic_name,
6610  void *ic_opaque);
6611 
6612 
6634 RD_EXPORT rd_kafka_resp_err_t
6635 rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf,
6636  const char *ic_name,
6638  void *ic_opaque);
6639 
6640 
6641 
6655  rd_kafka_t *rk,
6656  const char *ic_name,
6658  void *ic_opaque);
6659 
6660 
6673 RD_EXPORT rd_kafka_resp_err_t
6674 rd_kafka_interceptor_add_on_send(rd_kafka_t *rk,
6675  const char *ic_name,
6677  void *ic_opaque);
6678 
6692  rd_kafka_t *rk,
6693  const char *ic_name,
6694  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
6695  void *ic_opaque);
6696 
6697 
6711  rd_kafka_t *rk,
6712  const char *ic_name,
6714  void *ic_opaque);
6715 
6716 
6730  rd_kafka_t *rk,
6731  const char *ic_name,
6733  void *ic_opaque);
6734 
6735 
6749  rd_kafka_t *rk,
6750  const char *ic_name,
6752  void *ic_opaque);
6753 
6754 
6768  rd_kafka_t *rk,
6769  const char *ic_name,
6770  rd_kafka_interceptor_f_on_response_received_t *on_response_received,
6771  void *ic_opaque);
6772 
6773 
6787  rd_kafka_t *rk,
6788  const char *ic_name,
6790  void *ic_opaque);
6791 
6792 
6806  rd_kafka_t *rk,
6807  const char *ic_name,
6809  void *ic_opaque);
6810 
6811 
6824 RD_EXPORT
6826  rd_kafka_t *rk,
6827  const char *ic_name,
6828  rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change,
6829  void *ic_opaque);
6830 
6831 
6832 
6853 RD_EXPORT rd_kafka_resp_err_t
6854 rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres);
6855 
6862 RD_EXPORT const char *
6863 rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres);
6864 
6870 RD_EXPORT const char *
6871 rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres);
6872 
6882 RD_EXPORT const rd_kafka_error_t *
6883 rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres);
6884 
6890 RD_EXPORT const char *
6891 rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres);
6892 
6898 RD_EXPORT const rd_kafka_topic_partition_list_t *
6899 rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres);
6900 
6901 
6948 typedef enum rd_kafka_admin_op_t {
6979 
6992 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
6993 
7002  RD_KAFKA_ISOLATION_LEVEL_READ_UNCOMMITTED = 0,
7003  RD_KAFKA_ISOLATION_LEVEL_READ_COMMITTED = 1
7005 
7025 RD_EXPORT rd_kafka_AdminOptions_t *
7026 rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api);
7027 
7028 
7033 
7034 
7052 RD_EXPORT rd_kafka_resp_err_t
7054  int timeout_ms,
7055  char *errstr,
7056  size_t errstr_size);
7057 
7058 
7085 RD_EXPORT rd_kafka_resp_err_t
7087  int timeout_ms,
7088  char *errstr,
7089  size_t errstr_size);
7090 
7091 
7110 RD_EXPORT rd_kafka_resp_err_t
7112  int true_or_false,
7113  char *errstr,
7114  size_t errstr_size);
7115 
7116 
7144 RD_EXPORT rd_kafka_resp_err_t
7146  int32_t broker_id,
7147  char *errstr,
7148  size_t errstr_size);
7149 
7150 
7163 RD_EXPORT
7165  rd_kafka_AdminOptions_t *options,
7166  int true_or_false);
7167 
7182 RD_EXPORT
7184  rd_kafka_AdminOptions_t *options,
7185  int true_or_false);
7186 
7199 RD_EXPORT
7201  rd_kafka_AdminOptions_t *options,
7202  const rd_kafka_consumer_group_state_t *consumer_group_states,
7203  size_t consumer_group_states_cnt);
7204 
7208 RD_EXPORT
7209 rd_kafka_error_t *
7212 
7217 RD_EXPORT void
7219  void *ev_opaque);
7220 
7221 
7222 
7231  1,
7240  9,
7242  10,
7244  11,
7246  12,
7247  RD_KAFKA_ACL_OPERATION__CNT
7249 
7261 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
7262 
7284 RD_EXPORT rd_kafka_NewTopic_t *rd_kafka_NewTopic_new(const char *topic,
7285  int num_partitions,
7286  int replication_factor,
7287  char *errstr,
7288  size_t errstr_size);
7289 
7294 RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic);
7295 
7296 
7302 RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics,
7303  size_t new_topic_cnt);
7304 
7305 
7327 RD_EXPORT rd_kafka_resp_err_t
7329  int32_t partition,
7330  int32_t *broker_ids,
7331  size_t broker_id_cnt,
7332  char *errstr,
7333  size_t errstr_size);
7334 
7347 RD_EXPORT rd_kafka_resp_err_t
7349  const char *name,
7350  const char *value);
7351 
7352 
7371 RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk,
7372  rd_kafka_NewTopic_t **new_topics,
7373  size_t new_topic_cnt,
7374  const rd_kafka_AdminOptions_t *options,
7375  rd_kafka_queue_t *rkqu);
7376 
7377 
7378 /*
7379  * CreateTopics result type and methods
7380  */
7381 
7390 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_CreateTopics_result_topics(
7391  const rd_kafka_CreateTopics_result_t *result,
7392  size_t *cntp);
7393 
7394 
7395 
7396 /*
7397  * DeleteTopics - delete topics from cluster
7398  *
7399  */
7400 
7402 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
7403 
7413 RD_EXPORT rd_kafka_DeleteTopic_t *rd_kafka_DeleteTopic_new(const char *topic);
7414 
7419 RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic);
7420 
7426 RD_EXPORT void
7428  size_t del_topic_cnt);
7429 
7443 RD_EXPORT
7444 void rd_kafka_DeleteTopics(rd_kafka_t *rk,
7445  rd_kafka_DeleteTopic_t **del_topics,
7446  size_t del_topic_cnt,
7447  const rd_kafka_AdminOptions_t *options,
7448  rd_kafka_queue_t *rkqu);
7449 
7450 
7451 
7452 /*
7453  * DeleteTopics result type and methods
7454  */
7455 
7464 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_DeleteTopics_result_topics(
7465  const rd_kafka_DeleteTopics_result_t *result,
7466  size_t *cntp);
7467 
7468 
7479 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
7480 
7497 RD_EXPORT rd_kafka_NewPartitions_t *
7498 rd_kafka_NewPartitions_new(const char *topic,
7499  size_t new_total_cnt,
7500  char *errstr,
7501  size_t errstr_size);
7502 
7507 RD_EXPORT void
7509 
7515 RD_EXPORT void
7517  size_t new_parts_cnt);
7518 
7542  rd_kafka_NewPartitions_t *new_parts,
7543  int32_t new_partition_idx,
7544  int32_t *broker_ids,
7545  size_t broker_id_cnt,
7546  char *errstr,
7547  size_t errstr_size);
7548 
7549 
7568 RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk,
7569  rd_kafka_NewPartitions_t **new_parts,
7570  size_t new_parts_cnt,
7571  const rd_kafka_AdminOptions_t *options,
7572  rd_kafka_queue_t *rkqu);
7573 
7574 
7575 
7576 /*
7577  * CreatePartitions result type and methods
7578  */
7579 
7588 RD_EXPORT const rd_kafka_topic_result_t **
7590  const rd_kafka_CreatePartitions_result_t *result,
7591  size_t *cntp);
7592 
7628 
7632 
7633 
7637 RD_EXPORT const char *
7639 
7640 
7642 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
7643 
7647 RD_EXPORT const char *
7649 
7654 RD_EXPORT const char *
7656 
7660 RD_EXPORT rd_kafka_ConfigSource_t
7662 
7667 RD_EXPORT int
7669 
7675 RD_EXPORT int
7677 
7685 RD_EXPORT int
7687 
7691 RD_EXPORT int
7693 
7694 
7705 RD_EXPORT const rd_kafka_ConfigEntry_t **
7707  size_t *cntp);
7708 
7709 
7710 
7723 
7739  RD_KAFKA_RESOURCE_PATTERN_TYPE__CNT,
7741 
7747  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SET = 0,
7748  RD_KAFKA_ALTER_CONFIG_OP_TYPE_DELETE = 1,
7749  RD_KAFKA_ALTER_CONFIG_OP_TYPE_APPEND = 2,
7750  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SUBTRACT = 3,
7751  RD_KAFKA_ALTER_CONFIG_OP_TYPE__CNT,
7753 
7757 RD_EXPORT const char *rd_kafka_ResourcePatternType_name(
7758  rd_kafka_ResourcePatternType_t resource_pattern_type);
7759 
7763 RD_EXPORT const char *
7765 
7767 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
7768 
7769 
7778 RD_EXPORT rd_kafka_ConfigResource_t *
7780  const char *resname);
7781 
7786 RD_EXPORT void
7788 
7789 
7795 RD_EXPORT void
7797  size_t config_cnt);
7798 
7799 
7813 RD_EXPORT rd_kafka_resp_err_t
7815  const char *name,
7816  const char *value);
7817 
7818 
7837 RD_EXPORT rd_kafka_error_t *rd_kafka_ConfigResource_add_incremental_config(
7838  rd_kafka_ConfigResource_t *config,
7839  const char *name,
7841  const char *value);
7842 
7843 
7852 RD_EXPORT const rd_kafka_ConfigEntry_t **
7854  size_t *cntp);
7855 
7856 
7857 
7861 RD_EXPORT rd_kafka_ResourceType_t
7863 
7867 RD_EXPORT const char *
7869 
7873 RD_EXPORT rd_kafka_resp_err_t
7875 
7880 RD_EXPORT const char *
7882 
7883 
7884 /*
7885  * AlterConfigs - alter cluster configuration.
7886  *
7887  */
7888 
7889 
7912 RD_EXPORT
7913 void rd_kafka_AlterConfigs(rd_kafka_t *rk,
7914  rd_kafka_ConfigResource_t **configs,
7915  size_t config_cnt,
7916  const rd_kafka_AdminOptions_t *options,
7917  rd_kafka_queue_t *rkqu);
7918 
7919 
7920 /*
7921  * AlterConfigs result type and methods
7922  */
7923 
7938 RD_EXPORT const rd_kafka_ConfigResource_t **
7940  const rd_kafka_AlterConfigs_result_t *result,
7941  size_t *cntp);
7942 
7943 
7944 
7945 /*
7946  * IncrementalAlterConfigs - alter cluster configuration incrementally.
7947  *
7948  */
7949 
7950 
7971 RD_EXPORT
7972 void rd_kafka_IncrementalAlterConfigs(rd_kafka_t *rk,
7973  rd_kafka_ConfigResource_t **configs,
7974  size_t config_cnt,
7975  const rd_kafka_AdminOptions_t *options,
7976  rd_kafka_queue_t *rkqu);
7977 
7978 
7979 /*
7980  * IncrementalAlterConfigs result type and methods
7981  */
7982 
7998 RD_EXPORT const rd_kafka_ConfigResource_t **
8001  size_t *cntp);
8002 
8003 
8004 
8005 /*
8006  * DescribeConfigs - retrieve cluster configuration.
8007  *
8008  */
8009 
8010 
8035 RD_EXPORT
8036 void rd_kafka_DescribeConfigs(rd_kafka_t *rk,
8037  rd_kafka_ConfigResource_t **configs,
8038  size_t config_cnt,
8039  const rd_kafka_AdminOptions_t *options,
8040  rd_kafka_queue_t *rkqu);
8041 
8042 
8043 
8044 /*
8045  * DescribeConfigs result type and methods
8046  */
8047 
8056 RD_EXPORT const rd_kafka_ConfigResource_t **
8058  const rd_kafka_DescribeConfigs_result_t *result,
8059  size_t *cntp);
8060 
8061 
8072 typedef struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t;
8073 
8091  const rd_kafka_topic_partition_list_t *before_offsets);
8092 
8097 RD_EXPORT void
8099 
8105 RD_EXPORT void
8107  size_t del_record_cnt);
8108 
8130 RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk,
8131  rd_kafka_DeleteRecords_t **del_records,
8132  size_t del_record_cnt,
8133  const rd_kafka_AdminOptions_t *options,
8134  rd_kafka_queue_t *rkqu);
8135 
8136 
8137 /*
8138  * DeleteRecords result type and methods
8139  */
8140 
8150 RD_EXPORT const rd_kafka_topic_partition_list_t *
8152  const rd_kafka_DeleteRecords_result_t *result);
8153 
8165 typedef struct rd_kafka_TopicCollection_s rd_kafka_TopicCollection_t;
8166 
8171 typedef struct rd_kafka_TopicPartitionInfo_s rd_kafka_TopicPartitionInfo_t;
8172 
8177 typedef struct rd_kafka_TopicDescription_s rd_kafka_TopicDescription_t;
8178 
8188 RD_EXPORT
8190 rd_kafka_TopicCollection_of_topic_names(const char **topics, size_t topics_cnt);
8191 
8196 RD_EXPORT void
8198 
8213 RD_EXPORT
8214 void rd_kafka_DescribeTopics(rd_kafka_t *rk,
8215  const rd_kafka_TopicCollection_t *topics,
8216  const rd_kafka_AdminOptions_t *options,
8217  rd_kafka_queue_t *rkqu);
8218 
8228 RD_EXPORT
8230  const rd_kafka_DescribeTopics_result_t *result,
8231  size_t *cntp);
8232 
8233 
8244 RD_EXPORT
8246  const rd_kafka_TopicDescription_t *topicdesc,
8247  size_t *cntp);
8248 
8249 
8257 RD_EXPORT
8259  const rd_kafka_TopicPartitionInfo_t *partition);
8260 
8261 
8272 RD_EXPORT
8274  const rd_kafka_TopicPartitionInfo_t *partition);
8275 
8287 RD_EXPORT
8288 const rd_kafka_Node_t **
8290  size_t *cntp);
8291 
8303 RD_EXPORT
8305  const rd_kafka_TopicPartitionInfo_t *partition,
8306  size_t *cntp);
8307 
8320 RD_EXPORT
8322  const rd_kafka_TopicDescription_t *topicdesc,
8323  size_t *cntp);
8324 
8335 RD_EXPORT
8336 const char *
8338 
8348 RD_EXPORT const rd_kafka_Uuid_t *rd_kafka_TopicDescription_topic_id(
8349  const rd_kafka_TopicDescription_t *topicdesc);
8350 
8358 RD_EXPORT
8360  const rd_kafka_TopicDescription_t *topicdesc);
8361 
8372 RD_EXPORT
8373 const rd_kafka_error_t *
8375 
8376 
8396 RD_EXPORT
8397 void rd_kafka_DescribeCluster(rd_kafka_t *rk,
8398  const rd_kafka_AdminOptions_t *options,
8399  rd_kafka_queue_t *rkqu);
8400 
8411 RD_EXPORT
8413  const rd_kafka_DescribeCluster_result_t *result,
8414  size_t *cntp);
8415 
8427 RD_EXPORT
8430  const rd_kafka_DescribeCluster_result_t *result,
8431  size_t *cntp);
8432 
8440 RD_EXPORT
8442  const rd_kafka_DescribeCluster_result_t *result);
8443 
8453 RD_EXPORT
8455  const rd_kafka_DescribeCluster_result_t *result);
8456 
8471 typedef struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t;
8472 
8474 typedef struct rd_kafka_ListConsumerGroupsResult_s
8476 
8487 RD_EXPORT
8488 void rd_kafka_ListConsumerGroups(rd_kafka_t *rk,
8489  const rd_kafka_AdminOptions_t *options,
8490  rd_kafka_queue_t *rkqu);
8491 
8502 RD_EXPORT
8504  const rd_kafka_ConsumerGroupListing_t *grplist);
8505 
8514 RD_EXPORT
8516  const rd_kafka_ConsumerGroupListing_t *grplist);
8517 
8525 RD_EXPORT
8527  const rd_kafka_ConsumerGroupListing_t *grplist);
8528 
8540 RD_EXPORT
8544  size_t *cntp);
8545 
8559 RD_EXPORT
8560 const rd_kafka_error_t **rd_kafka_ListConsumerGroups_result_errors(
8562  size_t *cntp);
8563 
8575 typedef struct rd_kafka_ConsumerGroupDescription_s
8577 
8582 typedef struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t;
8583 
8588 typedef struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t;
8589 
8605 RD_EXPORT
8606 void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk,
8607  const char **groups,
8608  size_t groups_cnt,
8609  const rd_kafka_AdminOptions_t *options,
8610  rd_kafka_queue_t *rkqu);
8611 
8623 RD_EXPORT
8627  size_t *cntp);
8628 
8629 
8640 RD_EXPORT
8642  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8643 
8654 RD_EXPORT
8655 const rd_kafka_error_t *rd_kafka_ConsumerGroupDescription_error(
8656  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8657 
8665 RD_EXPORT
8667  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8668 
8669 
8680 RD_EXPORT
8682  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8683 
8696 RD_EXPORT
8699  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8700  size_t *cntp);
8701 
8709 RD_EXPORT
8711  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8712 
8723 RD_EXPORT
8725  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8726 
8734 RD_EXPORT
8736  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8737 
8750 RD_EXPORT
8752  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8753  size_t idx);
8754 
8765 RD_EXPORT
8767  const rd_kafka_MemberDescription_t *member);
8768 
8779 RD_EXPORT
8781  const rd_kafka_MemberDescription_t *member);
8782 
8793 RD_EXPORT
8795  const rd_kafka_MemberDescription_t *member);
8796 
8807 RD_EXPORT
8808 const char *
8810 
8821 RD_EXPORT
8823  const rd_kafka_MemberDescription_t *member);
8824 
8835 RD_EXPORT
8836 const rd_kafka_topic_partition_list_t *rd_kafka_MemberAssignment_partitions(
8837  const rd_kafka_MemberAssignment_t *assignment);
8838 
8850 typedef struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t;
8851 
8861 RD_EXPORT
8863 
8868 RD_EXPORT
8870 
8876 RD_EXPORT void
8878  size_t del_group_cnt);
8879 
8895 RD_EXPORT
8896 void rd_kafka_DeleteGroups(rd_kafka_t *rk,
8897  rd_kafka_DeleteGroup_t **del_groups,
8898  size_t del_group_cnt,
8899  const rd_kafka_AdminOptions_t *options,
8900  rd_kafka_queue_t *rkqu);
8901 
8902 
8903 
8904 /*
8905  * DeleteGroups result type and methods
8906  */
8907 
8916 RD_EXPORT const rd_kafka_group_result_t **rd_kafka_DeleteGroups_result_groups(
8917  const rd_kafka_DeleteGroups_result_t *result,
8918  size_t *cntp);
8919 
8930 typedef struct rd_kafka_ListConsumerGroupOffsets_s
8932 
8947  const char *group_id,
8948  const rd_kafka_topic_partition_list_t *partitions);
8949 
8955  rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets);
8956 
8963  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8964  size_t list_grpoffset_cnt);
8965 
8983 RD_EXPORT
8985  rd_kafka_t *rk,
8986  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8987  size_t list_grpoffsets_cnt,
8988  const rd_kafka_AdminOptions_t *options,
8989  rd_kafka_queue_t *rkqu);
8990 
8991 
8992 
8993 /*
8994  * ListConsumerGroupOffsets result type and methods
8995  */
8996 
9008 RD_EXPORT const rd_kafka_group_result_t **
9011  size_t *cntp);
9012 
9013 
9014 
9025 typedef struct rd_kafka_AlterConsumerGroupOffsets_s
9027 
9042  const char *group_id,
9043  const rd_kafka_topic_partition_list_t *partitions);
9044 
9050  rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets);
9051 
9058  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9059  size_t alter_grpoffset_cnt);
9060 
9079 RD_EXPORT
9081  rd_kafka_t *rk,
9082  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9083  size_t alter_grpoffsets_cnt,
9084  const rd_kafka_AdminOptions_t *options,
9085  rd_kafka_queue_t *rkqu);
9086 
9087 
9088 
9089 /*
9090  * AlterConsumerGroupOffsets result type and methods
9091  */
9092 
9104 RD_EXPORT const rd_kafka_group_result_t **
9107  size_t *cntp);
9108 
9109 
9110 
9121 typedef struct rd_kafka_DeleteConsumerGroupOffsets_s
9123 
9138  const char *group,
9139  const rd_kafka_topic_partition_list_t *partitions);
9140 
9146  rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets);
9147 
9154  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9155  size_t del_grpoffset_cnt);
9156 
9175 RD_EXPORT
9177  rd_kafka_t *rk,
9178  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9179  size_t del_grpoffsets_cnt,
9180  const rd_kafka_AdminOptions_t *options,
9181  rd_kafka_queue_t *rkqu);
9182 
9183 
9184 
9185 /*
9186  * DeleteConsumerGroupOffsets result type and methods
9187  */
9188 
9197 RD_EXPORT const rd_kafka_group_result_t **
9200  size_t *cntp);
9201 
9215  /* Used to retrieve the offset with the largest timestamp of a partition
9216  * as message timestamps can be specified client side this may not match
9217  * the log end offset returned by SPEC_LATEST.
9218  */
9219  RD_KAFKA_OFFSET_SPEC_MAX_TIMESTAMP = -3,
9220  /* Used to retrieve the offset with the earliest timestamp of a
9221  partition. */
9222  RD_KAFKA_OFFSET_SPEC_EARLIEST = -2,
9223  /* Used to retrieve the offset with the latest timestamp of a partition.
9224  */
9225  RD_KAFKA_OFFSET_SPEC_LATEST = -1,
9227 
9232 typedef struct rd_kafka_ListOffsetsResultInfo_s
9234 
9238 RD_EXPORT
9239 const rd_kafka_topic_partition_t *
9241  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9242 
9246 RD_EXPORT
9248  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9249 
9254 RD_EXPORT
9257  size_t *cntp);
9258 
9283 RD_EXPORT
9284 void rd_kafka_ListOffsets(rd_kafka_t *rk,
9285  rd_kafka_topic_partition_list_t *topic_partitions,
9286  const rd_kafka_AdminOptions_t *options,
9287  rd_kafka_queue_t *rkqu);
9288 
9301  RD_KAFKA_SCRAM_MECHANISM_UNKNOWN = 0,
9302  RD_KAFKA_SCRAM_MECHANISM_SHA_256 = 1,
9303  RD_KAFKA_SCRAM_MECHANISM_SHA_512 = 2,
9304  RD_KAFKA_SCRAM_MECHANISM__CNT
9306 
9312 typedef struct rd_kafka_ScramCredentialInfo_s rd_kafka_ScramCredentialInfo_t;
9313 
9317 RD_EXPORT
9319  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9320 
9324 RD_EXPORT
9326  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9327 
9334 typedef struct rd_kafka_UserScramCredentialsDescription_s
9336 
9340 RD_EXPORT
9342  const rd_kafka_UserScramCredentialsDescription_t *description);
9343 
9347 RD_EXPORT
9348 const rd_kafka_error_t *rd_kafka_UserScramCredentialsDescription_error(
9349  const rd_kafka_UserScramCredentialsDescription_t *description);
9350 
9355 RD_EXPORT
9357  const rd_kafka_UserScramCredentialsDescription_t *description);
9358 
9363 RD_EXPORT
9366  const rd_kafka_UserScramCredentialsDescription_t *description,
9367  size_t idx);
9368 
9378 RD_EXPORT
9382  size_t *cntp);
9383 
9395 RD_EXPORT
9397  rd_kafka_t *rk,
9398  const char **users,
9399  size_t user_cnt,
9400  const rd_kafka_AdminOptions_t *options,
9401  rd_kafka_queue_t *rkqu);
9402 
9406 typedef struct rd_kafka_UserScramCredentialAlteration_s
9408 
9429 RD_EXPORT
9431 rd_kafka_UserScramCredentialUpsertion_new(const char *username,
9432  rd_kafka_ScramMechanism_t mechanism,
9433  int32_t iterations,
9434  const unsigned char *password,
9435  size_t password_size,
9436  const unsigned char *salt,
9437  size_t salt_size);
9438 
9448 RD_EXPORT
9450 rd_kafka_UserScramCredentialDeletion_new(const char *username,
9451  rd_kafka_ScramMechanism_t mechanism);
9452 
9453 
9457 RD_EXPORT
9460 
9464 RD_EXPORT
9467  size_t alteration_cnt);
9468 
9472 typedef struct rd_kafka_AlterUserScramCredentials_result_response_s
9474 
9479 RD_EXPORT
9482 
9487 RD_EXPORT
9488 const rd_kafka_error_t *
9491 
9500 RD_EXPORT
9504  size_t *cntp);
9505 
9519 RD_EXPORT
9521  rd_kafka_t *rk,
9523  size_t alteration_cnt,
9524  const rd_kafka_AdminOptions_t *options,
9525  rd_kafka_queue_t *rkqu);
9526 
9539 typedef struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t;
9540 
9546 
9550 RD_EXPORT const rd_kafka_error_t *
9551 rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres);
9552 
9553 
9557 RD_EXPORT const char *
9559 
9567  1,
9570  RD_KAFKA_ACL_PERMISSION_TYPE__CNT
9572 
9576 RD_EXPORT const char *rd_kafka_AclPermissionType_name(
9577  rd_kafka_AclPermissionType_t acl_permission_type);
9578 
9597 RD_EXPORT rd_kafka_AclBinding_t *
9599  const char *name,
9600  rd_kafka_ResourcePatternType_t resource_pattern_type,
9601  const char *principal,
9602  const char *host,
9603  rd_kafka_AclOperation_t operation,
9604  rd_kafka_AclPermissionType_t permission_type,
9605  char *errstr,
9606  size_t errstr_size);
9607 
9634  rd_kafka_ResourceType_t restype,
9635  const char *name,
9636  rd_kafka_ResourcePatternType_t resource_pattern_type,
9637  const char *principal,
9638  const char *host,
9639  rd_kafka_AclOperation_t operation,
9640  rd_kafka_AclPermissionType_t permission_type,
9641  char *errstr,
9642  size_t errstr_size);
9643 
9647 RD_EXPORT rd_kafka_ResourceType_t
9649 
9655 RD_EXPORT const char *
9657 
9663 RD_EXPORT const char *
9665 
9671 RD_EXPORT const char *
9673 
9677 RD_EXPORT rd_kafka_AclOperation_t
9679 
9685 
9691 
9695 RD_EXPORT const rd_kafka_error_t *
9697 
9698 
9703 RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding);
9704 
9705 
9711 RD_EXPORT void
9713  size_t acl_bindings_cnt);
9714 
9722 RD_EXPORT const rd_kafka_acl_result_t **
9724  size_t *cntp);
9725 
9742 RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk,
9743  rd_kafka_AclBinding_t **new_acls,
9744  size_t new_acls_cnt,
9745  const rd_kafka_AdminOptions_t *options,
9746  rd_kafka_queue_t *rkqu);
9747 
9761 RD_EXPORT const rd_kafka_AclBinding_t **
9763  size_t *cntp);
9764 
9779 RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk,
9780  rd_kafka_AclBindingFilter_t *acl_filter,
9781  const rd_kafka_AdminOptions_t *options,
9782  rd_kafka_queue_t *rkqu);
9783 
9790 typedef struct rd_kafka_DeleteAcls_result_response_s
9792 
9800 RD_EXPORT const rd_kafka_DeleteAcls_result_response_t **
9802  size_t *cntp);
9803 
9808 RD_EXPORT const rd_kafka_error_t *rd_kafka_DeleteAcls_result_response_error(
9809  const rd_kafka_DeleteAcls_result_response_t *result_response);
9810 
9811 
9818 RD_EXPORT const rd_kafka_AclBinding_t **
9820  const rd_kafka_DeleteAcls_result_response_t *result_response,
9821  size_t *matching_acls_cntp);
9822 
9839 RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk,
9840  rd_kafka_AclBindingFilter_t **del_acls,
9841  size_t del_acls_cnt,
9842  const rd_kafka_AdminOptions_t *options,
9843  rd_kafka_queue_t *rkqu);
9844 
9897 RD_EXPORT
9899 rd_kafka_oauthbearer_set_token(rd_kafka_t *rk,
9900  const char *token_value,
9901  int64_t md_lifetime_ms,
9902  const char *md_principal_name,
9903  const char **extensions,
9904  size_t extension_size,
9905  char *errstr,
9906  size_t errstr_size);
9907 
9928 RD_EXPORT
9930  const char *errstr);
9931 
10118 RD_EXPORT
10119 rd_kafka_error_t *rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms);
10120 
10121 
10122 
10167 RD_EXPORT
10168 rd_kafka_error_t *rd_kafka_begin_transaction(rd_kafka_t *rk);
10169 
10170 
10242 RD_EXPORT
10243 rd_kafka_error_t *rd_kafka_send_offsets_to_transaction(
10244  rd_kafka_t *rk,
10245  const rd_kafka_topic_partition_list_t *offsets,
10246  const rd_kafka_consumer_group_metadata_t *cgmetadata,
10247  int timeout_ms);
10248 
10249 
10317 RD_EXPORT
10318 rd_kafka_error_t *rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms);
10319 
10320 
10379 RD_EXPORT
10380 rd_kafka_error_t *rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms);
10381 
10382 
10385 /* @cond NO_DOC */
10386 #ifdef __cplusplus
10387 }
10388 #endif
10389 #endif /* _RDKAFKA_H_ */
10390 /* @endcond NO_DOC */
rd_kafka_produceva
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_kafka_get_err_descs
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_kafka_topic_result_error_string
const RD_EXPORT char * rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres)
rd_kafka_admin_op_t
rd_kafka_admin_op_t
Admin operation enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:6948
rd_kafka_conf_set_dr_msg_cb
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_kafka_new
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...
RD_KAFKA_RESP_ERR__NOOP
@ RD_KAFKA_RESP_ERR__NOOP
Definition: rdkafka.h:405
rd_kafka_ConsumerGroupDescription_t
struct rd_kafka_ConsumerGroupDescription_s rd_kafka_ConsumerGroupDescription_t
DescribeConsumerGroups result type.
Definition: rdkafka.h:8575
rd_kafka_DeleteTopic_destroy_array
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_kafka_header_add
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_kafka_interceptor_f_on_broker_state_change_t
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:6546
rd_kafka_subscription
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_kafka_ConfigResource_add_incremental_config
RD_EXPORT rd_kafka_error_t * rd_kafka_ConfigResource_add_incremental_config(rd_kafka_ConfigResource_t *config, const char *name, rd_kafka_AlterConfigOpType_t op_type, const char *value)
Add the value of the configuration entry for a subsequent incremental alter config operation....
rd_kafka_AdminOptions_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_kafka_thread_cnt
RD_EXPORT int rd_kafka_thread_cnt(void)
Retrieve the current number of threads in use by librdkafka.
rd_kafka_set_log_queue
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_kafka_conf_set_engine_callback_data
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_kafka_metadata::orig_broker_id
int32_t orig_broker_id
Definition: rdkafka.h:5023
rd_kafka_DeleteConsumerGroupOffsets_destroy
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_kafka_unittest
RD_EXPORT int rd_kafka_unittest(void)
Run librdkafka's built-in unit-tests.
RD_KAFKA_RESP_ERR__UNDERFLOW
@ RD_KAFKA_RESP_ERR__UNDERFLOW
Definition: rdkafka.h:377
rd_kafka_purge
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_kafka_DeleteRecords
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_kafka_ListOffsets_result_infos
const RD_EXPORT rd_kafka_ListOffsetsResultInfo_t ** rd_kafka_ListOffsets_result_infos(const rd_kafka_ListOffsets_result_t *result, size_t *cntp)
Returns the array of ListOffsetsResultInfo in result and populates the size of the array in cntp.
RD_KAFKA_RESP_ERR__GAPLESS_GUARANTEE
@ RD_KAFKA_RESP_ERR__GAPLESS_GUARANTEE
Definition: rdkafka.h:391
rd_kafka_consumer_close
RD_EXPORT rd_kafka_resp_err_t rd_kafka_consumer_close(rd_kafka_t *rk)
Close the consumer.
rd_kafka_DeleteAcls_result_responses
const RD_EXPORT 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.
RD_KAFKA_CONFIG_SOURCE_DYNAMIC_DEFAULT_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_DEFAULT_BROKER_CONFIG
Definition: rdkafka.h:7621
rd_kafka_AdminOptions_set_match_consumer_group_states
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.
rd_kafka_msg_partitioner_fnv1a
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_kafka_opaque
RD_EXPORT void * rd_kafka_opaque(const rd_kafka_t *rk)
Retrieves the opaque pointer previously set with rd_kafka_conf_set_opaque()
RD_KAFKA_RESP_ERR__WAIT_CACHE
@ RD_KAFKA_RESP_ERR__WAIT_CACHE
Definition: rdkafka.h:359
rd_kafka_conf_dump_free
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_kafka_interceptor_f_on_destroy_t
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:6305
RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafka.h:498
rd_kafka_topic_conf_set_partitioner_cb
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_kafka_queue_get_consumer
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_consumer(rd_kafka_t *rk)
rd_kafka_topic_partition_s::offset
int64_t offset
Definition: rdkafka.h:916
rd_kafka_AclPermissionType_name
const RD_EXPORT char * rd_kafka_AclPermissionType_name(rd_kafka_AclPermissionType_t acl_permission_type)
rd_kafka_message_destroy
RD_EXPORT void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage)
Frees resources for rkmessage and hands ownership back to rdkafka.
rd_kafka_last_error
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_KAFKA_RESP_ERR_DELEGATION_TOKEN_EXPIRED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_EXPIRED
Definition: rdkafka.h:570
rd_kafka_event_stats
const RD_EXPORT char * rd_kafka_event_stats(rd_kafka_event_t *rkev)
Extract stats from the event.
rd_kafka_DeleteTopics_result_t
rd_kafka_event_t rd_kafka_DeleteTopics_result_t
Definition: rdkafka.h:5754
RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
@ RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
Definition: rdkafka.h:632
RD_KAFKA_RESP_ERR__TRANSPORT
@ RD_KAFKA_RESP_ERR__TRANSPORT
Definition: rdkafka.h:294
rd_kafka_CreateTopics_result_topics
const RD_EXPORT 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_kafka_default_topic_conf_dup
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_kafka_group_info::members
struct rd_kafka_group_member_info * members
Definition: rdkafka.h:5165
rd_kafka_topic_partition_list_new
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_kafka_log_print
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_KAFKA_RESP_ERR_THROTTLING_QUOTA_EXCEEDED
@ RD_KAFKA_RESP_ERR_THROTTLING_QUOTA_EXCEEDED
Definition: rdkafka.h:618
RD_KAFKA_RESOURCE_PATTERN_LITERAL
@ RD_KAFKA_RESOURCE_PATTERN_LITERAL
Definition: rdkafka.h:7736
RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
@ RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafka.h:616
rd_kafka_group_member_info::client_host
char * client_host
Definition: rdkafka.h:5131
RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
@ RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
Definition: rdkafka.h:634
RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
Definition: rdkafka.h:7624
rd_kafka_controllerid
RD_EXPORT int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms)
Returns the current ControllerId as reported in broker metadata.
rd_kafka_topic_partition_list_sort
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_kafka_ListConsumerGroupOffsets_result_t
rd_kafka_event_t rd_kafka_ListConsumerGroupOffsets_result_t
Definition: rdkafka.h:5782
rd_kafka_topic_conf_get
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_kafka_message_status
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_kafka_error_new
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_INVALID_PARTITIONS
@ RD_KAFKA_RESP_ERR_INVALID_PARTITIONS
Definition: rdkafka.h:506
rd_kafka_event_DeleteConsumerGroupOffsets_result
const RD_EXPORT rd_kafka_DeleteConsumerGroupOffsets_result_t * rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get DeleteConsumerGroupOffsets result.
rd_kafka_ConsumerGroupDescription_state
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_kafka_topic_partition_s::opaque
void * opaque
Definition: rdkafka.h:919
rd_kafka_ConfigEntry_is_read_only
RD_EXPORT int rd_kafka_ConfigEntry_is_read_only(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_message_s
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1453
RD_KAFKA_RESP_ERR__INVALID_ARG
@ RD_KAFKA_RESP_ERR__INVALID_ARG
Definition: rdkafka.h:315
rd_kafka_conf_enable_sasl_queue
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_kafka_group_info::protocol_type
char * protocol_type
Definition: rdkafka.h:5163
rd_kafka_AlterConfigOpType_t
rd_kafka_AlterConfigOpType_t
Incremental alter configs operations.
Definition: rdkafka.h:7746
rd_kafka_UserScramCredentialAlteration_destroy
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy(rd_kafka_UserScramCredentialAlteration_t *alteration)
Destroys a UserScramCredentialAlteration given its pointer.
rd_kafka_CreatePartitions_result_topics
const RD_EXPORT 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_KAFKA_ACL_OPERATION_DELETE
@ RD_KAFKA_ACL_OPERATION_DELETE
Definition: rdkafka.h:7236
rd_kafka_mem_malloc
RD_EXPORT void * rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size)
Allocate memory using the same allocator librdkafka uses.
rd_kafka_AclBindingFilter_new
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_KAFKA_ADMIN_OP_INCREMENTALALTERCONFIGS
@ RD_KAFKA_ADMIN_OP_INCREMENTALALTERCONFIGS
Definition: rdkafka.h:6969
rd_kafka_metadata_broker::host
char * host
Definition: rdkafka.h:4985
rd_kafka_NewTopic_new
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_KAFKA_RESP_ERR__UNKNOWN_PROTOCOL
@ RD_KAFKA_RESP_ERR__UNKNOWN_PROTOCOL
Definition: rdkafka.h:345
rd_kafka_ListConsumerGroups
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_kafka_conf_set_ssl_cert_verify_cb
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_kafka_event_type
RD_EXPORT rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev)
rd_kafka_ConsumerGroupListing_is_simple_consumer_group
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_kafka_topic_partition_list_set_offset
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_kafka_AdminOptions_set_validate_only
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_kafka_interceptor_f_on_conf_destroy_t
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:6268
rd_kafka_conf_set_default_topic_conf
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_KAFKA_CERT_CA
@ RD_KAFKA_CERT_CA
Definition: rdkafka.h:2480
rd_kafka_ConsumerGroupDescription_partition_assignor
const RD_EXPORT char * rd_kafka_ConsumerGroupDescription_partition_assignor(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the partition assignor for the grpdesc group.
rd_kafka_topic_conf_set
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_kafka_DescribeUserScramCredentials_result_t
rd_kafka_event_t rd_kafka_DescribeUserScramCredentials_result_t
Definition: rdkafka.h:5788
rd_kafka_group_info::member_cnt
int member_cnt
Definition: rdkafka.h:5166
RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
@ RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
Definition: rdkafka.h:403
RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
@ RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
Definition: rdkafka.h:607
RD_KAFKA_ADMIN_OP_DELETERECORDS
@ RD_KAFKA_ADMIN_OP_DELETERECORDS
Definition: rdkafka.h:6955
rd_kafka_conf_set
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_TopicPartitionInfo_t
struct rd_kafka_TopicPartitionInfo_s rd_kafka_TopicPartitionInfo_t
TopicPartition represents a partition in the DescribeTopics result.
Definition: rdkafka.h:8171
rd_kafka_ConsumerGroupListing_t
struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t
ListConsumerGroups result for a single group.
Definition: rdkafka.h:8471
rd_kafka_group_info::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:5161
rd_kafka_topic_partition_s::topic
char * topic
Definition: rdkafka.h:914
RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
@ RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6973
rd_kafka_message_leader_epoch
RD_EXPORT int32_t rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage)
rd_kafka_offset_store
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_kafka_metadata::topic_cnt
int topic_cnt
Definition: rdkafka.h:5020
rd_kafka_group_list::groups
struct rd_kafka_group_info * groups
Definition: rdkafka.h:5175
RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
@ RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
Definition: rdkafka.h:335
rd_kafka_metadata_partition::leader
int32_t leader
Definition: rdkafka.h:4995
RD_KAFKA_RESP_ERR__FS
@ RD_KAFKA_RESP_ERR__FS
Definition: rdkafka.h:309
rd_kafka_consume_start_queue
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...
rd_kafka_DeleteConsumerGroupOffsets_t
struct rd_kafka_DeleteConsumerGroupOffsets_s rd_kafka_DeleteConsumerGroupOffsets_t
Definition: rdkafka.h:9121
rd_kafka_event_CreateTopics_result
const RD_EXPORT rd_kafka_CreateTopics_result_t * rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev)
Get CreateTopics result.
rd_kafka_AlterConfigs_result_t
rd_kafka_event_t rd_kafka_AlterConfigs_result_t
Definition: rdkafka.h:5764
RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafka.h:472
rd_kafka_group_info::group
char * group
Definition: rdkafka.h:5160
rd_kafka_message_s::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:1454
rd_kafka_produce
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_kafka_group_member_info
Group member information.
Definition: rdkafka.h:5128
rd_kafka_consumer_group_metadata_member_id
const RD_EXPORT char * rd_kafka_consumer_group_metadata_member_id(const rd_kafka_consumer_group_metadata_t *group_metadata)
Get member id of a group metadata.
rd_kafka_MemberDescription_assignment
const RD_EXPORT rd_kafka_MemberAssignment_t * rd_kafka_MemberDescription_assignment(const rd_kafka_MemberDescription_t *member)
Gets assignment of member.
rd_kafka_interceptor_add_on_broker_state_change
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_kafka_DescribeConsumerGroups
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_KAFKA_RESP_ERR__TIMED_OUT
@ RD_KAFKA_RESP_ERR__TIMED_OUT
Definition: rdkafka.h:317
RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
@ RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafka.h:534
RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
@ RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafka.h:541
RD_KAFKA_RESP_ERR__RETRY
@ RD_KAFKA_RESP_ERR__RETRY
Definition: rdkafka.h:381
rd_kafka_header_cnt
RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs)
Returns the number of header key/value pairs.
RD_KAFKA_ACL_OPERATION_READ
@ RD_KAFKA_ACL_OPERATION_READ
Definition: rdkafka.h:7233
rd_kafka_poll_set_consumer
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_kafka_DescribeConfigs
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_ADMIN_OP_DESCRIBECONFIGS
@ RD_KAFKA_ADMIN_OP_DESCRIBECONFIGS
Definition: rdkafka.h:6954
rd_kafka_ConfigResource_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_error(const rd_kafka_ConfigResource_t *config)
rd_kafka_CreatePartitions
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_KAFKA_RESP_ERR_INVALID_CONFIG
@ RD_KAFKA_RESP_ERR_INVALID_CONFIG
Definition: rdkafka.h:512
rd_kafka_consume_batch
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_KAFKA_CONFIG_SOURCE_UNKNOWN_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_UNKNOWN_CONFIG
Definition: rdkafka.h:7614
RD_KAFKA_VTYPE_PARTITION
@ RD_KAFKA_VTYPE_PARTITION
Definition: rdkafka.h:1135
RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
@ RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafka.h:554
RD_KAFKA_RESP_ERR__INCONSISTENT
@ RD_KAFKA_RESP_ERR__INCONSISTENT
Definition: rdkafka.h:389
rd_kafka_DeleteTopics_result_topics
const RD_EXPORT 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_kafka_event_DescribeAcls_result
const RD_EXPORT rd_kafka_DescribeAcls_result_t * rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR_PRODUCER_FENCED
@ RD_KAFKA_RESP_ERR_PRODUCER_FENCED
Definition: rdkafka.h:621
rd_kafka_rebalance_protocol
const RD_EXPORT 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_KAFKA_MSG_STATUS_NOT_PERSISTED
@ RD_KAFKA_MSG_STATUS_NOT_PERSISTED
Definition: rdkafka.h:1621
RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
@ RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
Definition: rdkafka.h:574
RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
@ RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
Definition: rdkafka.h:500
rd_kafka_event_CreateAcls_result
const RD_EXPORT rd_kafka_CreateAcls_result_t * rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev)
rd_kafka_interceptor_f_on_request_sent_t
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:6432
RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
@ RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
Definition: rdkafka.h:549
rd_kafka_msg_partitioner_consistent
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_kafka_event_ListConsumerGroupOffsets_result
const RD_EXPORT rd_kafka_ListConsumerGroupOffsets_result_t * rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get ListConsumerGroupOffsets result.
rd_kafka_topic_partition_s
Topic+Partition place holder.
Definition: rdkafka.h:913
rd_kafka_AdminOptions_set_require_stable_offsets
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_KAFKA_RESP_ERR_DELEGATION_TOKEN_OWNER_MISMATCH
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_OWNER_MISMATCH
Definition: rdkafka.h:564
rd_kafka_err_desc::desc
const char * desc
Definition: rdkafka.h:661
rd_kafka_conf_res_t
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1740
rd_kafka_event_topic_partition
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_event_topic_partition(rd_kafka_event_t *rkev)
rd_kafka_set_log_level
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_KAFKA_RESP_ERR__ASSIGN_PARTITIONS
@ RD_KAFKA_RESP_ERR__ASSIGN_PARTITIONS
Definition: rdkafka.h:337
rd_kafka_queue_forward
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_kafka_DeleteRecords_destroy_array
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_KAFKA_RESP_ERR_MSG_SIZE_TOO_LARGE
@ RD_KAFKA_RESP_ERR_MSG_SIZE_TOO_LARGE
Definition: rdkafka.h:443
rd_kafka_ConfigSource_t
rd_kafka_ConfigSource_t
Apache Kafka config sources.
Definition: rdkafka.h:7611
rd_kafka_metadata_broker::id
int32_t id
Definition: rdkafka.h:4984
RD_KAFKA_RESP_ERR__INTR
@ RD_KAFKA_RESP_ERR__INTR
Definition: rdkafka.h:361
rd_kafka_ConsumerGroupDescription_authorized_operations
const RD_EXPORT rd_kafka_AclOperation_t * rd_kafka_ConsumerGroupDescription_authorized_operations(const rd_kafka_ConsumerGroupDescription_t *grpdesc, size_t *cntp)
Gets the authorized ACL operations for the grpdesc group.
rd_kafka_interceptor_add_on_consume
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_kafka_consumer_closed
RD_EXPORT int rd_kafka_consumer_closed(rd_kafka_t *rk)
rd_kafka_AdminOptions_set_request_timeout
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_kafka_DescribeTopics
RD_EXPORT void rd_kafka_DescribeTopics(rd_kafka_t *rk, const rd_kafka_TopicCollection_t *topics, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe topics as specified by the topics array of size topics_cnt elements.
rd_kafka_consume_start
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_kafka_AlterConsumerGroupOffsets_t
struct rd_kafka_AlterConsumerGroupOffsets_s rd_kafka_AlterConsumerGroupOffsets_t
Definition: rdkafka.h:9025
rd_kafka_topic_name
const RD_EXPORT char * rd_kafka_topic_name(const rd_kafka_topic_t *rkt)
Returns the topic name.
rd_kafka_message_errstr
const RD_EXPORT 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_kafka_conf_set_consume_cb
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_kafka_MemberDescription_t
struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t
Member description included in ConsumerGroupDescription.
Definition: rdkafka.h:8582
rd_kafka_message_s::key_len
size_t key_len
Definition: rdkafka.h:1466
rd_kafka_CreateAcls_result_t
rd_kafka_event_t rd_kafka_CreateAcls_result_t
Definition: rdkafka.h:5756
RD_KAFKA_RESP_ERR_INVALID_MSG
@ RD_KAFKA_RESP_ERR_INVALID_MSG
Definition: rdkafka.h:425
rd_kafka_AclBinding_name
const RD_EXPORT char * rd_kafka_AclBinding_name(const rd_kafka_AclBinding_t *acl)
rd_kafka_name
const RD_EXPORT char * rd_kafka_name(const rd_kafka_t *rk)
Returns Kafka handle name.
rd_kafka_AlterConfigs
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_kafka_abort_transaction
RD_EXPORT rd_kafka_error_t * rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms)
Aborts the ongoing transaction.
rd_kafka_message_headers
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_kafka_DescribeCluster_result_controller
const RD_EXPORT rd_kafka_Node_t * rd_kafka_DescribeCluster_result_controller(const rd_kafka_DescribeCluster_result_t *result)
Gets the current controller for the result cluster.
rd_kafka_DeleteTopics
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_kafka_metadata::orig_broker_name
char * orig_broker_name
Definition: rdkafka.h:5024
rd_kafka_interceptor_f_on_consume_t
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:6378
RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
@ RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
Definition: rdkafka.h:9569
rd_kafka_AlterConsumerGroupOffsets_result_t
rd_kafka_event_t rd_kafka_AlterConsumerGroupOffsets_result_t
Definition: rdkafka.h:5780
rd_kafka_ListConsumerGroupsResult_t
struct rd_kafka_ListConsumerGroupsResult_s rd_kafka_ListConsumerGroupsResult_t
Definition: rdkafka.h:8474
RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
@ RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
Definition: rdkafka.h:449
rd_kafka_interceptor_add_on_commit
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_log
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_kafka_ConsumerGroupDescription_group_id
const RD_EXPORT char * rd_kafka_ConsumerGroupDescription_group_id(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the group id for the grpdesc group.
RD_KAFKA_RESOURCE_PATTERN_ANY
@ RD_KAFKA_RESOURCE_PATTERN_ANY
Definition: rdkafka.h:7732
rd_kafka_topic_result_error
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_kafka_ListOffsetsResultInfo_timestamp
RD_EXPORT int64_t rd_kafka_ListOffsetsResultInfo_timestamp(const rd_kafka_ListOffsetsResultInfo_t *result_info)
Returns the timestamp corresponding to the offset in result_info.
RD_KAFKA_MSG_STATUS_PERSISTED
@ RD_KAFKA_MSG_STATUS_PERSISTED
Definition: rdkafka.h:1631
rd_kafka_group_member_info::member_metadata_size
int member_metadata_size
Definition: rdkafka.h:5134
rd_kafka_conf_set_error_cb
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_kafka_ConfigEntry_source
RD_EXPORT rd_kafka_ConfigSource_t rd_kafka_ConfigEntry_source(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_topic_conf_dump
const RD_EXPORT 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_KAFKA_RESP_ERR__READ_ONLY
@ RD_KAFKA_RESP_ERR__READ_ONLY
Definition: rdkafka.h:373
RD_KAFKA_ADMIN_OP_DELETEGROUPS
@ RD_KAFKA_ADMIN_OP_DELETEGROUPS
Definition: rdkafka.h:6956
rd_kafka_conf_set_oauthbearer_token_refresh_cb
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_kafka_event_name
const RD_EXPORT char * rd_kafka_event_name(const rd_kafka_event_t *rkev)
rd_kafka_queue_get_background
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_background(rd_kafka_t *rk)
RD_KAFKA_RESP_ERR_INVALID_SESSION_TIMEOUT
@ RD_KAFKA_RESP_ERR_INVALID_SESSION_TIMEOUT
Definition: rdkafka.h:484
rd_kafka_AlterUserScramCredentials_result_t
rd_kafka_event_t rd_kafka_AlterUserScramCredentials_result_t
Definition: rdkafka.h:5790
rd_kafka_headers_new
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_new(size_t initial_count)
Create a new headers list.
rd_kafka_metadata_topic::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:5009
rd_kafka_group_member_info::member_id
char * member_id
Definition: rdkafka.h:5129
rd_kafka_AlterConsumerGroupOffsets_destroy
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_kafka_err_desc::name
const char * name
Definition: rdkafka.h:660
rd_kafka_Uuid_destroy
RD_EXPORT void rd_kafka_Uuid_destroy(rd_kafka_Uuid_t *uuid)
Destroy the provided uuid.
RD_KAFKA_VTYPE_END
@ RD_KAFKA_VTYPE_END
Definition: rdkafka.h:1132
rd_kafka_AlterUserScramCredentials_result_responses
const RD_EXPORT rd_kafka_AlterUserScramCredentials_result_response_t ** rd_kafka_AlterUserScramCredentials_result_responses(const rd_kafka_AlterUserScramCredentials_result_t *result, size_t *cntp)
Get an array of responses from a AlterUserScramCredentials result.
rd_kafka_ConfigResource_configs
const RD_EXPORT 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_kafka_ConfigResource_set_config
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_kafka_vu_s::vtype
rd_kafka_vtype_t vtype
Definition: rdkafka.h:1158
rd_kafka_mem_calloc
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_kafka_commit_transaction
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_kafka_ConfigEntry_synonyms
const RD_EXPORT rd_kafka_ConfigEntry_t ** rd_kafka_ConfigEntry_synonyms(const rd_kafka_ConfigEntry_t *entry, size_t *cntp)
RD_KAFKA_ADMIN_OP_CREATEACLS
@ RD_KAFKA_ADMIN_OP_CREATEACLS
Definition: rdkafka.h:6959
RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:598
rd_kafka_CreateTopics_result_t
rd_kafka_event_t rd_kafka_CreateTopics_result_t
Definition: rdkafka.h:5752
RD_KAFKA_RESP_ERR__FATAL
@ RD_KAFKA_RESP_ERR__FATAL
Definition: rdkafka.h:387
rd_kafka_CreateAcls_result_acls
const RD_EXPORT 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_KAFKA_ADMIN_OP_LISTOFFSETS
@ RD_KAFKA_ADMIN_OP_LISTOFFSETS
Definition: rdkafka.h:6976
RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
@ RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
Definition: rdkafka.h:367
rd_kafka_oauthbearer_set_token_failure
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_kafka_ListConsumerGroups_result_errors
const RD_EXPORT 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_kafka_destroy_flags
RD_EXPORT void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags)
Destroy Kafka handle according to specified destroy flags.
rd_kafka_interceptor_add_on_acknowledgement
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_incremental_assign
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_kafka_TopicDescription_topic_id
const RD_EXPORT rd_kafka_Uuid_t * rd_kafka_TopicDescription_topic_id(const rd_kafka_TopicDescription_t *topicdesc)
Gets the topic id for the topicdesc topic.
rd_kafka_conf_set_throttle_cb
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_KAFKA_ADMIN_OP_LISTCONSUMERGROUPOFFSETS
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6965
RD_KAFKA_RESP_ERR_SECURITY_DISABLED
@ RD_KAFKA_RESP_ERR_SECURITY_DISABLED
Definition: rdkafka.h:545
rd_kafka_vtype_t
rd_kafka_vtype_t
Var-arg tag types.
Definition: rdkafka.h:1131
rd_kafka_CreatePartitions_result_t
rd_kafka_event_t rd_kafka_CreatePartitions_result_t
Definition: rdkafka.h:5762
rd_kafka_DescribeUserScramCredentials
RD_EXPORT void rd_kafka_DescribeUserScramCredentials(rd_kafka_t *rk, const char **users, size_t user_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe SASL/SCRAM credentials. This operation is supported by brokers with version 2....
RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS
Definition: rdkafka.h:470
rd_kafka_log_syslog
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.
RD_KAFKA_ADMIN_OP_DELETECONSUMERGROUPOFFSETS
@ RD_KAFKA_ADMIN_OP_DELETECONSUMERGROUPOFFSETS
Definition: rdkafka.h:6958
rd_kafka_ScramCredentialInfo_iterations
RD_EXPORT int32_t rd_kafka_ScramCredentialInfo_iterations(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the iterations of a given ScramCredentialInfo.
rd_kafka_NewTopic_destroy
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_KAFKA_RESP_ERR__NOT_IMPLEMENTED
@ RD_KAFKA_RESP_ERR__NOT_IMPLEMENTED
Definition: rdkafka.h:347
rd_kafka_ListConsumerGroupOffsets_result_groups
const RD_EXPORT 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_kafka_dump
RD_EXPORT void rd_kafka_dump(FILE *fp, rd_kafka_t *rk)
Dumps rdkafka's internal state for handle rk to stream fp.
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_NOT_FOUND
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_NOT_FOUND
Definition: rdkafka.h:562
RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
Definition: rdkafka.h:592
rd_kafka_conf_dump
const RD_EXPORT 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_kafka_ConsumerGroupListing_state
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_kafka_DescribeCluster_result_nodes
const RD_EXPORT rd_kafka_Node_t ** rd_kafka_DescribeCluster_result_nodes(const rd_kafka_DescribeCluster_result_t *result, size_t *cntp)
Gets the broker nodes for the result cluster.
rd_kafka_error_name
const RD_EXPORT char * rd_kafka_error_name(const rd_kafka_error_t *error)
RD_KAFKA_RESOURCE_PATTERN_UNKNOWN
@ RD_KAFKA_RESOURCE_PATTERN_UNKNOWN
Definition: rdkafka.h:7730
RD_KAFKA_RESP_ERR__QUEUE_FULL
@ RD_KAFKA_RESP_ERR__QUEUE_FULL
Definition: rdkafka.h:319
RD_KAFKA_VTYPE_KEY
@ RD_KAFKA_VTYPE_KEY
Definition: rdkafka.h:1137
rd_kafka_consumer_group_metadata_new
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_outq_len
RD_EXPORT int rd_kafka_outq_len(rd_kafka_t *rk)
Returns the current out queue length.
RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
Definition: rdkafka.h:451
rd_kafka_Uuid_most_significant_bits
RD_EXPORT int64_t rd_kafka_Uuid_most_significant_bits(const rd_kafka_Uuid_t *uuid)
Gets most significant 64 bits for the given UUID.
rd_kafka_ResourceType_t
rd_kafka_ResourceType_t
Apache Kafka resource types.
Definition: rdkafka.h:7715
rd_kafka_AclBinding_restype
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_AclBinding_restype(const rd_kafka_AclBinding_t *acl)
rd_kafka_AlterUserScramCredentials_result_response_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_AlterUserScramCredentials_result_response_error(const rd_kafka_AlterUserScramCredentials_result_response_t *response)
Returns the error of a rd_kafka_AlterUserScramCredentials_result_response.
RD_KAFKA_CONF_OK
@ RD_KAFKA_CONF_OK
Definition: rdkafka.h:1745
rd_kafka_conf_set_log_cb
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_kafka_group_result_error
const RD_EXPORT 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_KAFKA_ACL_PERMISSION_TYPE_DENY
@ RD_KAFKA_ACL_PERMISSION_TYPE_DENY
Definition: rdkafka.h:9568
rd_kafka_CreateTopics
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_kafka_DeleteRecords_new
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_kafka_msg_partitioner_consistent_random
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_KAFKA_RESP_ERR__PURGE_INFLIGHT
@ RD_KAFKA_RESP_ERR__PURGE_INFLIGHT
Definition: rdkafka.h:385
RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
@ RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
Definition: rdkafka.h:603
rd_kafka_consumer_group_metadata_read
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_kafka_ConsumerGroupDescription_is_simple_consumer_group
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_KAFKA_RESP_ERR_TOPIC_ALREADY_EXISTS
@ RD_KAFKA_RESP_ERR_TOPIC_ALREADY_EXISTS
Definition: rdkafka.h:504
rd_kafka_interceptor_f_on_send_t
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:6329
rd_kafka_offsets_for_times
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_THREAD_MAIN
@ RD_KAFKA_THREAD_MAIN
Definition: rdkafka.h:5404
RD_KAFKA_THREAD_BACKGROUND
@ RD_KAFKA_THREAD_BACKGROUND
Definition: rdkafka.h:5405
rd_kafka_DeleteAcls
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_kafka_ConsumerGroupDescription_coordinator
const RD_EXPORT rd_kafka_Node_t * rd_kafka_ConsumerGroupDescription_coordinator(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the coordinator for the grpdesc group.
rd_kafka_consumer_group_metadata
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata(rd_kafka_t *rk)
RD_KAFKA_RESP_ERR_INVALID_PRINCIPAL_TYPE
@ RD_KAFKA_RESP_ERR_INVALID_PRINCIPAL_TYPE
Definition: rdkafka.h:572
rd_kafka_AdminOptions_destroy
RD_EXPORT void rd_kafka_AdminOptions_destroy(rd_kafka_AdminOptions_t *options)
Destroy a AdminOptions object.
rd_kafka_message_timestamp
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_kafka_DeleteGroup_new
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_kafka_TopicCollection_destroy
RD_EXPORT void rd_kafka_TopicCollection_destroy(rd_kafka_TopicCollection_t *topics)
Destroy and free a TopicCollection object created with rd_kafka_TopicCollection_new_* methods.
rd_kafka_interceptor_f_on_thread_start_t
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:6499
rd_kafka_metadata_broker::port
int port
Definition: rdkafka.h:4986
rd_kafka_AclPermissionType_t
rd_kafka_AclPermissionType_t
Apache Kafka ACL permission types.
Definition: rdkafka.h:9564
RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
@ RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6967
rd_kafka_DeleteGroup_destroy_array
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_KAFKA_RESP_ERR_OFFSET_OUT_OF_RANGE
@ RD_KAFKA_RESP_ERR_OFFSET_OUT_OF_RANGE
Definition: rdkafka.h:423
rd_kafka_DescribeCluster_result_t
rd_kafka_event_t rd_kafka_DescribeCluster_result_t
Definition: rdkafka.h:5786
rd_kafka_AclBindingFilter_t
rd_kafka_AclBinding_t rd_kafka_AclBindingFilter_t
ACL Binding filter is used to filter access control lists.
Definition: rdkafka.h:9545
RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
Definition: rdkafka.h:7618
rd_kafka_conf_set_connect_cb
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_kafka_ListConsumerGroupOffsets_new
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_group_list::group_cnt
int group_cnt
Definition: rdkafka.h:5176
RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
@ RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
Definition: rdkafka.h:363
rd_kafka_commit_message
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_AlterConsumerGroupOffsets_destroy_array
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_RESP_ERR__BEGIN
@ RD_KAFKA_RESP_ERR__BEGIN
Definition: rdkafka.h:284
RD_KAFKA_RESP_ERR__BAD_COMPRESSION
@ RD_KAFKA_RESP_ERR__BAD_COMPRESSION
Definition: rdkafka.h:288
rd_kafka_topic_conf_new
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_new(void)
Create topic configuration object.
RD_KAFKA_RESP_ERR_UNKNOWN_MEMBER_ID
@ RD_KAFKA_RESP_ERR_UNKNOWN_MEMBER_ID
Definition: rdkafka.h:482
rd_kafka_interceptor_f_on_thread_exit_t
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:6525
RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
@ RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
Definition: rdkafka.h:528
RD_KAFKA_VTYPE_HEADERS
@ RD_KAFKA_VTYPE_HEADERS
Definition: rdkafka.h:1147
rd_kafka_metadata_partition::replica_cnt
int replica_cnt
Definition: rdkafka.h:4996
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafka.h:566
RD_KAFKA_RESP_ERR_INVALID_RECORD
@ RD_KAFKA_RESP_ERR_INVALID_RECORD
Definition: rdkafka.h:614
rd_kafka_error_txn_requires_abort
RD_EXPORT int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error)
RD_KAFKA_RESP_ERR_TOPIC_DELETION_DISABLED
@ RD_KAFKA_RESP_ERR_TOPIC_DELETION_DISABLED
Definition: rdkafka.h:584
rd_kafka_interceptor_add_on_request_sent
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_KAFKA_RESP_ERR__NOT_CONFIGURED
@ RD_KAFKA_RESP_ERR__NOT_CONFIGURED
Definition: rdkafka.h:397
rd_kafka_metadata_broker
Broker information.
Definition: rdkafka.h:4983
rd_kafka_conf_get_default_topic_conf
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_kafka_version
RD_EXPORT int rd_kafka_version(void)
Returns the librdkafka version as integer.
rd_kafka_topic_partition_list_s
A growable list of Topic+Partitions.
Definition: rdkafka.h:964
rd_kafka_NewTopic_t
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:7261
rd_kafka_conf_set_background_event_cb
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_kafka_wait_destroyed
RD_EXPORT int rd_kafka_wait_destroyed(int timeout_ms)
Wait for all rd_kafka_t objects to be destroyed.
RD_KAFKA_RESP_ERR_GROUP_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_GROUP_AUTHORIZATION_FAILED
Definition: rdkafka.h:492
rd_kafka_subscribe
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.
rd_kafka_AdminOptions_set_operation_timeout
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_KAFKA_RESP_ERR_OFFSET_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_OFFSET_NOT_AVAILABLE
Definition: rdkafka.h:594
rd_kafka_ConfigResource_new
RD_EXPORT rd_kafka_ConfigResource_t * rd_kafka_ConfigResource_new(rd_kafka_ResourceType_t restype, const char *resname)
Create new ConfigResource object.
rd_kafka_conf_new
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_new(void)
Create configuration object.
RD_KAFKA_RESP_ERR_UNSUPPORTED_COMPRESSION_TYPE
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_COMPRESSION_TYPE
Definition: rdkafka.h:590
rd_kafka_destroy
RD_EXPORT void rd_kafka_destroy(rd_kafka_t *rk)
Destroy Kafka handle.
rd_kafka_metadata_topic::partitions
struct rd_kafka_metadata_partition * partitions
Definition: rdkafka.h:5008
rd_kafka_topic_partition_destroy
RD_EXPORT void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar)
Destroy a rd_kafka_topic_partition_t.
RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPS
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPS
Definition: rdkafka.h:6962
RD_KAFKA_TIMESTAMP_LOG_APPEND_TIME
@ RD_KAFKA_TIMESTAMP_LOG_APPEND_TIME
Definition: rdkafka.h:223
rd_kafka_commit
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_KAFKA_RESP_ERR__FAIL
@ RD_KAFKA_RESP_ERR__FAIL
Definition: rdkafka.h:292
rd_kafka_event_DeleteGroups_result
const RD_EXPORT rd_kafka_DeleteGroups_result_t * rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev)
Get DeleteGroups result.
RD_KAFKA_RESP_ERR_CLUSTER_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_CLUSTER_AUTHORIZATION_FAILED
Definition: rdkafka.h:494
rd_kafka_conf_interceptor_add_on_conf_destroy
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_kafka_Uuid_copy
RD_EXPORT rd_kafka_Uuid_t * rd_kafka_Uuid_copy(const rd_kafka_Uuid_t *uuid)
Copies the given UUID.
RD_KAFKA_RESP_ERR_INVALID_REPLICATION_FACTOR
@ RD_KAFKA_RESP_ERR_INVALID_REPLICATION_FACTOR
Definition: rdkafka.h:508
rd_kafka_event_IncrementalAlterConfigs_result
const RD_EXPORT rd_kafka_IncrementalAlterConfigs_result_t * rd_kafka_event_IncrementalAlterConfigs_result(rd_kafka_event_t *rkev)
Get IncrementalAlterConfigs result.
rd_kafka_interceptor_add_on_send
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_KAFKA_RESP_ERR__UNKNOWN_BROKER
@ RD_KAFKA_RESP_ERR__UNKNOWN_BROKER
Definition: rdkafka.h:395
rd_kafka_DescribeUserScramCredentials_result_descriptions
const RD_EXPORT rd_kafka_UserScramCredentialsDescription_t ** rd_kafka_DescribeUserScramCredentials_result_descriptions(const rd_kafka_DescribeUserScramCredentials_result_t *result, size_t *cntp)
Get an array of descriptions from a DescribeUserScramCredentials result.
RD_KAFKA_RESP_ERR_LISTENER_NOT_FOUND
@ RD_KAFKA_RESP_ERR_LISTENER_NOT_FOUND
Definition: rdkafka.h:582
RD_KAFKA_ADMIN_OP_DESCRIBETOPICS
@ RD_KAFKA_ADMIN_OP_DESCRIBETOPICS
Definition: rdkafka.h:6974
RD_KAFKA_VTYPE_TOPIC
@ RD_KAFKA_VTYPE_TOPIC
Definition: rdkafka.h:1133
rd_kafka_metadata_partition::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:4994
rd_kafka_AdminOptions_set_include_authorized_operations
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_include_authorized_operations(rd_kafka_AdminOptions_t *options, int true_or_false)
Whether broker should return authorized operations for the given resource in the DescribeConsumerGrou...
rd_kafka_msg_partitioner_random
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_KAFKA_RESOURCE_BROKER
@ RD_KAFKA_RESOURCE_BROKER
Definition: rdkafka.h:7720
rd_kafka_send_offsets_to_transaction
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_kafka_ConfigResource_destroy
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_KAFKA_RESP_ERR_RESOURCE_NOT_FOUND
@ RD_KAFKA_RESP_ERR_RESOURCE_NOT_FOUND
Definition: rdkafka.h:623
rd_kafka_OffsetSpec_t
rd_kafka_OffsetSpec_t
Allows to specify the desired offsets when using ListOffsets.
Definition: rdkafka.h:9214
rd_kafka_metadata::brokers
struct rd_kafka_metadata_broker * brokers
Definition: rdkafka.h:5018
rd_kafka_topic_partition_s::metadata
void * metadata
Definition: rdkafka.h:917
rd_kafka_message_s::payload
void * payload
Definition: rdkafka.h:1457
rd_kafka_ConfigResource_error_string
const RD_EXPORT char * rd_kafka_ConfigResource_error_string(const rd_kafka_ConfigResource_t *config)
RD_KAFKA_CONFIG_SOURCE__CNT
@ RD_KAFKA_CONFIG_SOURCE__CNT
Definition: rdkafka.h:7630
RD_KAFKA_RESOURCE_GROUP
@ RD_KAFKA_RESOURCE_GROUP
Definition: rdkafka.h:7719
rd_kafka_DescribeConfigs_result_resources
const RD_EXPORT 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_kafka_DeleteRecords_t
struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t
Definition: rdkafka.h:8072
rd_kafka_mem_free
RD_EXPORT void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr)
Free pointer returned by librdkafka.
rd_kafka_event_DescribeUserScramCredentials_result
const RD_EXPORT rd_kafka_DescribeUserScramCredentials_result_t * rd_kafka_event_DescribeUserScramCredentials_result(rd_kafka_event_t *rkev)
Get DescribeUserScramCredentials result.
rd_kafka_type
RD_EXPORT rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk)
Returns Kafka handle type.
rd_kafka_conf_set_ssl_cert
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_kafka_group_member_info::member_assignment_size
int member_assignment_size
Definition: rdkafka.h:5137
rd_kafka_TopicDescription_name
const RD_EXPORT char * rd_kafka_TopicDescription_name(const rd_kafka_TopicDescription_t *topicdesc)
Gets the topic name for the topicdesc topic.
rd_kafka_group_info::broker
struct rd_kafka_metadata_broker broker
Definition: rdkafka.h:5159
RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:558
rd_kafka_Uuid_least_significant_bits
RD_EXPORT int64_t rd_kafka_Uuid_least_significant_bits(const rd_kafka_Uuid_t *uuid)
Gets least significant 64 bits for the given UUID.
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTH_DISABLED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTH_DISABLED
Definition: rdkafka.h:560
rd_kafka_brokers_add
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_kafka_topic_partition_available
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_kafka_queue_length
RD_EXPORT size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu)
RD_KAFKA_RESP_ERR_FETCH_SESSION_ID_NOT_FOUND
@ RD_KAFKA_RESP_ERR_FETCH_SESSION_ID_NOT_FOUND
Definition: rdkafka.h:578
RD_KAFKA_ACL_OPERATION_WRITE
@ RD_KAFKA_ACL_OPERATION_WRITE
Definition: rdkafka.h:7234
rd_kafka_message_s::offset
int64_t offset
Definition: rdkafka.h:1468
rd_kafka_DescribeTopics_result_topics
const RD_EXPORT rd_kafka_TopicDescription_t ** rd_kafka_DescribeTopics_result_topics(const rd_kafka_DescribeTopics_result_t *result, size_t *cntp)
Get an array of topic results from a DescribeTopics result.
rd_kafka_consumer_group_state_t
rd_kafka_consumer_group_state_t
Consumer group state.
Definition: rdkafka.h:5145
rd_kafka_queue_destroy
RD_EXPORT void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu)
RD_KAFKA_RESP_ERR__OUTDATED
@ RD_KAFKA_RESP_ERR__OUTDATED
Definition: rdkafka.h:353
rd_kafka_event_AlterUserScramCredentials_result
const RD_EXPORT rd_kafka_AlterUserScramCredentials_result_t * rd_kafka_event_AlterUserScramCredentials_result(rd_kafka_event_t *rkev)
Get AlterUserScramCredentials result.
rd_kafka_queue_yield
RD_EXPORT void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu)
Cancels the current rd_kafka_queue_poll() on rkqu.
rd_kafka_errno
RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void)
Returns the thread-local system errno.
rd_kafka_ListOffsetsResultInfo_t
struct rd_kafka_ListOffsetsResultInfo_s rd_kafka_ListOffsetsResultInfo_t
Information returned from a ListOffsets call for a specific rd_kafka_topic_partition_t.
Definition: rdkafka.h:9232
rd_kafka_message_latency
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_KAFKA_RESOURCE_TOPIC
@ RD_KAFKA_RESOURCE_TOPIC
Definition: rdkafka.h:7718
rd_kafka_ResourcePatternType_name
const RD_EXPORT char * rd_kafka_ResourcePatternType_name(rd_kafka_ResourcePatternType_t resource_pattern_type)
RD_KAFKA_RESP_ERR_INVALID_TIMESTAMP
@ RD_KAFKA_RESP_ERR_INVALID_TIMESTAMP
Definition: rdkafka.h:496
rd_kafka_event_error_string
const RD_EXPORT char * rd_kafka_event_error_string(rd_kafka_event_t *rkev)
rd_kafka_pause_partitions
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_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
@ RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafka.h:537
rd_kafka_DeleteRecords_result_offsets
const RD_EXPORT 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_KAFKA_RESP_ERR_UNKNOWN_TOPIC_OR_PART
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_OR_PART
Definition: rdkafka.h:427
RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
@ RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafka.h:447
rd_kafka_group_list_destroy
RD_EXPORT void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist)
Release list memory.
rd_kafka_event_debug_contexts
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_KAFKA_RESP_ERR_BROKER_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_BROKER_NOT_AVAILABLE
Definition: rdkafka.h:439
RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_ID
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_ID
Definition: rdkafka.h:638
rd_kafka_header_get
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_kafka_NewTopic_destroy_array
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_conf_set_socket_cb
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_kafka_event_ListConsumerGroups_result
const RD_EXPORT rd_kafka_ListConsumerGroups_result_t * rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev)
Get ListConsumerGroups result.
rd_kafka_event_error_is_fatal
RD_EXPORT int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR__AUTHENTICATION
@ RD_KAFKA_RESP_ERR__AUTHENTICATION
Definition: rdkafka.h:349
rd_kafka_AclBinding_t
struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t
ACL Binding is used to create access control lists.
Definition: rdkafka.h:9539
rd_kafka_UserScramCredentialsDescription_scramcredentialinfo_count
RD_EXPORT size_t rd_kafka_UserScramCredentialsDescription_scramcredentialinfo_count(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the count of ScramCredentialInfos of a UserScramCredentialsDescription.
rd_kafka_DeleteConsumerGroupOffsets_result_t
rd_kafka_event_t rd_kafka_DeleteConsumerGroupOffsets_result_t
Definition: rdkafka.h:5778
rd_kafka_consume_stop
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_kafka_conf_get
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_KAFKA_RESP_ERR_LEADER_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:431
RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
@ RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
Definition: rdkafka.h:329
RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
@ RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafka.h:636
rd_kafka_topic_partition_list_s::elems
rd_kafka_topic_partition_t * elems
Definition: rdkafka.h:967
RD_KAFKA_TIMESTAMP_CREATE_TIME
@ RD_KAFKA_TIMESTAMP_CREATE_TIME
Definition: rdkafka.h:222
rd_kafka_metadata_partition::id
int32_t id
Definition: rdkafka.h:4993
RD_KAFKA_RESP_ERR__RESOLVE
@ RD_KAFKA_RESP_ERR__RESOLVE
Definition: rdkafka.h:298
rd_kafka_ConsumerGroupDescription_member
const RD_EXPORT rd_kafka_MemberDescription_t * rd_kafka_ConsumerGroupDescription_member(const rd_kafka_ConsumerGroupDescription_t *grpdesc, size_t idx)
Gets a member of grpdesc group.
rd_kafka_TopicDescription_partitions
const RD_EXPORT rd_kafka_TopicPartitionInfo_t ** rd_kafka_TopicDescription_partitions(const rd_kafka_TopicDescription_t *topicdesc, size_t *cntp)
Gets an array of partitions for the topicdesc topic.
RD_KAFKA_RESP_ERR_INVALID_COMMIT_OFFSET_SIZE
@ RD_KAFKA_RESP_ERR_INVALID_COMMIT_OFFSET_SIZE
Definition: rdkafka.h:488
RD_KAFKA_ADMIN_OP_DELETEACLS
@ RD_KAFKA_ADMIN_OP_DELETEACLS
Definition: rdkafka.h:6961
rd_kafka_assign
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.
rd_kafka_DeleteAcls_result_response_t
struct rd_kafka_DeleteAcls_result_response_s rd_kafka_DeleteAcls_result_response_t
Definition: rdkafka.h:9790
RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafka.h:605
rd_kafka_acl_result_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres)
rd_kafka_interceptor_add_on_thread_exit
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_KAFKA_ACL_OPERATION_ALTER_CONFIGS
@ RD_KAFKA_ACL_OPERATION_ALTER_CONFIGS
Definition: rdkafka.h:7243
rd_kafka_NewPartitions_t
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:7479
rd_kafka_topic_partition_list_del_by_idx
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_kafka_UserScramCredentialAlteration_t
struct rd_kafka_UserScramCredentialAlteration_s rd_kafka_UserScramCredentialAlteration_t
A request to alter a user's SASL/SCRAM credentials.
Definition: rdkafka.h:9406
rd_kafka_TopicDescription_authorized_operations
const RD_EXPORT rd_kafka_AclOperation_t * rd_kafka_TopicDescription_authorized_operations(const rd_kafka_TopicDescription_t *topicdesc, size_t *cntp)
Gets the topic authorized ACL operations for the topicdesc topic.
rd_kafka_metadata
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_KAFKA_RESP_ERR_OPERATION_NOT_ATTEMPTED
@ RD_KAFKA_RESP_ERR_OPERATION_NOT_ATTEMPTED
Definition: rdkafka.h:547
RD_KAFKA_VTYPE_TIMESTAMP
@ RD_KAFKA_VTYPE_TIMESTAMP
Definition: rdkafka.h:1144
rd_kafka_AdminOptions_t
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:6992
rd_kafka_Uuid_base64str
const RD_EXPORT char * rd_kafka_Uuid_base64str(const rd_kafka_Uuid_t *uuid)
Computes base64 encoding for the given uuid string.
RD_KAFKA_RESP_ERR_DUPLICATE_SEQUENCE_NUMBER
@ RD_KAFKA_RESP_ERR_DUPLICATE_SEQUENCE_NUMBER
Definition: rdkafka.h:524
RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
Definition: rdkafka.h:486
rd_kafka_conf_dup
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_kafka_AlterUserScramCredentials_result_response_user
const RD_EXPORT char * rd_kafka_AlterUserScramCredentials_result_response_user(const rd_kafka_AlterUserScramCredentials_result_response_t *response)
Returns the username for a rd_kafka_AlterUserScramCredentials_result_response.
rd_kafka_UserScramCredentialsDescription_user
const RD_EXPORT char * rd_kafka_UserScramCredentialsDescription_user(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the username of a UserScramCredentialsDescription.
rd_kafka_ConfigResource_destroy_array
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_kafka_get_debug_contexts
const RD_EXPORT char * rd_kafka_get_debug_contexts(void)
Retrieve supported debug contexts for use with the "debug" configuration property....
rd_kafka_group_member_info::member_metadata
void * member_metadata
Definition: rdkafka.h:5132
rd_kafka_consumer_group_metadata_new_with_genid
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_KAFKA_RESP_ERR_ILLEGAL_GENERATION
@ RD_KAFKA_RESP_ERR_ILLEGAL_GENERATION
Definition: rdkafka.h:476
rd_kafka_topic_partition_get_leader_epoch
RD_EXPORT int32_t rd_kafka_topic_partition_get_leader_epoch(const rd_kafka_topic_partition_t *rktpar)
rd_kafka_offset_store_message
RD_EXPORT rd_kafka_error_t * rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage)
Store offset +1 for the consumed message.
RD_KAFKA_RESP_ERR__CONFLICT
@ RD_KAFKA_RESP_ERR__CONFLICT
Definition: rdkafka.h:341
RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
@ RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafka.h:468
rd_kafka_TopicDescription_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_TopicDescription_error(const rd_kafka_TopicDescription_t *topicdesc)
Gets the error for the topicdesc topic.
rd_kafka_topic_conf_set_opaque
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_kafka_DescribeConfigs_result_t
rd_kafka_event_t rd_kafka_DescribeConfigs_result_t
Definition: rdkafka.h:5768
RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafka.h:526
rd_kafka_Uuid_new
RD_EXPORT rd_kafka_Uuid_t * rd_kafka_Uuid_new(int64_t most_significant_bits, int64_t least_significant_bits)
Creates a new UUID.
rd_kafka_clusterid
RD_EXPORT char * rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms)
Returns the ClusterId as reported in broker metadata.
RD_KAFKA_RESP_ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
Definition: rdkafka.h:543
RD_KAFKA_ADMIN_OP_DELETETOPICS
@ RD_KAFKA_ADMIN_OP_DELETETOPICS
Definition: rdkafka.h:6951
rd_kafka_AlterUserScramCredentials_result_response_t
struct rd_kafka_AlterUserScramCredentials_result_response_s rd_kafka_AlterUserScramCredentials_result_response_t
Result of a single user SCRAM alteration.
Definition: rdkafka.h:9472
rd_kafka_topic_conf_set_msg_order_cmp
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_kafka_queue_poll_callback
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_kafka_conf
const RD_EXPORT rd_kafka_conf_t * rd_kafka_conf(rd_kafka_t *rk)
rd_kafka_set_logger
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_kafka_AlterConsumerGroupOffsets_result_groups
const RD_EXPORT 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_kafka_DeleteConsumerGroupOffsets_new
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_KAFKA_RESP_ERR__APPLICATION
@ RD_KAFKA_RESP_ERR__APPLICATION
Definition: rdkafka.h:401
RD_KAFKA_RESP_ERR__INVALID_DIFFERENT_RECORD
@ RD_KAFKA_RESP_ERR__INVALID_DIFFERENT_RECORD
Definition: rdkafka.h:412
rd_kafka_conf_set_rebalance_cb
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_kafka_IncrementalAlterConfigs_result_resources
const RD_EXPORT rd_kafka_ConfigResource_t ** rd_kafka_IncrementalAlterConfigs_result_resources(const rd_kafka_IncrementalAlterConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a IncrementalAlterConfigs result.
rd_kafka_AclOperation_name
const RD_EXPORT char * rd_kafka_AclOperation_name(rd_kafka_AclOperation_t acl_operation)
rd_kafka_topic_partition_s::metadata_size
size_t metadata_size
Definition: rdkafka.h:918
rd_kafka_DescribeConsumerGroups_result_t
rd_kafka_event_t rd_kafka_DescribeConsumerGroups_result_t
Definition: rdkafka.h:5774
rd_kafka_AlterConsumerGroupOffsets
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_KAFKA_RESP_ERR__DESTROY
@ RD_KAFKA_RESP_ERR__DESTROY
Definition: rdkafka.h:290
rd_kafka_conf_interceptor_add_on_new
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_kafka_conf_set_closesocket_cb
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_kafka_NewTopic_set_config
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_kafka_ListConsumerGroups_result_valid
const RD_EXPORT 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_kafka_incremental_unassign
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_kafka_event_DeleteAcls_result
const RD_EXPORT rd_kafka_DeleteAcls_result_t * rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR_MEMBER_ID_REQUIRED
@ RD_KAFKA_RESP_ERR_MEMBER_ID_REQUIRED
Definition: rdkafka.h:596
rd_kafka_msg_partitioner_murmur2
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).
rd_kafka_Node_id
RD_EXPORT int rd_kafka_Node_id(const rd_kafka_Node_t *node)
Get the id of node.
RD_KAFKA_RESP_ERR_STALE_MEMBER_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_MEMBER_EPOCH
Definition: rdkafka.h:648
RD_KAFKA_ACL_OPERATION_ALTER
@ RD_KAFKA_ACL_OPERATION_ALTER
Definition: rdkafka.h:7237
rd_kafka_consume_batch_queue
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_kafka_AclBinding_permission_type
RD_EXPORT rd_kafka_AclPermissionType_t rd_kafka_AclBinding_permission_type(const rd_kafka_AclBinding_t *acl)
rd_kafka_group_list
List of groups.
Definition: rdkafka.h:5174
rd_kafka_queue_poll
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_kafka_TopicPartitionInfo_isr
const RD_EXPORT rd_kafka_Node_t ** rd_kafka_TopicPartitionInfo_isr(const rd_kafka_TopicPartitionInfo_t *partition, size_t *cntp)
Gets the partition in-sync replicas for partition.
rd_kafka_Node_host
const RD_EXPORT char * rd_kafka_Node_host(const rd_kafka_Node_t *node)
Get the host of node.
rd_kafka_TopicDescription_t
struct rd_kafka_TopicDescription_s rd_kafka_TopicDescription_t
DescribeTopics result type.
Definition: rdkafka.h:8177
rd_kafka_topic_partition_list_s::cnt
int cnt
Definition: rdkafka.h:965
RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
@ RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
Definition: rdkafka.h:393
rd_kafka_topic_partition_list_del
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_kafka_AdminOptions_set_opaque
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_kafka_consumer_close_queue
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_kafka_thread_type_t
rd_kafka_thread_type_t
librdkafka internal thread type.
Definition: rdkafka.h:5403
rd_kafka_Node_t
struct rd_kafka_Node_s rd_kafka_Node_t
Node (broker) information.
Definition: rdkafka.h:5064
rd_kafka_get_watermark_offsets
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_kafka_ListConsumerGroups_result_t
rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t
Definition: rdkafka.h:5772
rd_kafka_assignment_lost
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_kafka_AclBinding_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_AclBinding_error(const rd_kafka_AclBinding_t *acl)
rd_kafka_UserScramCredentialsDescription_scramcredentialinfo
const RD_EXPORT rd_kafka_ScramCredentialInfo_t * rd_kafka_UserScramCredentialsDescription_scramcredentialinfo(const rd_kafka_UserScramCredentialsDescription_t *description, size_t idx)
Returns the ScramCredentialInfo at index idx of UserScramCredentialsDescription.
RD_KAFKA_RESP_ERR_INVALID_GROUP_ID
@ RD_KAFKA_RESP_ERR_INVALID_GROUP_ID
Definition: rdkafka.h:480
rd_kafka_topic_conf_dup
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__LOG_TRUNCATION
@ RD_KAFKA_RESP_ERR__LOG_TRUNCATION
Definition: rdkafka.h:409
rd_kafka_ConfigEntry_is_synonym
RD_EXPORT int rd_kafka_ConfigEntry_is_synonym(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_message_produce_errstr
const RD_EXPORT char * rd_kafka_message_produce_errstr(const rd_kafka_message_t *rkmessage)
Returns the error string for an errored produced rd_kafka_message_t or NULL if there was no error.
rd_kafka_ConfigSource_name
const RD_EXPORT char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
@ RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
Definition: rdkafka.h:6952
rd_kafka_queue_get_main
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_main(rd_kafka_t *rk)
rd_kafka_event_type_t
int rd_kafka_event_type_t
Event types.
Definition: rdkafka.h:5466
RD_KAFKA_CONF_INVALID
@ RD_KAFKA_CONF_INVALID
Definition: rdkafka.h:1742
rd_kafka_metadata_topic::partition_cnt
int partition_cnt
Definition: rdkafka.h:5007
rd_kafka_vu_s
VTYPE + argument container for use with rd_kafka_produce_va()
Definition: rdkafka.h:1157
RD_KAFKA_ADMIN_OP_CREATETOPICS
@ RD_KAFKA_ADMIN_OP_CREATETOPICS
Definition: rdkafka.h:6950
rd_kafka_ListOffsetsResultInfo_topic_partition
const RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_ListOffsetsResultInfo_topic_partition(const rd_kafka_ListOffsetsResultInfo_t *result_info)
Returns the topic partition of the passed result_info.
rd_kafka_Node_rack
const RD_EXPORT char * rd_kafka_Node_rack(const rd_kafka_Node_t *node)
Get the rack of node.
RD_KAFKA_CONF_UNKNOWN
@ RD_KAFKA_CONF_UNKNOWN
Definition: rdkafka.h:1741
RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
@ RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
Definition: rdkafka.h:466
RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
@ RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafka.h:600
RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
@ RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafka.h:478
rd_kafka_queue_get_partition
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_partition(rd_kafka_t *rk, const char *topic, int32_t partition)
RD_KAFKA_RESP_ERR_POLICY_VIOLATION
@ RD_KAFKA_RESP_ERR_POLICY_VIOLATION
Definition: rdkafka.h:520
rd_kafka_list_groups
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_metadata::topics
struct rd_kafka_metadata_topic * topics
Definition: rdkafka.h:5021
rd_kafka_error_string
const RD_EXPORT char * rd_kafka_error_string(const rd_kafka_error_t *error)
RD_KAFKA_RESOURCE_ANY
@ RD_KAFKA_RESOURCE_ANY
Definition: rdkafka.h:7717
RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
@ RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafka.h:588
rd_kafka_queue_get_sasl
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_sasl(rd_kafka_t *rk)
RD_KAFKA_RESP_ERR__UNSUPPORTED_FEATURE
@ RD_KAFKA_RESP_ERR__UNSUPPORTED_FEATURE
Definition: rdkafka.h:357
rd_kafka_conf_set_offset_commit_cb
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_KAFKA_RESP_ERR__PURGE_QUEUE
@ RD_KAFKA_RESP_ERR__PURGE_QUEUE
Definition: rdkafka.h:383
rd_kafka_DeleteConsumerGroupOffsets_result_groups
const RD_EXPORT 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_KAFKA_CERT_PUBLIC_KEY
@ RD_KAFKA_CERT_PUBLIC_KEY
Definition: rdkafka.h:2478
rd_kafka_conf_set_events
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_KAFKA_ACL_OPERATION_IDEMPOTENT_WRITE
@ RD_KAFKA_ACL_OPERATION_IDEMPOTENT_WRITE
Definition: rdkafka.h:7245
rd_kafka_MemberDescription_client_id
const RD_EXPORT char * rd_kafka_MemberDescription_client_id(const rd_kafka_MemberDescription_t *member)
Gets client id of member.
rd_kafka_sasl_background_callbacks_enable
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_kafka_flush
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_kafka_TopicDescription_is_internal
RD_EXPORT int rd_kafka_TopicDescription_is_internal(const rd_kafka_TopicDescription_t *topicdesc)
Gets if the topicdesc topic is internal.
rd_kafka_metadata_destroy
RD_EXPORT void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata)
Release metadata memory.
rd_kafka_topic_partition_s::partition
int32_t partition
Definition: rdkafka.h:915
RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
@ RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6971
rd_kafka_err2str
const RD_EXPORT char * rd_kafka_err2str(rd_kafka_resp_err_t err)
Returns a human readable representation of a kafka error.
RD_KAFKA_RESP_ERR__INVALID_TYPE
@ RD_KAFKA_RESP_ERR__INVALID_TYPE
Definition: rdkafka.h:379
rd_kafka_queue_io_event_enable
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_topic_partition_s::_private
void * _private
Definition: rdkafka.h:921
rd_kafka_AclBinding_resource_pattern_type
RD_EXPORT rd_kafka_ResourcePatternType_t rd_kafka_AclBinding_resource_pattern_type(const rd_kafka_AclBinding_t *acl)
RD_KAFKA_ADMIN_OP_DESCRIBECLUSTER
@ RD_KAFKA_ADMIN_OP_DESCRIBECLUSTER
Definition: rdkafka.h:6975
RD_KAFKA_VTYPE_VALUE
@ RD_KAFKA_VTYPE_VALUE
Definition: rdkafka.h:1136
rd_kafka_ListOffsets
RD_EXPORT void rd_kafka_ListOffsets(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *topic_partitions, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List offsets for the specified topic_partitions. This operation enables to find the beginning offset,...
RD_KAFKA_RESP_ERR_NO_ERROR
@ RD_KAFKA_RESP_ERR_NO_ERROR
Definition: rdkafka.h:421
RD_KAFKA_RESOURCE_UNKNOWN
@ RD_KAFKA_RESOURCE_UNKNOWN
Definition: rdkafka.h:7716
rd_kafka_assignment
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_kafka_TopicPartitionInfo_leader
const RD_EXPORT rd_kafka_Node_t * rd_kafka_TopicPartitionInfo_leader(const rd_kafka_TopicPartitionInfo_t *partition)
Gets the partition leader for partition.
rd_kafka_TopicPartitionInfo_partition
const RD_EXPORT int rd_kafka_TopicPartitionInfo_partition(const rd_kafka_TopicPartitionInfo_t *partition)
Gets the partition id for partition.
rd_kafka_event_message_count
RD_EXPORT size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev)
rd_kafka_DescribeConsumerGroups_result_groups
const RD_EXPORT 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_kafka_message_s::key
void * key
Definition: rdkafka.h:1464
rd_kafka_AlterUserScramCredentials
RD_EXPORT void rd_kafka_AlterUserScramCredentials(rd_kafka_t *rk, rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Alter SASL/SCRAM credentials. This operation is supported by brokers with version 2....
rd_kafka_cert_enc_t
rd_kafka_cert_enc_t
SSL certificate encoding.
Definition: rdkafka.h:2491
rd_kafka_ScramCredentialInfo_mechanism
RD_EXPORT rd_kafka_ScramMechanism_t rd_kafka_ScramCredentialInfo_mechanism(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the mechanism of a given ScramCredentialInfo.
RD_KAFKA_RESP_ERR_INCONSISTENT_VOTER_SET
@ RD_KAFKA_RESP_ERR_INCONSISTENT_VOTER_SET
Definition: rdkafka.h:630
RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
@ RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
Definition: rdkafka.h:7241
rd_kafka_event_destroy
RD_EXPORT void rd_kafka_event_destroy(rd_kafka_event_t *rkev)
Destroy an event.
rd_kafka_event_DescribeCluster_result
const RD_EXPORT rd_kafka_DescribeCluster_result_t * rd_kafka_event_DescribeCluster_result(rd_kafka_event_t *rkev)
Get DescribeCluster result.
rd_kafka_NewPartitions_set_replica_assignment
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_kafka_err2name
const RD_EXPORT char * rd_kafka_err2name(rd_kafka_resp_err_t err)
Returns the error code name (enum name).
RD_KAFKA_VTYPE_MSGFLAGS
@ RD_KAFKA_VTYPE_MSGFLAGS
Definition: rdkafka.h:1143
rd_kafka_error_destroy
RD_EXPORT void rd_kafka_error_destroy(rd_kafka_error_t *error)
Free and destroy an error object.
RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION
@ RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION
Definition: rdkafka.h:307
rd_kafka_DescribeAcls_result_t
rd_kafka_event_t rd_kafka_DescribeAcls_result_t
Definition: rdkafka.h:5758
rd_kafka_DeleteGroup_t
struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t
Definition: rdkafka.h:8850
rd_kafka_IncrementalAlterConfigs
RD_EXPORT void rd_kafka_IncrementalAlterConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Incrementally update the configuration for the specified resources. Updates are not transactional so ...
rd_kafka_message_s::_private
void * _private
Definition: rdkafka.h:1476
rd_kafka_event_DescribeConfigs_result
const RD_EXPORT rd_kafka_DescribeConfigs_result_t * rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev)
Get DescribeConfigs result.
rd_kafka_ConfigResource_type
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_ConfigResource_type(const rd_kafka_ConfigResource_t *config)
rd_kafka_TopicCollection_of_topic_names
RD_EXPORT rd_kafka_TopicCollection_t * rd_kafka_TopicCollection_of_topic_names(const char **topics, size_t topics_cnt)
Creates a new TopicCollection for passing to rd_kafka_DescribeTopics.
rd_kafka_AclBinding_new
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_kafka_commit_queue
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_DeleteAcls_result_response_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_DeleteAcls_result_response_error(const rd_kafka_DeleteAcls_result_response_t *result_response)
rd_kafka_metadata_partition::replicas
int32_t * replicas
Definition: rdkafka.h:4997
RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
@ RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
Definition: rdkafka.h:576
rd_kafka_interceptor_add_on_thread_start
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_kafka_version_str
const RD_EXPORT char * rd_kafka_version_str(void)
Returns the librdkafka version as string.
rd_kafka_event_DeleteTopics_result
const RD_EXPORT rd_kafka_DeleteTopics_result_t * rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev)
Get DeleteTopics result.
RD_KAFKA_RESP_ERR__NOENT
@ RD_KAFKA_RESP_ERR__NOENT
Definition: rdkafka.h:375
rd_kafka_interceptor_add_on_destroy
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_KAFKA_RESP_ERR_FENCED_LEADER_EPOCH
@ RD_KAFKA_RESP_ERR_FENCED_LEADER_EPOCH
Definition: rdkafka.h:586
rd_kafka_timestamp_type_t
rd_kafka_timestamp_type_t
Definition: rdkafka.h:220
rd_kafka_headers_copy
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_copy(const rd_kafka_headers_t *src)
Make a copy of headers list src.
rd_kafka_committed
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_group_member_info::member_assignment
void * member_assignment
Definition: rdkafka.h:5135
RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
@ RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
Definition: rdkafka.h:339
rd_kafka_metadata
Metadata container.
Definition: rdkafka.h:5016
RD_KAFKA_RESP_ERR__FENCED
@ RD_KAFKA_RESP_ERR__FENCED
Definition: rdkafka.h:399
rd_kafka_interceptor_f_on_new_t
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:6289
rd_kafka_group_info
Group information.
Definition: rdkafka.h:5158
RD_KAFKA_CERT_ENC_PEM
@ RD_KAFKA_CERT_ENC_PEM
Definition: rdkafka.h:2494
rd_kafka_DescribeAcls_result_acls
const RD_EXPORT 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_kafka_AlterConsumerGroupOffsets_new
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_kafka_DeleteRecords_destroy
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_KAFKA_VTYPE_HEADER
@ RD_KAFKA_VTYPE_HEADER
Definition: rdkafka.h:1145
rd_kafka_err_desc
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:658
RD_KAFKA_RESP_ERR_INVALID_REQUEST
@ RD_KAFKA_RESP_ERR_INVALID_REQUEST
Definition: rdkafka.h:516
rd_kafka_NewTopic_set_replica_assignment
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_kafka_error_is_fatal
RD_EXPORT int rd_kafka_error_is_fatal(const rd_kafka_error_t *error)
rd_kafka_message_s::partition
int32_t partition
Definition: rdkafka.h:1456
rd_kafka_errno2err
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_kafka_event_DescribeConsumerGroups_result
const RD_EXPORT rd_kafka_DescribeConsumerGroups_result_t * rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev)
Get DescribeConsumerGroups result.
RD_KAFKA_RESOURCE_PATTERN_PREFIXED
@ RD_KAFKA_RESOURCE_PATTERN_PREFIXED
Definition: rdkafka.h:7738
RD_KAFKA_RESP_ERR_NOT_COORDINATOR
@ RD_KAFKA_RESP_ERR_NOT_COORDINATOR
Definition: rdkafka.h:461
RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
Definition: rdkafka.h:445
rd_kafka_topic_partition_list_s::size
int size
Definition: rdkafka.h:966
rd_kafka_ConfigEntry_t
struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t
Definition: rdkafka.h:7642
rd_kafka_DeleteConsumerGroupOffsets_destroy_array
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_NewPartitions_destroy
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_kafka_UserScramCredentialAlteration_destroy_array
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy_array(rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt)
Destroys an array of UserScramCredentialAlteration.
rd_kafka_event_message_array
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_kafka_message_broker_id
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_topic_partition_list_find
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_KAFKA_TIMESTAMP_NOT_AVAILABLE
@ RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafka.h:221
RD_KAFKA_RESP_ERR_UNSUPPORTED_ASSIGNOR
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_ASSIGNOR
Definition: rdkafka.h:646
RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
@ RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
Definition: rdkafka.h:7239
rd_kafka_UserScramCredentialsDescription_t
struct rd_kafka_UserScramCredentialsDescription_s rd_kafka_UserScramCredentialsDescription_t
Representation of all SASL/SCRAM credentials associated with a user that can be retrieved,...
Definition: rdkafka.h:9334
rd_kafka_ConfigEntry_name
const RD_EXPORT char * rd_kafka_ConfigEntry_name(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_headers_destroy
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_kafka_consumer_poll
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_kafka_UserScramCredentialUpsertion_new
RD_EXPORT rd_kafka_UserScramCredentialAlteration_t * rd_kafka_UserScramCredentialUpsertion_new(const char *username, rd_kafka_ScramMechanism_t mechanism, int32_t iterations, const unsigned char *password, size_t password_size, const unsigned char *salt, size_t salt_size)
Allocates a new UserScramCredentialUpsertion given its fields. If salt isn't given a 64 B salt is gen...
rd_kafka_consumer_group_metadata_destroy
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().
RD_KAFKA_RESP_ERR__END
@ RD_KAFKA_RESP_ERR__END
Definition: rdkafka.h:415
RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafka.h:490
rd_kafka_event_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev)
rd_kafka_begin_transaction
RD_EXPORT rd_kafka_error_t * rd_kafka_begin_transaction(rd_kafka_t *rk)
Begin a new transaction.
rd_kafka_conf_set_opaque
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.
RD_KAFKA_RESP_ERR_REPLICA_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_REPLICA_NOT_AVAILABLE
Definition: rdkafka.h:441
RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
@ RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
Definition: rdkafka.h:474
rd_kafka_AclBinding_destroy_array
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_kafka_AclBinding_host
const RD_EXPORT char * rd_kafka_AclBinding_host(const rd_kafka_AclBinding_t *acl)
RD_KAFKA_ACL_OPERATION_ANY
@ RD_KAFKA_ACL_OPERATION_ANY
Definition: rdkafka.h:7230
rd_kafka_conf_set_dr_cb
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_kafka_AclBinding_destroy
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_kafka_DeleteTopic_destroy
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_kafka_ListConsumerGroupOffsets
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_kafka_AdminOptions_set_broker
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_KAFKA_ACL_PERMISSION_TYPE_ANY
@ RD_KAFKA_ACL_PERMISSION_TYPE_ANY
Definition: rdkafka.h:9566
RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
@ RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
Definition: rdkafka.h:296
rd_kafka_ConfigResource_name
const RD_EXPORT char * rd_kafka_ConfigResource_name(const rd_kafka_ConfigResource_t *config)
rd_kafka_ConfigResource_t
struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t
Definition: rdkafka.h:7767
RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
@ RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafka.h:522
rd_kafka_ConsumerGroupDescription_member_count
RD_EXPORT size_t rd_kafka_ConsumerGroupDescription_member_count(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the members count of grpdesc group.
RD_KAFKA_RESOURCE_PATTERN_MATCH
@ RD_KAFKA_RESOURCE_PATTERN_MATCH
Definition: rdkafka.h:7734
rd_kafka_interceptor_f_on_commit_t
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:6404
RD_KAFKA_ACL_OPERATION_CREATE
@ RD_KAFKA_ACL_OPERATION_CREATE
Definition: rdkafka.h:7235
rd_kafka_conf_set_stats_cb
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_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
@ RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafka.h:627
rd_kafka_DeleteConsumerGroupOffsets
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_kafka_DescribeCluster
RD_EXPORT void rd_kafka_DescribeCluster(rd_kafka_t *rk, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describes the cluster.
rd_kafka_message_set_headers
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_KAFKA_PRODUCER
@ RD_KAFKA_PRODUCER
Definition: rdkafka.h:210
rd_kafka_poll
RD_EXPORT int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms)
Polls the provided kafka handle for events.
rd_kafka_topic_partition_list_add_range
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_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
Definition: rdkafka.h:456
rd_kafka_conf_properties_show
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_KAFKA_RESP_ERR__VALUE_SERIALIZATION
@ RD_KAFKA_RESP_ERR__VALUE_SERIALIZATION
Definition: rdkafka.h:365
rd_kafka_seek
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_kafka_DeleteGroups
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_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
@ RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
Definition: rdkafka.h:369
rd_kafka_metadata::broker_cnt
int broker_cnt
Definition: rdkafka.h:5017
rd_kafka_produce_batch
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_CERT_ENC_PKCS12
@ RD_KAFKA_CERT_ENC_PKCS12
Definition: rdkafka.h:2492
RD_KAFKA_ACL_OPERATION_UNKNOWN
@ RD_KAFKA_ACL_OPERATION_UNKNOWN
Definition: rdkafka.h:7229
rd_kafka_sasl_set_credentials
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_kafka_ConfigEntry_is_default
RD_EXPORT int rd_kafka_ConfigEntry_is_default(const rd_kafka_ConfigEntry_t *entry)
RD_KAFKA_RESP_ERR__NODE_UPDATE
@ RD_KAFKA_RESP_ERR__NODE_UPDATE
Definition: rdkafka.h:323
rd_kafka_DescribeTopics_result_t
rd_kafka_event_t rd_kafka_DescribeTopics_result_t
Definition: rdkafka.h:5784
RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
@ RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
Definition: rdkafka.h:407
rd_kafka_queue_cb_event_enable
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_kafka_event_AlterConfigs_result
const RD_EXPORT rd_kafka_AlterConfigs_result_t * rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev)
Get AlterConfigs result.
rd_kafka_topic_opaque
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_kafka_group_info::protocol
char * protocol
Definition: rdkafka.h:5164
rd_kafka_cert_type_t
rd_kafka_cert_type_t
SSL certificate type.
Definition: rdkafka.h:2477
RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
@ RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafka.h:580
rd_kafka_DeleteTopic_new
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_kafka_vu_s::_pad
char _pad[64]
Definition: rdkafka.h:1177
rd_kafka_DeleteGroup_destroy
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_KAFKA_RESP_ERR__PARTITION_EOF
@ RD_KAFKA_RESP_ERR__PARTITION_EOF
Definition: rdkafka.h:305
rd_kafka_metadata_topic::topic
char * topic
Definition: rdkafka.h:5006
rd_kafka_error_code
RD_EXPORT rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error)
rd_kafka_fatal_error
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_topic_partition_list
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR_UNKNOWN_PRODUCER_ID
@ RD_KAFKA_RESP_ERR_UNKNOWN_PRODUCER_ID
Definition: rdkafka.h:556
rd_kafka_consumer_group_state_name
const RD_EXPORT char * rd_kafka_consumer_group_state_name(rd_kafka_consumer_group_state_t state)
Returns a name for a state code.
rd_kafka_ConsumerGroupListing_group_id
const RD_EXPORT char * rd_kafka_ConsumerGroupListing_group_id(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets the group id for the grplist group.
rd_kafka_err_desc::code
rd_kafka_resp_err_t code
Definition: rdkafka.h:659
rd_kafka_ScramMechanism_t
rd_kafka_ScramMechanism_t
Apache Kafka ScramMechanism values.
Definition: rdkafka.h:9300
RD_KAFKA_VTYPE_RKT
@ RD_KAFKA_VTYPE_RKT
Definition: rdkafka.h:1134
rd_kafka_DescribeCluster_result_authorized_operations
const RD_EXPORT rd_kafka_AclOperation_t * rd_kafka_DescribeCluster_result_authorized_operations(const rd_kafka_DescribeCluster_result_t *result, size_t *cntp)
Gets the authorized ACL operations for the result cluster.
rd_kafka_AlterConfigs_result_resources
const RD_EXPORT 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_kafka_DeleteTopic_t
struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t
Definition: rdkafka.h:7402
rd_kafka_conf_destroy
RD_EXPORT void rd_kafka_conf_destroy(rd_kafka_conf_t *conf)
Destroys a conf object.
rd_kafka_msg_partitioner_fnv1a_random
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_kafka_event_DescribeTopics_result
const RD_EXPORT rd_kafka_DescribeTopics_result_t * rd_kafka_event_DescribeTopics_result(rd_kafka_event_t *rkev)
Get DescribeTopics result.
rd_kafka_TopicCollection_t
struct rd_kafka_TopicCollection_s rd_kafka_TopicCollection_t
Represents a collection of topics, to be passed to DescribeTopics.
Definition: rdkafka.h:8165
rd_kafka_AdminOptions_set_isolation_level
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_isolation_level(rd_kafka_AdminOptions_t *options, rd_kafka_IsolationLevel_t value)
Set Isolation Level to an allowed rd_kafka_IsolationLevel_t value.
rd_kafka_ListOffsets_result_t
rd_kafka_event_t rd_kafka_ListOffsets_result_t
Definition: rdkafka.h:5792
rd_kafka_event_ListOffsets_result
const RD_EXPORT rd_kafka_ListOffsets_result_t * rd_kafka_event_ListOffsets_result(rd_kafka_event_t *rkev)
Get ListOffsets result.
rd_kafka_TopicPartitionInfo_replicas
const RD_EXPORT rd_kafka_Node_t ** rd_kafka_TopicPartitionInfo_replicas(const rd_kafka_TopicPartitionInfo_t *partition, size_t *cntp)
Gets the partition replicas for partition.
rd_kafka_topic_partition_s::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:920
rd_kafka_conf_interceptor_add_on_conf_set
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_kafka_query_watermark_offsets
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_conf_set_resolve_cb
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_KAFKA_RESP_ERR_DUPLICATE_RESOURCE
@ RD_KAFKA_RESP_ERR_DUPLICATE_RESOURCE
Definition: rdkafka.h:625
rd_kafka_consume
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_kafka_DeleteGroups_result_t
rd_kafka_event_t rd_kafka_DeleteGroups_result_t
Definition: rdkafka.h:5776
rd_kafka_DeleteAcls_result_response_matching_acls
const RD_EXPORT 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_KAFKA_ACL_OPERATION_DESCRIBE
@ RD_KAFKA_ACL_OPERATION_DESCRIBE
Definition: rdkafka.h:7238
RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
@ RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
Definition: rdkafka.h:333
RD_KAFKA_RESP_ERR__SSL
@ RD_KAFKA_RESP_ERR__SSL
Definition: rdkafka.h:325
rd_kafka_event_config_string
const RD_EXPORT char * rd_kafka_event_config_string(rd_kafka_event_t *rkev)
rd_kafka_unsubscribe
RD_EXPORT rd_kafka_resp_err_t rd_kafka_unsubscribe(rd_kafka_t *rk)
Unsubscribe from the current subscription set.
rd_kafka_DeleteRecords_result_t
rd_kafka_event_t rd_kafka_DeleteRecords_result_t
Definition: rdkafka.h:5770
rd_kafka_DescribeAcls
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_kafka_vu_s::u
union rd_kafka_vu_s::@0 u
rd_kafka_ListConsumerGroupOffsets_destroy
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_kafka_message_detach_headers
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_kafka_Node_port
RD_EXPORT uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node)
Get the port of node.
RD_KAFKA_CONSUMER
@ RD_KAFKA_CONSUMER
Definition: rdkafka.h:211
rd_kafka_memberid
RD_EXPORT char * rd_kafka_memberid(const rd_kafka_t *rk)
Returns this client's broker-assigned group member id.
RD_KAFKA_RESP_ERR_INVALID_MSG_SIZE
@ RD_KAFKA_RESP_ERR_INVALID_MSG_SIZE
Definition: rdkafka.h:429
rd_kafka_error_is_retriable
RD_EXPORT int rd_kafka_error_is_retriable(const rd_kafka_error_t *error)
RD_KAFKA_RESP_ERR_GROUP_SUBSCRIBED_TO_TOPIC
@ RD_KAFKA_RESP_ERR_GROUP_SUBSCRIBED_TO_TOPIC
Definition: rdkafka.h:612
rd_kafka_event_AlterConsumerGroupOffsets_result
const RD_EXPORT rd_kafka_AlterConsumerGroupOffsets_result_t * rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get AlterConsumerGroupOffsets result.
rd_kafka_ResourcePatternType_t
rd_kafka_ResourcePatternType_t
Apache Kafka pattern types.
Definition: rdkafka.h:7728
rd_kafka_conf_set_open_cb
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_metadata_partition
Partition information.
Definition: rdkafka.h:4992
RD_KAFKA_VTYPE_OPAQUE
@ RD_KAFKA_VTYPE_OPAQUE
Definition: rdkafka.h:1138
RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
@ RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
Definition: rdkafka.h:552
RD_KAFKA_CERT_ENC_DER
@ RD_KAFKA_CERT_ENC_DER
Definition: rdkafka.h:2493
rd_kafka_conf_interceptor_add_on_conf_dup
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_KAFKA_RESP_ERR_REQUEST_TIMED_OUT
@ RD_KAFKA_RESP_ERR_REQUEST_TIMED_OUT
Definition: rdkafka.h:437
rd_kafka_consume_callback_queue
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_KAFKA_CERT_PRIVATE_KEY
@ RD_KAFKA_CERT_PRIVATE_KEY
Definition: rdkafka.h:2479
rd_kafka_topic_destroy
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_KAFKA_RESP_ERR__ALL_BROKERS_DOWN
@ RD_KAFKA_RESP_ERR__ALL_BROKERS_DOWN
Definition: rdkafka.h:313
rd_kafka_ListConsumerGroupOffsets_t
struct rd_kafka_ListConsumerGroupOffsets_s rd_kafka_ListConsumerGroupOffsets_t
Definition: rdkafka.h:8930
RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
@ RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafka.h:510
rd_kafka_topic_partition_list_destroy
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_kafka_producev
RD_EXPORT rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk,...)
Produce and send a single message to broker.
RD_KAFKA_RESP_ERR__STATE
@ RD_KAFKA_RESP_ERR__STATE
Definition: rdkafka.h:343
RD_KAFKA_RESP_ERR__NO_OFFSET
@ RD_KAFKA_RESP_ERR__NO_OFFSET
Definition: rdkafka.h:351
rd_kafka_yield
RD_EXPORT void rd_kafka_yield(rd_kafka_t *rk)
Cancels the current callback dispatcher (rd_kafka_poll(), rd_kafka_consume_callback(),...
rd_kafka_NewPartitions_new
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_KAFKA_RESP_ERR__TIMED_OUT_QUEUE
@ RD_KAFKA_RESP_ERR__TIMED_OUT_QUEUE
Definition: rdkafka.h:355
rd_kafka_topic_result_name
const RD_EXPORT char * rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres)
rd_kafka_interceptor_f_on_response_received_t
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:6469
rd_kafka_MemberAssignment_partitions
const RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_MemberAssignment_partitions(const rd_kafka_MemberAssignment_t *assignment)
Gets assigned partitions of a member assignment.
RD_KAFKA_ADMIN_OP__CNT
@ RD_KAFKA_ADMIN_OP__CNT
Definition: rdkafka.h:6977
RD_KAFKA_THREAD_BROKER
@ RD_KAFKA_THREAD_BROKER
Definition: rdkafka.h:5406
rd_kafka_interceptor_add_on_response_received
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_kafka_header_remove
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_KAFKA_RESP_ERR_NO_REASSIGNMENT_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_NO_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:609
RD_KAFKA_RESP_ERR__ISR_INSUFF
@ RD_KAFKA_RESP_ERR__ISR_INSUFF
Definition: rdkafka.h:321
rd_kafka_AclBinding_operation
RD_EXPORT rd_kafka_AclOperation_t rd_kafka_AclBinding_operation(const rd_kafka_AclBinding_t *acl)
rd_kafka_ScramCredentialInfo_t
struct rd_kafka_ScramCredentialInfo_s rd_kafka_ScramCredentialInfo_t
Scram credential info. Mechanism and iterations for a SASL/SCRAM credential associated with a user.
Definition: rdkafka.h:9312
rd_kafka_topic_conf_destroy
RD_EXPORT void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf)
Destroys a topic conf object.
rd_kafka_IncrementalAlterConfigs_result_t
rd_kafka_event_t rd_kafka_IncrementalAlterConfigs_result_t
Definition: rdkafka.h:5766
rd_kafka_MemberAssignment_t
struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t
Member assignment included in MemberDescription.
Definition: rdkafka.h:8588
rd_kafka_ListConsumerGroupOffsets_destroy_array
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...
RD_KAFKA_RESP_ERR_NOT_CONTROLLER
@ RD_KAFKA_RESP_ERR_NOT_CONTROLLER
Definition: rdkafka.h:514
rd_kafka_MemberDescription_consumer_id
const RD_EXPORT char * rd_kafka_MemberDescription_consumer_id(const rd_kafka_MemberDescription_t *member)
Gets consumer id of member.
RD_KAFKA_ADMIN_OP_DESCRIBECONSUMERGROUPS
@ RD_KAFKA_ADMIN_OP_DESCRIBECONSUMERGROUPS
Definition: rdkafka.h:6963
RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
@ RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
Definition: rdkafka.h:9565
RD_KAFKA_RESP_ERR_UNRELEASED_INSTANCE_ID
@ RD_KAFKA_RESP_ERR_UNRELEASED_INSTANCE_ID
Definition: rdkafka.h:643
rd_kafka_ConfigEntry_value
const RD_EXPORT char * rd_kafka_ConfigEntry_value(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_consumer_group_state_code
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_kafka_topic_partition_list_add
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_kafka_msg_partitioner_murmur2_random
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_kafka_IsolationLevel_t
rd_kafka_IsolationLevel_t
IsolationLevel enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:7001
rd_kafka_DeleteGroups_result_groups
const RD_EXPORT 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_kafka_UserScramCredentialDeletion_new
RD_EXPORT rd_kafka_UserScramCredentialAlteration_t * rd_kafka_UserScramCredentialDeletion_new(const char *username, rd_kafka_ScramMechanism_t mechanism)
Allocates a new UserScramCredentialDeletion given its fields.
rd_kafka_position
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_kafka_message_s::len
size_t len
Definition: rdkafka.h:1461
rd_kafka_resume_partitions
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_kafka_DeleteAcls_result_t
rd_kafka_event_t rd_kafka_DeleteAcls_result_t
Definition: rdkafka.h:5760
RD_KAFKA_ADMIN_OP_ALTERCONFIGS
@ RD_KAFKA_ADMIN_OP_ALTERCONFIGS
Definition: rdkafka.h:6953
rd_kafka_queue_new
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_new(rd_kafka_t *rk)
Create a new message queue.
rd_kafka_header_get_all
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_KAFKA_CONFIG_SOURCE_DYNAMIC_TOPIC_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_TOPIC_CONFIG
Definition: rdkafka.h:7616
RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
Definition: rdkafka.h:7627
rd_kafka_topic_partition_list_copy
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_kafka_DescribeCluster_result_cluster_id
const RD_EXPORT char * rd_kafka_DescribeCluster_result_cluster_id(const rd_kafka_DescribeCluster_result_t *result)
Gets the cluster id for the result cluster.
rd_kafka_group_member_info::client_id
char * client_id
Definition: rdkafka.h:5130
rd_kafka_consume_queue
RD_EXPORT rd_kafka_message_t * rd_kafka_consume_queue(rd_kafka_queue_t *rkqu, int timeout_ms)
Consume from queue.
rd_kafka_event_opaque
RD_EXPORT void * rd_kafka_event_opaque(rd_kafka_event_t *rkev)
rd_kafka_CreateAcls
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_group_result_partitions
const RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres)
RD_KAFKA_RESP_ERR_UNKNOWN
@ RD_KAFKA_RESP_ERR_UNKNOWN
Definition: rdkafka.h:419
rd_kafka_ResourceType_name
const RD_EXPORT char * rd_kafka_ResourceType_name(rd_kafka_ResourceType_t restype)
rd_kafka_plugin_f_conf_init_t
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:6138
rd_kafka_interceptor_f_on_conf_set_t
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:6224
rd_kafka_offsets_store
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_KAFKA_RESP_ERR__BAD_MSG
@ RD_KAFKA_RESP_ERR__BAD_MSG
Definition: rdkafka.h:286
rd_kafka_event_DeleteRecords_result
const RD_EXPORT rd_kafka_DeleteRecords_result_t * rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev)
rd_kafka_group_info::state
char * state
Definition: rdkafka.h:5162
RD_KAFKA_ADMIN_OP_ANY
@ RD_KAFKA_ADMIN_OP_ANY
Definition: rdkafka.h:6949
rd_kafka_consumer_group_metadata_write
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_KAFKA_RESP_ERR__MSG_TIMED_OUT
@ RD_KAFKA_RESP_ERR__MSG_TIMED_OUT
Definition: rdkafka.h:300
rd_kafka_consume_callback
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_KAFKA_RESP_ERR_UNSUPPORTED_VERSION
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_VERSION
Definition: rdkafka.h:502
rd_kafka_AclOperation_t
rd_kafka_AclOperation_t
Apache Kafka ACL operation types. Common type for multiple Admin API functions.
Definition: rdkafka.h:7228
rd_kafka_init_transactions
RD_EXPORT rd_kafka_error_t * rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms)
Initialize transactions for the producer instance.
rd_kafka_message_s::rkt
rd_kafka_topic_t * rkt
Definition: rdkafka.h:1455
RD_KAFKA_RESOURCE__CNT
@ RD_KAFKA_RESOURCE__CNT
Definition: rdkafka.h:7721
rd_kafka_topic_new
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_KAFKA_RESP_ERR_FENCED_MEMBER_EPOCH
@ RD_KAFKA_RESP_ERR_FENCED_MEMBER_EPOCH
Definition: rdkafka.h:640
rd_kafka_metadata_topic
Topic information.
Definition: rdkafka.h:5005
rd_kafka_group_result_name
const RD_EXPORT char * rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres)
rd_kafka_type_t
rd_kafka_type_t
rd_kafka_t handle type.
Definition: rdkafka.h:209
rd_kafka_metadata_partition::isrs
int32_t * isrs
Definition: rdkafka.h:4999
rd_kafka_NewPartitions_destroy_array
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_kafka_oauthbearer_set_token
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_kafka_MemberDescription_host
const RD_EXPORT char * rd_kafka_MemberDescription_host(const rd_kafka_MemberDescription_t *member)
Gets host of member.
rd_kafka_interceptor_f_on_conf_dup_t
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:6254
rd_kafka_ConfigEntry_is_sensitive
RD_EXPORT int rd_kafka_ConfigEntry_is_sensitive(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_metadata_partition::isr_cnt
int isr_cnt
Definition: rdkafka.h:4998
rd_kafka_MemberDescription_group_instance_id
const RD_EXPORT char * rd_kafka_MemberDescription_group_instance_id(const rd_kafka_MemberDescription_t *member)
Gets group instance id of member.
rd_kafka_resp_err_t
rd_kafka_resp_err_t
Error codes.
Definition: rdkafka.h:281
RD_KAFKA_RESP_ERR__PARTIAL
@ RD_KAFKA_RESP_ERR__PARTIAL
Definition: rdkafka.h:371
rd_kafka_AclBinding_principal
const RD_EXPORT char * rd_kafka_AclBinding_principal(const rd_kafka_AclBinding_t *acl)
RD_KAFKA_RESP_ERR__WAIT_COORD
@ RD_KAFKA_RESP_ERR__WAIT_COORD
Definition: rdkafka.h:327
rd_kafka_conf_dup_filter
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_ConsumerGroupDescription_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_ConsumerGroupDescription_error(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the error for the grpdesc group.
RD_KAFKA_ADMIN_OP_DESCRIBEACLS
@ RD_KAFKA_ADMIN_OP_DESCRIBEACLS
Definition: rdkafka.h:6960
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafka.h:568
RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafka.h:518
RD_KAFKA_ACL_OPERATION_ALL
@ RD_KAFKA_ACL_OPERATION_ALL
Definition: rdkafka.h:7232
rd_kafka_seek_partitions
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_KAFKA_RESP_ERR_INVALID_PRODUCER_ID_MAPPING
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_ID_MAPPING
Definition: rdkafka.h:531
rd_kafka_msg_status_t
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:1617
rd_kafka_UserScramCredentialsDescription_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_UserScramCredentialsDescription_error(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the error associated with a UserScramCredentialsDescription.
RD_KAFKA_RESP_ERR__IN_PROGRESS
@ RD_KAFKA_RESP_ERR__IN_PROGRESS
Definition: rdkafka.h:331
rd_kafka_topic_partition_set_leader_epoch
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_RESP_ERR__UNKNOWN_TOPIC
@ RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC
Definition: rdkafka.h:311
rd_kafka_event_message_next
const RD_EXPORT rd_kafka_message_t * rd_kafka_event_message_next(rd_kafka_event_t *rkev)
rd_kafka_test_fatal_error
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_kafka_event_CreatePartitions_result
const RD_EXPORT rd_kafka_CreatePartitions_result_t * rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev)
Get CreatePartitions result.
RD_KAFKA_MSG_STATUS_POSSIBLY_PERSISTED
@ RD_KAFKA_MSG_STATUS_POSSIBLY_PERSISTED
Definition: rdkafka.h:1626
rd_kafka_interceptor_f_on_acknowledgement_t
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:6356
rd_kafka_header_get_last
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.