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 0x020500ff
171 
180 RD_EXPORT
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
458 #define RD_KAFKA_RESP_ERR_GROUP_COORDINATOR_NOT_AVAILABLE \
459  RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
463 #define RD_KAFKA_RESP_ERR_NOT_COORDINATOR_FOR_GROUP \
464  RD_KAFKA_RESP_ERR_NOT_COORDINATOR
655  RD_KAFKA_RESP_ERR_END_ALL,
657 
658 
666  const char *name;
667  const char *desc;
668 };
669 
670 
674 RD_EXPORT
675 void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs,
676  size_t *cntp);
677 
678 
679 
685 RD_EXPORT
687 
688 
689 
695 RD_EXPORT
697 
698 
724 RD_EXPORT
726 
727 
752 RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox);
753 
754 
767 RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void);
768 
769 
770 
797 RD_EXPORT
799 rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size);
800 
801 
821  const char *reason);
822 
823 
828 RD_EXPORT
829 rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error);
830 
839 RD_EXPORT
840 const char *rd_kafka_error_name(const rd_kafka_error_t *error);
841 
848 RD_EXPORT
849 const char *rd_kafka_error_string(const rd_kafka_error_t *error);
850 
851 
856 RD_EXPORT
857 int rd_kafka_error_is_fatal(const rd_kafka_error_t *error);
858 
859 
864 RD_EXPORT
865 int rd_kafka_error_is_retriable(const rd_kafka_error_t *error);
866 
867 
878 RD_EXPORT
879 int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error);
880 
886 RD_EXPORT
887 void rd_kafka_error_destroy(rd_kafka_error_t *error);
888 
889 
898 RD_EXPORT
900  const char *fmt,
901  ...) RD_FORMAT(printf, 2, 3);
902 
903 
919 typedef struct rd_kafka_topic_partition_s {
920  char *topic;
921  int32_t partition;
922  int64_t offset;
923  void *metadata;
924  size_t metadata_size;
925  void *opaque;
927  void *_private;
932 
937 RD_EXPORT
939 
940 
949 RD_EXPORT
952  int32_t leader_epoch);
953 
962 RD_EXPORT
964  const rd_kafka_topic_partition_t *rktpar);
965 
970 typedef struct rd_kafka_topic_partition_list_s {
971  int cnt;
972  int size;
975 
990 RD_EXPORT
992 
996 RD_EXPORT
999 
1009 RD_EXPORT
1012  const char *topic,
1013  int32_t partition);
1014 
1015 
1024 RD_EXPORT
1026  rd_kafka_topic_partition_list_t *rktparlist,
1027  const char *topic,
1028  int32_t start,
1029  int32_t stop);
1030 
1031 
1032 
1044 RD_EXPORT
1046  rd_kafka_topic_partition_list_t *rktparlist,
1047  const char *topic,
1048  int32_t partition);
1049 
1050 
1058 RD_EXPORT
1060  rd_kafka_topic_partition_list_t *rktparlist,
1061  int idx);
1062 
1063 
1071 RD_EXPORT
1074 
1075 
1076 
1084 RD_EXPORT
1086  rd_kafka_topic_partition_list_t *rktparlist,
1087  const char *topic,
1088  int32_t partition,
1089  int64_t offset);
1090 
1091 
1092 
1098 RD_EXPORT
1100  const rd_kafka_topic_partition_list_t *rktparlist,
1101  const char *topic,
1102  int32_t partition);
1103 
1104 
1115  rd_kafka_topic_partition_list_t *rktparlist,
1116  int (*cmp)(const void *a, const void *b, void *cmp_opaque),
1117  void *cmp_opaque);
1118 
1119 
1137 typedef enum rd_kafka_vtype_t {
1155 
1156 
1163 typedef struct rd_kafka_vu_s {
1166  union {
1167  const char *cstr;
1168  rd_kafka_topic_t *rkt;
1169  int i;
1170  int32_t i32;
1171  int64_t i64;
1172  struct {
1173  void *ptr;
1174  size_t size;
1175  } mem;
1176  struct {
1177  const char *name;
1178  const void *val;
1179  ssize_t size;
1180  } header;
1181  rd_kafka_headers_t *headers;
1182  void *ptr;
1183  char _pad[64];
1184  } u;
1185 } rd_kafka_vu_t;
1186 
1195 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
1196 
1202 #define RD_KAFKA_V_TOPIC(topic) \
1203  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
1204  (const char *)topic
1210 #define RD_KAFKA_V_RKT(rkt) \
1211  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
1212  (rd_kafka_topic_t *)rkt
1218 #define RD_KAFKA_V_PARTITION(partition) \
1219  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
1220  (int32_t)partition
1226 #define RD_KAFKA_V_VALUE(VALUE, LEN) \
1227  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
1228  (void *)VALUE, (size_t)LEN
1234 #define RD_KAFKA_V_KEY(KEY, LEN) \
1235  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
1236  (void *)KEY, (size_t)LEN
1244 #define RD_KAFKA_V_OPAQUE(msg_opaque) \
1245  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, msg_opaque), \
1246  (void *)msg_opaque
1253 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
1254  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), (int)msgflags
1261 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
1262  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
1263  (int64_t)timestamp
1272 #define RD_KAFKA_V_HEADER(NAME, VALUE, LEN) \
1273  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
1274  const void *, VALUE, ssize_t, LEN), \
1275  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
1276 
1288 #define RD_KAFKA_V_HEADERS(HDRS) \
1289  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
1290  (rd_kafka_headers_t *)HDRS
1291 
1292 
1323 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new(size_t initial_count);
1324 
1329 RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs);
1330 
1334 RD_EXPORT rd_kafka_headers_t *
1335 rd_kafka_headers_copy(const rd_kafka_headers_t *src);
1336 
1354 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs,
1355  const char *name,
1356  ssize_t name_size,
1357  const void *value,
1358  ssize_t value_size);
1359 
1367 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs,
1368  const char *name);
1369 
1370 
1388 RD_EXPORT rd_kafka_resp_err_t
1389 rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs,
1390  const char *name,
1391  const void **valuep,
1392  size_t *sizep);
1393 
1407 RD_EXPORT rd_kafka_resp_err_t
1408 rd_kafka_header_get(const rd_kafka_headers_t *hdrs,
1409  size_t idx,
1410  const char *name,
1411  const void **valuep,
1412  size_t *sizep);
1413 
1414 
1422 RD_EXPORT rd_kafka_resp_err_t
1423 rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs,
1424  size_t idx,
1425  const char **namep,
1426  const void **valuep,
1427  size_t *sizep);
1428 
1429 
1430 
1443 // FIXME: This doesn't show up in docs for some reason
1444 // "Compound rd_kafka_message_t is not documented."
1445 
1459 typedef struct rd_kafka_message_s {
1461  rd_kafka_topic_t *rkt;
1462  int32_t partition;
1463  void *payload;
1467  size_t len;
1470  void *key;
1472  size_t key_len;
1474  int64_t offset;
1482  void *_private;
1490 
1491 
1495 RD_EXPORT
1497 
1498 
1499 
1506 RD_EXPORT
1507 const char *rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage);
1508 
1515 RD_EXPORT
1516 const char *
1518 
1519 
1531 RD_EXPORT
1533  rd_kafka_timestamp_type_t *tstype);
1534 
1535 
1536 
1543 RD_EXPORT
1545 
1546 
1553 RD_EXPORT
1555 
1556 
1573 RD_EXPORT rd_kafka_resp_err_t
1575  rd_kafka_headers_t **hdrsp);
1576 
1588 RD_EXPORT rd_kafka_resp_err_t
1590  rd_kafka_headers_t **hdrsp);
1591 
1592 
1605 RD_EXPORT
1607  rd_kafka_headers_t *hdrs);
1608 
1609 
1615 RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs);
1616 
1617 
1623 typedef enum {
1628 
1633 
1639 
1640 
1647 RD_EXPORT rd_kafka_msg_status_t
1649 
1650 
1658 RD_EXPORT int32_t
1660 
1661 
1678 RD_EXPORT const char *rd_kafka_Uuid_base64str(const rd_kafka_Uuid_t *uuid);
1679 
1687 RD_EXPORT int64_t
1688 rd_kafka_Uuid_least_significant_bits(const rd_kafka_Uuid_t *uuid);
1689 
1690 
1698 RD_EXPORT int64_t
1699 rd_kafka_Uuid_most_significant_bits(const rd_kafka_Uuid_t *uuid);
1700 
1701 
1711 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_new(int64_t most_significant_bits,
1712  int64_t least_significant_bits);
1713 
1722 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_copy(const rd_kafka_Uuid_t *uuid);
1723 
1729 RD_EXPORT void rd_kafka_Uuid_destroy(rd_kafka_Uuid_t *uuid);
1730 
1746 typedef enum {
1748  RD_KAFKA_CONF_INVALID = -1,
1751  RD_KAFKA_CONF_OK = 0
1753 
1754 
1788 RD_EXPORT
1789 rd_kafka_conf_t *rd_kafka_conf_new(void);
1790 
1791 
1795 RD_EXPORT
1796 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1797 
1798 
1805 RD_EXPORT
1806 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1807 
1808 
1813 RD_EXPORT
1814 rd_kafka_conf_t *rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf,
1815  size_t filter_cnt,
1816  const char **filter);
1817 
1818 
1819 
1828 RD_EXPORT
1829 const rd_kafka_conf_t *rd_kafka_conf(rd_kafka_t *rk);
1830 
1831 
1851 RD_EXPORT
1853  const char *name,
1854  const char *value,
1855  char *errstr,
1856  size_t errstr_size);
1857 
1858 
1864 RD_EXPORT
1865 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1866 
1867 
1904  rd_kafka_conf_t *conf,
1905  void (*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque));
1906 
1907 
1911 RD_EXPORT
1912 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1913  void (*dr_cb)(rd_kafka_t *rk,
1914  void *payload,
1915  size_t len,
1916  rd_kafka_resp_err_t err,
1917  void *opaque,
1918  void *msg_opaque));
1919 
1950 RD_EXPORT
1952  rd_kafka_conf_t *conf,
1953  void (*dr_msg_cb)(rd_kafka_t *rk,
1954  const rd_kafka_message_t *rkmessage,
1955  void *opaque));
1956 
1957 
1965 RD_EXPORT
1967  rd_kafka_conf_t *conf,
1968  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque));
1969 
2074 RD_EXPORT
2076  rd_kafka_conf_t *conf,
2077  void (*rebalance_cb)(rd_kafka_t *rk,
2078  rd_kafka_resp_err_t err,
2079  rd_kafka_topic_partition_list_t *partitions,
2080  void *opaque));
2081 
2082 
2083 
2101 RD_EXPORT
2103  rd_kafka_conf_t *conf,
2104  void (*offset_commit_cb)(rd_kafka_t *rk,
2105  rd_kafka_resp_err_t err,
2107  void *opaque));
2108 
2109 
2132 RD_EXPORT
2133 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
2134  void (*error_cb)(rd_kafka_t *rk,
2135  int err,
2136  const char *reason,
2137  void *opaque));
2138 
2156 RD_EXPORT
2157 void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf,
2158  void (*throttle_cb)(rd_kafka_t *rk,
2159  const char *broker_name,
2160  int32_t broker_id,
2161  int throttle_time_ms,
2162  void *opaque));
2163 
2164 
2181 RD_EXPORT
2182 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
2183  void (*log_cb)(const rd_kafka_t *rk,
2184  int level,
2185  const char *fac,
2186  const char *buf));
2187 
2188 
2211 RD_EXPORT
2213  rd_kafka_conf_t *conf,
2214  int (*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque));
2215 
2265 RD_EXPORT
2267  rd_kafka_conf_t *conf,
2268  void (*oauthbearer_token_refresh_cb)(rd_kafka_t *rk,
2269  const char *oauthbearer_config,
2270  void *opaque));
2271 
2299 RD_EXPORT
2300 void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable);
2301 
2302 
2320 RD_EXPORT
2322  rd_kafka_conf_t *conf,
2323  int (*socket_cb)(int domain, int type, int protocol, void *opaque));
2324 
2325 
2326 
2342 RD_EXPORT void
2343 rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf,
2344  int (*connect_cb)(int sockfd,
2345  const struct sockaddr *addr,
2346  int addrlen,
2347  const char *id,
2348  void *opaque));
2349 
2361  rd_kafka_conf_t *conf,
2362  int (*closesocket_cb)(int sockfd, void *opaque));
2363 
2364 
2365 
2366 #ifndef _WIN32
2384 RD_EXPORT
2386  rd_kafka_conf_t *conf,
2387  int (*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque));
2388 #endif
2389 
2391 struct addrinfo;
2392 
2412 RD_EXPORT void
2413 rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf,
2414  int (*resolve_cb)(const char *node,
2415  const char *service,
2416  const struct addrinfo *hints,
2417  struct addrinfo **res,
2418  void *opaque));
2419 
2461 RD_EXPORT
2463  rd_kafka_conf_t *conf,
2464  int (*ssl_cert_verify_cb)(rd_kafka_t *rk,
2465  const char *broker_name,
2466  int32_t broker_id,
2467  int *x509_error,
2468  int depth,
2469  const char *buf,
2470  size_t size,
2471  char *errstr,
2472  size_t errstr_size,
2473  void *opaque));
2474 
2475 
2483 typedef enum rd_kafka_cert_type_t {
2487  RD_KAFKA_CERT__CNT,
2489 
2497 typedef enum rd_kafka_cert_enc_t {
2501  RD_KAFKA_CERT_ENC__CNT,
2503 
2504 
2547 RD_EXPORT rd_kafka_conf_res_t
2548 rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf,
2549  rd_kafka_cert_type_t cert_type,
2550  rd_kafka_cert_enc_t cert_enc,
2551  const void *buffer,
2552  size_t size,
2553  char *errstr,
2554  size_t errstr_size);
2555 
2556 
2571 RD_EXPORT
2572 void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf,
2573  void *callback_data);
2574 
2575 
2581 RD_EXPORT
2582 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
2583 
2588 RD_EXPORT
2589 void *rd_kafka_opaque(const rd_kafka_t *rk);
2590 
2591 
2592 
2606 RD_EXPORT
2607 void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf,
2608  rd_kafka_topic_conf_t *tconf);
2609 
2622 RD_EXPORT rd_kafka_topic_conf_t *
2624 
2625 
2645 RD_EXPORT
2646 rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf,
2647  const char *name,
2648  char *dest,
2649  size_t *dest_size);
2650 
2651 
2657 RD_EXPORT
2658 rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf,
2659  const char *name,
2660  char *dest,
2661  size_t *dest_size);
2662 
2663 
2672 RD_EXPORT
2673 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
2674 
2675 
2684 RD_EXPORT
2685 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
2686  size_t *cntp);
2687 
2692 RD_EXPORT
2693 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
2694 
2702 RD_EXPORT
2704 
2721 RD_EXPORT
2722 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
2723 
2724 
2728 RD_EXPORT
2729 rd_kafka_topic_conf_t *
2730 rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf);
2731 
2736 RD_EXPORT
2737 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup(rd_kafka_t *rk);
2738 
2739 
2743 RD_EXPORT
2744 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
2745 
2746 
2755 RD_EXPORT
2756 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
2757  const char *name,
2758  const char *value,
2759  char *errstr,
2760  size_t errstr_size);
2761 
2768 RD_EXPORT
2769 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf,
2770  void *rkt_opaque);
2771 
2772 
2792 RD_EXPORT
2794  rd_kafka_topic_conf_t *topic_conf,
2795  int32_t (*partitioner)(const rd_kafka_topic_t *rkt,
2796  const void *keydata,
2797  size_t keylen,
2798  int32_t partition_cnt,
2799  void *rkt_opaque,
2800  void *msg_opaque));
2801 
2802 
2829  rd_kafka_topic_conf_t *topic_conf,
2830  int (*msg_order_cmp)(const rd_kafka_message_t *a,
2831  const rd_kafka_message_t *b));
2832 
2833 
2841 RD_EXPORT
2842 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
2843  int32_t partition);
2844 
2845 
2846 /*******************************************************************
2847  * *
2848  * Partitioners provided by rdkafka *
2849  * *
2850  *******************************************************************/
2851 
2865 RD_EXPORT
2866 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
2867  const void *key,
2868  size_t keylen,
2869  int32_t partition_cnt,
2870  void *rkt_opaque,
2871  void *msg_opaque);
2872 
2886 RD_EXPORT
2887 int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt,
2888  const void *key,
2889  size_t keylen,
2890  int32_t partition_cnt,
2891  void *rkt_opaque,
2892  void *msg_opaque);
2893 
2909 RD_EXPORT
2910 int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt,
2911  const void *key,
2912  size_t keylen,
2913  int32_t partition_cnt,
2914  void *rkt_opaque,
2915  void *msg_opaque);
2916 
2917 
2931 RD_EXPORT
2932 int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt,
2933  const void *key,
2934  size_t keylen,
2935  int32_t partition_cnt,
2936  void *rkt_opaque,
2937  void *msg_opaque);
2938 
2953 RD_EXPORT
2954 int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt,
2955  const void *key,
2956  size_t keylen,
2957  int32_t partition_cnt,
2958  void *rkt_opaque,
2959  void *msg_opaque);
2960 
2961 
2975 RD_EXPORT
2976 int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt,
2977  const void *key,
2978  size_t keylen,
2979  int32_t partition_cnt,
2980  void *rkt_opaque,
2981  void *msg_opaque);
2982 
2983 
2998 RD_EXPORT
2999 int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt,
3000  const void *key,
3001  size_t keylen,
3002  int32_t partition_cnt,
3003  void *rkt_opaque,
3004  void *msg_opaque);
3005 
3006 
3046 RD_EXPORT
3048  rd_kafka_conf_t *conf,
3049  char *errstr,
3050  size_t errstr_size);
3051 
3052 
3067 RD_EXPORT
3068 void rd_kafka_destroy(rd_kafka_t *rk);
3069 
3070 
3075 RD_EXPORT
3076 void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags);
3077 
3092 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
3093 
3094 
3095 
3099 RD_EXPORT
3100 const char *rd_kafka_name(const rd_kafka_t *rk);
3101 
3102 
3106 RD_EXPORT
3107 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
3108 
3109 
3120 RD_EXPORT
3121 char *rd_kafka_memberid(const rd_kafka_t *rk);
3122 
3123 
3124 
3143 RD_EXPORT
3144 char *rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms);
3145 
3146 
3162 RD_EXPORT
3163 int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms);
3164 
3165 
3187 RD_EXPORT
3188 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk,
3189  const char *topic,
3190  rd_kafka_topic_conf_t *conf);
3191 
3192 
3193 
3202 RD_EXPORT
3203 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
3204 
3205 
3209 RD_EXPORT
3210 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
3211 
3212 
3217 RD_EXPORT
3218 void *rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt);
3219 
3220 
3227 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
3228 
3229 
3257 RD_EXPORT
3258 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
3259 
3260 
3271 RD_EXPORT
3272 void rd_kafka_yield(rd_kafka_t *rk);
3273 
3274 
3275 
3283 RD_EXPORT rd_kafka_resp_err_t
3285  rd_kafka_topic_partition_list_t *partitions);
3286 
3287 
3288 
3296 RD_EXPORT rd_kafka_resp_err_t
3298  rd_kafka_topic_partition_list_t *partitions);
3299 
3300 
3301 
3310 RD_EXPORT rd_kafka_resp_err_t
3312  const char *topic,
3313  int32_t partition,
3314  int64_t *low,
3315  int64_t *high,
3316  int timeout_ms);
3317 
3318 
3336  const char *topic,
3337  int32_t partition,
3338  int64_t *low,
3339  int64_t *high);
3340 
3341 
3342 
3368 RD_EXPORT rd_kafka_resp_err_t
3371  int timeout_ms);
3372 
3373 
3374 
3388 RD_EXPORT
3389 void *rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size);
3390 
3391 
3392 
3406 RD_EXPORT
3407 void *rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size);
3408 
3409 
3410 
3427 RD_EXPORT
3428 void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr);
3429 
3430 
3452 RD_EXPORT
3453 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
3454 
3458 RD_EXPORT
3459 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
3460 
3461 
3468 RD_EXPORT
3469 rd_kafka_queue_t *rd_kafka_queue_get_main(rd_kafka_t *rk);
3470 
3471 
3472 
3482 RD_EXPORT
3483 rd_kafka_queue_t *rd_kafka_queue_get_sasl(rd_kafka_t *rk);
3484 
3485 
3501 RD_EXPORT
3502 rd_kafka_error_t *rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk);
3503 
3504 
3519 RD_EXPORT
3520 rd_kafka_error_t *rd_kafka_sasl_set_credentials(rd_kafka_t *rk,
3521  const char *username,
3522  const char *password);
3523 
3539 RD_EXPORT
3540 rd_kafka_queue_t *rd_kafka_queue_get_consumer(rd_kafka_t *rk);
3541 
3552 RD_EXPORT
3553 rd_kafka_queue_t *rd_kafka_queue_get_partition(rd_kafka_t *rk,
3554  const char *topic,
3555  int32_t partition);
3556 
3579 RD_EXPORT
3580 rd_kafka_queue_t *rd_kafka_queue_get_background(rd_kafka_t *rk);
3581 
3582 
3593 RD_EXPORT
3594 void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
3595 
3614 RD_EXPORT
3616  rd_kafka_queue_t *rkqu);
3617 
3618 
3622 RD_EXPORT
3623 size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu);
3624 
3625 
3643 RD_EXPORT
3644 void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu,
3645  int fd,
3646  const void *payload,
3647  size_t size);
3648 
3665 RD_EXPORT
3666 void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu,
3667  void (*event_cb)(rd_kafka_t *rk,
3668  void *qev_opaque),
3669  void *qev_opaque);
3670 
3671 
3679 RD_EXPORT
3680 void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu);
3681 
3682 
3693 #define RD_KAFKA_OFFSET_BEGINNING \
3694  -2
3696 #define RD_KAFKA_OFFSET_END \
3697  -1
3699 #define RD_KAFKA_OFFSET_STORED \
3700  -1000
3702 #define RD_KAFKA_OFFSET_INVALID -1001
3706 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
3714 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
3715 
3749 RD_EXPORT
3750 int rd_kafka_consume_start(rd_kafka_topic_t *rkt,
3751  int32_t partition,
3752  int64_t offset);
3753 
3768 RD_EXPORT
3769 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt,
3770  int32_t partition,
3771  int64_t offset,
3772  rd_kafka_queue_t *rkqu);
3773 
3787 RD_EXPORT
3788 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
3789 
3790 
3791 
3820 RD_EXPORT
3821 rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt,
3822  int32_t partition,
3823  int64_t offset,
3824  int timeout_ms);
3825 
3826 
3827 
3859 RD_EXPORT rd_kafka_error_t *
3861  rd_kafka_topic_partition_list_t *partitions,
3862  int timeout_ms);
3863 
3864 
3889 RD_EXPORT
3891 rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms);
3892 
3893 
3894 
3920 RD_EXPORT
3921 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt,
3922  int32_t partition,
3923  int timeout_ms,
3924  rd_kafka_message_t **rkmessages,
3925  size_t rkmessages_size);
3926 
3927 
3928 
3962 RD_EXPORT
3963 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt,
3964  int32_t partition,
3965  int timeout_ms,
3966  void (*consume_cb)(rd_kafka_message_t *rkmessage,
3967  void *commit_opaque),
3968  void *commit_opaque);
3969 
3970 
3989 RD_EXPORT
3991  int timeout_ms);
3992 
3998 RD_EXPORT
3999 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
4000  int timeout_ms,
4001  rd_kafka_message_t **rkmessages,
4002  size_t rkmessages_size);
4003 
4013 RD_EXPORT
4015  rd_kafka_queue_t *rkqu,
4016  int timeout_ms,
4017  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque),
4018  void *commit_opaque);
4019 
4020 
4060 RD_EXPORT
4062 rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset);
4063 
4064 
4097 RD_EXPORT rd_kafka_resp_err_t
4098 rd_kafka_offsets_store(rd_kafka_t *rk,
4100 
4101 
4122 RD_EXPORT
4124 
4178 RD_EXPORT rd_kafka_resp_err_t
4179 rd_kafka_subscribe(rd_kafka_t *rk,
4180  const rd_kafka_topic_partition_list_t *topics);
4181 
4182 
4186 RD_EXPORT
4188 
4189 
4199 RD_EXPORT rd_kafka_resp_err_t
4201 
4202 
4203 
4233 RD_EXPORT
4234 rd_kafka_message_t *rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms);
4235 
4253 RD_EXPORT
4255 
4256 
4274 RD_EXPORT
4275 rd_kafka_error_t *rd_kafka_consumer_close_queue(rd_kafka_t *rk,
4276  rd_kafka_queue_t *rkqu);
4277 
4278 
4287 RD_EXPORT
4288 int rd_kafka_consumer_closed(rd_kafka_t *rk);
4289 
4290 
4308 RD_EXPORT rd_kafka_error_t *
4310  const rd_kafka_topic_partition_list_t *partitions);
4311 
4312 
4330 RD_EXPORT rd_kafka_error_t *rd_kafka_incremental_unassign(
4331  rd_kafka_t *rk,
4332  const rd_kafka_topic_partition_list_t *partitions);
4333 
4334 
4348 RD_EXPORT
4349 const char *rd_kafka_rebalance_protocol(rd_kafka_t *rk);
4350 
4351 
4372 RD_EXPORT rd_kafka_resp_err_t
4373 rd_kafka_assign(rd_kafka_t *rk,
4374  const rd_kafka_topic_partition_list_t *partitions);
4375 
4392 RD_EXPORT rd_kafka_resp_err_t
4393 rd_kafka_assignment(rd_kafka_t *rk,
4394  rd_kafka_topic_partition_list_t **partitions);
4395 
4396 
4413 RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk);
4414 
4415 
4451 RD_EXPORT rd_kafka_resp_err_t
4452 rd_kafka_commit(rd_kafka_t *rk,
4453  const rd_kafka_topic_partition_list_t *offsets,
4454  int async);
4455 
4456 
4463 RD_EXPORT rd_kafka_resp_err_t
4465  const rd_kafka_message_t *rkmessage,
4466  int async);
4467 
4468 
4491 RD_EXPORT rd_kafka_resp_err_t
4492 rd_kafka_commit_queue(rd_kafka_t *rk,
4493  const rd_kafka_topic_partition_list_t *offsets,
4494  rd_kafka_queue_t *rkqu,
4495  void (*cb)(rd_kafka_t *rk,
4496  rd_kafka_resp_err_t err,
4498  void *commit_opaque),
4499  void *commit_opaque);
4500 
4501 
4519 RD_EXPORT rd_kafka_resp_err_t
4520 rd_kafka_committed(rd_kafka_t *rk,
4521  rd_kafka_topic_partition_list_t *partitions,
4522  int timeout_ms);
4523 
4524 
4525 
4542 RD_EXPORT rd_kafka_resp_err_t
4544 
4545 
4546 
4558 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4560 
4561 
4571 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4573 
4574 
4587 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4589  int32_t generation_id,
4590  const char *member_id,
4591  const char *group_instance_id);
4592 
4593 
4603 RD_EXPORT
4605  const rd_kafka_consumer_group_metadata_t *group_metadata);
4606 
4607 
4612 RD_EXPORT void
4613 rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *);
4614 
4615 
4636 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_write(
4637  const rd_kafka_consumer_group_metadata_t *cgmd,
4638  void **bufferp,
4639  size_t *sizep);
4640 
4661 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_read(
4662  rd_kafka_consumer_group_metadata_t **cgmdp,
4663  const void *buffer,
4664  size_t size);
4665 
4681 #define RD_KAFKA_MSG_F_FREE \
4682  0x1
4684 #define RD_KAFKA_MSG_F_COPY \
4685  0x2
4687 #define RD_KAFKA_MSG_F_BLOCK \
4688  0x4
4699 #define RD_KAFKA_MSG_F_PARTITION \
4700  0x8
4805 RD_EXPORT
4806 int rd_kafka_produce(rd_kafka_topic_t *rkt,
4807  int32_t partition,
4808  int msgflags,
4809  void *payload,
4810  size_t len,
4811  const void *key,
4812  size_t keylen,
4813  void *msg_opaque);
4814 
4815 
4829 RD_EXPORT
4831 
4832 
4844 RD_EXPORT
4845 rd_kafka_error_t *
4846 rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt);
4847 
4848 
4878 RD_EXPORT
4879 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt,
4880  int32_t partition,
4881  int msgflags,
4882  rd_kafka_message_t *rkmessages,
4883  int message_cnt);
4884 
4885 
4886 
4911 RD_EXPORT
4912 rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms);
4913 
4914 
4915 
4946 RD_EXPORT
4947 rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags);
4948 
4949 
4957 #define RD_KAFKA_PURGE_F_QUEUE 0x1
4958 
4966 #define RD_KAFKA_PURGE_F_INFLIGHT 0x2
4967 
4968 
4972 #define RD_KAFKA_PURGE_F_NON_BLOCKING 0x4
4973 
4974 
4989 typedef struct rd_kafka_metadata_broker {
4990  int32_t id;
4991  char *host;
4992  int port;
4994 
4998 typedef struct rd_kafka_metadata_partition {
4999  int32_t id;
5001  int32_t leader;
5003  int32_t *replicas;
5004  int isr_cnt;
5005  int32_t *isrs;
5007 
5011 typedef struct rd_kafka_metadata_topic {
5012  char *topic;
5014  struct rd_kafka_metadata_partition *partitions;
5017 
5018 
5022 typedef struct rd_kafka_metadata {
5024  struct rd_kafka_metadata_broker *brokers;
5027  struct rd_kafka_metadata_topic *topics;
5029  int32_t orig_broker_id;
5032 
5053 RD_EXPORT
5055 rd_kafka_metadata(rd_kafka_t *rk,
5056  int all_topics,
5057  rd_kafka_topic_t *only_rkt,
5058  const struct rd_kafka_metadata **metadatap,
5059  int timeout_ms);
5060 
5064 RD_EXPORT
5065 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
5066 
5070 typedef struct rd_kafka_Node_s rd_kafka_Node_t;
5071 
5079 RD_EXPORT
5081 
5092 RD_EXPORT
5093 const char *rd_kafka_Node_host(const rd_kafka_Node_t *node);
5094 
5102 RD_EXPORT
5103 uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node);
5104 
5112 RD_EXPORT
5113 const char *rd_kafka_Node_rack(const rd_kafka_Node_t *node);
5114 
5135  char *member_id;
5136  char *client_id;
5137  char *client_host;
5144 };
5145 
5151 typedef enum {
5152  RD_KAFKA_CONSUMER_GROUP_STATE_UNKNOWN = 0,
5153  RD_KAFKA_CONSUMER_GROUP_STATE_PREPARING_REBALANCE = 1,
5154  RD_KAFKA_CONSUMER_GROUP_STATE_COMPLETING_REBALANCE = 2,
5155  RD_KAFKA_CONSUMER_GROUP_STATE_STABLE = 3,
5156  RD_KAFKA_CONSUMER_GROUP_STATE_DEAD = 4,
5157  RD_KAFKA_CONSUMER_GROUP_STATE_EMPTY = 5,
5158  RD_KAFKA_CONSUMER_GROUP_STATE__CNT
5160 
5165  struct rd_kafka_metadata_broker broker;
5166  char *group;
5168  char *state;
5170  char *protocol;
5173 };
5174 
5183 };
5184 
5185 
5216 RD_EXPORT
5218 rd_kafka_list_groups(rd_kafka_t *rk,
5219  const char *group,
5220  const struct rd_kafka_group_list **grplistp,
5221  int timeout_ms);
5222 
5230 RD_EXPORT
5231 const char *
5233 
5241 RD_EXPORT
5244 
5248 RD_EXPORT
5250 
5251 
5295 RD_EXPORT
5296 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
5297 
5298 
5299 
5312 RD_EXPORT RD_DEPRECATED void
5313 rd_kafka_set_logger(rd_kafka_t *rk,
5314  void (*func)(const rd_kafka_t *rk,
5315  int level,
5316  const char *fac,
5317  const char *buf));
5318 
5319 
5329 RD_EXPORT
5330 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
5331 
5332 
5336 RD_EXPORT
5337 void rd_kafka_log_print(const rd_kafka_t *rk,
5338  int level,
5339  const char *fac,
5340  const char *buf);
5341 
5342 
5348 RD_EXPORT
5349 void rd_kafka_log_syslog(const rd_kafka_t *rk,
5350  int level,
5351  const char *fac,
5352  const char *buf);
5353 
5354 
5377 RD_EXPORT
5378 int rd_kafka_outq_len(rd_kafka_t *rk);
5379 
5380 
5381 
5388 RD_EXPORT
5389 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
5390 
5391 
5392 
5398 RD_EXPORT
5400 
5401 
5414 
5415 
5424 RD_EXPORT
5425 int rd_kafka_wait_destroyed(int timeout_ms);
5426 
5427 
5433 RD_EXPORT
5435 
5436 
5453 RD_EXPORT
5455 
5456 
5473 #define RD_KAFKA_EVENT_NONE 0x0
5474 #define RD_KAFKA_EVENT_DR 0x1
5475 #define RD_KAFKA_EVENT_FETCH 0x2
5476 #define RD_KAFKA_EVENT_LOG 0x4
5477 #define RD_KAFKA_EVENT_ERROR 0x8
5478 #define RD_KAFKA_EVENT_REBALANCE 0x10
5479 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
5480 #define RD_KAFKA_EVENT_STATS 0x40
5481 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
5482 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
5483 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT \
5484  102
5485 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
5486 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT \
5487  104
5488 #define RD_KAFKA_EVENT_DELETERECORDS_RESULT 105
5489 #define RD_KAFKA_EVENT_DELETEGROUPS_RESULT 106
5491 #define RD_KAFKA_EVENT_DELETECONSUMERGROUPOFFSETS_RESULT 107
5493 #define RD_KAFKA_EVENT_OAUTHBEARER_TOKEN_REFRESH 0x100
5494 #define RD_KAFKA_EVENT_BACKGROUND 0x200
5495 #define RD_KAFKA_EVENT_CREATEACLS_RESULT 0x400
5496 #define RD_KAFKA_EVENT_DESCRIBEACLS_RESULT 0x800
5497 #define RD_KAFKA_EVENT_DELETEACLS_RESULT 0x1000
5499 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPS_RESULT 0x2000
5501 #define RD_KAFKA_EVENT_DESCRIBECONSUMERGROUPS_RESULT 0x4000
5503 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPOFFSETS_RESULT 0x8000
5505 #define RD_KAFKA_EVENT_ALTERCONSUMERGROUPOFFSETS_RESULT 0x10000
5507 #define RD_KAFKA_EVENT_INCREMENTALALTERCONFIGS_RESULT 0x20000
5509 #define RD_KAFKA_EVENT_DESCRIBEUSERSCRAMCREDENTIALS_RESULT 0x40000
5511 #define RD_KAFKA_EVENT_ALTERUSERSCRAMCREDENTIALS_RESULT 0x80000
5513 #define RD_KAFKA_EVENT_DESCRIBETOPICS_RESULT 0x100000
5515 #define RD_KAFKA_EVENT_DESCRIBECLUSTER_RESULT 0x200000
5517 #define RD_KAFKA_EVENT_LISTOFFSETS_RESULT 0x400000
5518 
5525 RD_EXPORT
5526 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev);
5527 
5534 RD_EXPORT
5535 const char *rd_kafka_event_name(const rd_kafka_event_t *rkev);
5536 
5537 
5547 RD_EXPORT
5548 void rd_kafka_event_destroy(rd_kafka_event_t *rkev);
5549 
5550 
5566 RD_EXPORT
5567 const rd_kafka_message_t *rd_kafka_event_message_next(rd_kafka_event_t *rkev);
5568 
5569 
5583 RD_EXPORT
5584 size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev,
5585  const rd_kafka_message_t **rkmessages,
5586  size_t size);
5587 
5588 
5596 RD_EXPORT
5597 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev);
5598 
5599 
5611 RD_EXPORT
5612 const char *rd_kafka_event_config_string(rd_kafka_event_t *rkev);
5613 
5614 
5623 RD_EXPORT
5625 
5626 
5635 RD_EXPORT
5636 const char *rd_kafka_event_error_string(rd_kafka_event_t *rkev);
5637 
5638 
5647 RD_EXPORT
5648 int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev);
5649 
5650 
5677 RD_EXPORT
5678 void *rd_kafka_event_opaque(rd_kafka_event_t *rkev);
5679 
5680 
5689 RD_EXPORT
5690 int rd_kafka_event_log(rd_kafka_event_t *rkev,
5691  const char **fac,
5692  const char **str,
5693  int *level);
5694 
5695 
5707 RD_EXPORT
5708 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev,
5709  char *dst,
5710  size_t dstsize);
5711 
5712 
5725 RD_EXPORT
5726 const char *rd_kafka_event_stats(rd_kafka_event_t *rkev);
5727 
5728 
5740 rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev);
5741 
5742 
5753 RD_EXPORT rd_kafka_topic_partition_t *
5754 rd_kafka_event_topic_partition(rd_kafka_event_t *rkev);
5755 
5756 
5758 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
5760 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
5762 typedef rd_kafka_event_t rd_kafka_CreateAcls_result_t;
5764 typedef rd_kafka_event_t rd_kafka_DescribeAcls_result_t;
5766 typedef rd_kafka_event_t rd_kafka_DeleteAcls_result_t;
5768 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
5770 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
5774 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
5776 typedef rd_kafka_event_t rd_kafka_DeleteRecords_result_t;
5778 typedef rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t;
5782 typedef rd_kafka_event_t rd_kafka_DeleteGroups_result_t;
5790 typedef rd_kafka_event_t rd_kafka_DescribeTopics_result_t;
5792 typedef rd_kafka_event_t rd_kafka_DescribeCluster_result_t;
5798 typedef rd_kafka_event_t rd_kafka_ListOffsets_result_t;
5799 
5809 RD_EXPORT const rd_kafka_CreateTopics_result_t *
5810 rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev);
5811 
5821 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
5822 rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev);
5823 
5833 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
5835 
5845 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
5846 rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev);
5847 
5859 
5869 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
5871 
5879 RD_EXPORT const rd_kafka_DeleteRecords_result_t *
5880 rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev);
5881 
5894 RD_EXPORT const rd_kafka_ListConsumerGroups_result_t *
5896 
5911 
5924 RD_EXPORT const rd_kafka_DescribeTopics_result_t *
5926 
5939 RD_EXPORT const rd_kafka_DescribeCluster_result_t *
5950 RD_EXPORT const rd_kafka_DeleteGroups_result_t *
5951 rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev);
5952 
5964 
5972 RD_EXPORT const rd_kafka_CreateAcls_result_t *
5973 rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev);
5974 
5982 RD_EXPORT const rd_kafka_DescribeAcls_result_t *
5983 rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev);
5984 
5992 RD_EXPORT const rd_kafka_DeleteAcls_result_t *
5993 rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev);
5994 
6009 
6024 
6037 RD_EXPORT const rd_kafka_ListOffsets_result_t *
6038 rd_kafka_event_ListOffsets_result(rd_kafka_event_t *rkev);
6039 
6040 
6055 
6070 
6080 RD_EXPORT
6081 rd_kafka_event_t *rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms);
6082 
6097 RD_EXPORT
6098 int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms);
6099 
6100 
6145  rd_kafka_conf_t *conf,
6146  void **plug_opaquep,
6147  char *errstr,
6148  size_t errstr_size);
6149 
6231  rd_kafka_conf_t *conf,
6232  const char *name,
6233  const char *val,
6234  char *errstr,
6235  size_t errstr_size,
6236  void *ic_opaque);
6237 
6238 
6261  rd_kafka_conf_t *new_conf,
6262  const rd_kafka_conf_t *old_conf,
6263  size_t filter_cnt,
6264  const char **filter,
6265  void *ic_opaque);
6266 
6267 
6275  void *ic_opaque);
6276 
6277 
6296  rd_kafka_t *rk,
6297  const rd_kafka_conf_t *conf,
6298  void *ic_opaque,
6299  char *errstr,
6300  size_t errstr_size);
6301 
6302 
6311  rd_kafka_interceptor_f_on_destroy_t)(rd_kafka_t *rk, void *ic_opaque);
6312 
6313 
6314 
6336  rd_kafka_t *rk,
6337  rd_kafka_message_t *rkmessage,
6338  void *ic_opaque);
6339 
6363  rd_kafka_t *rk,
6364  rd_kafka_message_t *rkmessage,
6365  void *ic_opaque);
6366 
6367 
6385  rd_kafka_t *rk,
6386  rd_kafka_message_t *rkmessage,
6387  void *ic_opaque);
6388 
6411  rd_kafka_t *rk,
6412  const rd_kafka_topic_partition_list_t *offsets,
6414  void *ic_opaque);
6415 
6416 
6439  rd_kafka_t *rk,
6440  int sockfd,
6441  const char *brokername,
6442  int32_t brokerid,
6443  int16_t ApiKey,
6444  int16_t ApiVersion,
6445  int32_t CorrId,
6446  size_t size,
6447  void *ic_opaque);
6448 
6449 
6476  rd_kafka_t *rk,
6477  int sockfd,
6478  const char *brokername,
6479  int32_t brokerid,
6480  int16_t ApiKey,
6481  int16_t ApiVersion,
6482  int32_t CorrId,
6483  size_t size,
6484  int64_t rtt,
6486  void *ic_opaque);
6487 
6488 
6506  rd_kafka_t *rk,
6507  rd_kafka_thread_type_t thread_type,
6508  const char *thread_name,
6509  void *ic_opaque);
6510 
6511 
6532  rd_kafka_t *rk,
6533  rd_kafka_thread_type_t thread_type,
6534  const char *thread_name,
6535  void *ic_opaque);
6536 
6537 
6553  rd_kafka_t *rk,
6554  int32_t broker_id,
6555  const char *secproto,
6556  const char *name,
6557  int port,
6558  const char *state,
6559  void *ic_opaque);
6560 
6561 
6575  rd_kafka_conf_t *conf,
6576  const char *ic_name,
6578  void *ic_opaque);
6579 
6580 
6594  rd_kafka_conf_t *conf,
6595  const char *ic_name,
6597  void *ic_opaque);
6598 
6613  rd_kafka_conf_t *conf,
6614  const char *ic_name,
6616  void *ic_opaque);
6617 
6618 
6640 RD_EXPORT rd_kafka_resp_err_t
6642  const char *ic_name,
6644  void *ic_opaque);
6645 
6646 
6647 
6661  rd_kafka_t *rk,
6662  const char *ic_name,
6664  void *ic_opaque);
6665 
6666 
6679 RD_EXPORT rd_kafka_resp_err_t
6681  const char *ic_name,
6683  void *ic_opaque);
6684 
6698  rd_kafka_t *rk,
6699  const char *ic_name,
6700  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
6701  void *ic_opaque);
6702 
6703 
6717  rd_kafka_t *rk,
6718  const char *ic_name,
6720  void *ic_opaque);
6721 
6722 
6736  rd_kafka_t *rk,
6737  const char *ic_name,
6739  void *ic_opaque);
6740 
6741 
6755  rd_kafka_t *rk,
6756  const char *ic_name,
6758  void *ic_opaque);
6759 
6760 
6774  rd_kafka_t *rk,
6775  const char *ic_name,
6776  rd_kafka_interceptor_f_on_response_received_t *on_response_received,
6777  void *ic_opaque);
6778 
6779 
6793  rd_kafka_t *rk,
6794  const char *ic_name,
6796  void *ic_opaque);
6797 
6798 
6812  rd_kafka_t *rk,
6813  const char *ic_name,
6815  void *ic_opaque);
6816 
6817 
6830 RD_EXPORT
6832  rd_kafka_t *rk,
6833  const char *ic_name,
6834  rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change,
6835  void *ic_opaque);
6836 
6837 
6838 
6859 RD_EXPORT rd_kafka_resp_err_t
6860 rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres);
6861 
6868 RD_EXPORT const char *
6869 rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres);
6870 
6876 RD_EXPORT const char *
6877 rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres);
6878 
6888 RD_EXPORT const rd_kafka_error_t *
6889 rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres);
6890 
6896 RD_EXPORT const char *
6897 rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres);
6898 
6904 RD_EXPORT const rd_kafka_topic_partition_list_t *
6905 rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres);
6906 
6907 
6954 typedef enum rd_kafka_admin_op_t {
6985 
6998 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
6999 
7008  RD_KAFKA_ISOLATION_LEVEL_READ_UNCOMMITTED = 0,
7009  RD_KAFKA_ISOLATION_LEVEL_READ_COMMITTED = 1
7011 
7031 RD_EXPORT rd_kafka_AdminOptions_t *
7033 
7034 
7039 
7040 
7058 RD_EXPORT rd_kafka_resp_err_t
7060  int timeout_ms,
7061  char *errstr,
7062  size_t errstr_size);
7063 
7064 
7091 RD_EXPORT rd_kafka_resp_err_t
7093  int timeout_ms,
7094  char *errstr,
7095  size_t errstr_size);
7096 
7097 
7116 RD_EXPORT rd_kafka_resp_err_t
7118  int true_or_false,
7119  char *errstr,
7120  size_t errstr_size);
7121 
7122 
7150 RD_EXPORT rd_kafka_resp_err_t
7152  int32_t broker_id,
7153  char *errstr,
7154  size_t errstr_size);
7155 
7156 
7169 RD_EXPORT
7171  rd_kafka_AdminOptions_t *options,
7172  int true_or_false);
7173 
7188 RD_EXPORT
7190  rd_kafka_AdminOptions_t *options,
7191  int true_or_false);
7192 
7205 RD_EXPORT
7207  rd_kafka_AdminOptions_t *options,
7208  const rd_kafka_consumer_group_state_t *consumer_group_states,
7209  size_t consumer_group_states_cnt);
7210 
7214 RD_EXPORT
7215 rd_kafka_error_t *
7218 
7223 RD_EXPORT void
7225  void *ev_opaque);
7226 
7227 
7228 
7237  1,
7246  9,
7248  10,
7250  11,
7252  12,
7253  RD_KAFKA_ACL_OPERATION__CNT
7255 
7267 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
7268 
7290 RD_EXPORT rd_kafka_NewTopic_t *rd_kafka_NewTopic_new(const char *topic,
7291  int num_partitions,
7292  int replication_factor,
7293  char *errstr,
7294  size_t errstr_size);
7295 
7301 
7302 
7309  size_t new_topic_cnt);
7310 
7311 
7333 RD_EXPORT rd_kafka_resp_err_t
7335  int32_t partition,
7336  int32_t *broker_ids,
7337  size_t broker_id_cnt,
7338  char *errstr,
7339  size_t errstr_size);
7340 
7353 RD_EXPORT rd_kafka_resp_err_t
7355  const char *name,
7356  const char *value);
7357 
7358 
7377 RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk,
7378  rd_kafka_NewTopic_t **new_topics,
7379  size_t new_topic_cnt,
7380  const rd_kafka_AdminOptions_t *options,
7381  rd_kafka_queue_t *rkqu);
7382 
7383 
7384 /*
7385  * CreateTopics result type and methods
7386  */
7387 
7396 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_CreateTopics_result_topics(
7397  const rd_kafka_CreateTopics_result_t *result,
7398  size_t *cntp);
7399 
7400 
7401 
7402 /*
7403  * DeleteTopics - delete topics from cluster
7404  *
7405  */
7406 
7408 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
7409 
7419 RD_EXPORT rd_kafka_DeleteTopic_t *rd_kafka_DeleteTopic_new(const char *topic);
7420 
7426 
7432 RD_EXPORT void
7434  size_t del_topic_cnt);
7435 
7449 RD_EXPORT
7450 void rd_kafka_DeleteTopics(rd_kafka_t *rk,
7451  rd_kafka_DeleteTopic_t **del_topics,
7452  size_t del_topic_cnt,
7453  const rd_kafka_AdminOptions_t *options,
7454  rd_kafka_queue_t *rkqu);
7455 
7456 
7457 
7458 /*
7459  * DeleteTopics result type and methods
7460  */
7461 
7470 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_DeleteTopics_result_topics(
7471  const rd_kafka_DeleteTopics_result_t *result,
7472  size_t *cntp);
7473 
7474 
7485 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
7486 
7503 RD_EXPORT rd_kafka_NewPartitions_t *
7504 rd_kafka_NewPartitions_new(const char *topic,
7505  size_t new_total_cnt,
7506  char *errstr,
7507  size_t errstr_size);
7508 
7513 RD_EXPORT void
7515 
7521 RD_EXPORT void
7523  size_t new_parts_cnt);
7524 
7548  rd_kafka_NewPartitions_t *new_parts,
7549  int32_t new_partition_idx,
7550  int32_t *broker_ids,
7551  size_t broker_id_cnt,
7552  char *errstr,
7553  size_t errstr_size);
7554 
7555 
7574 RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk,
7575  rd_kafka_NewPartitions_t **new_parts,
7576  size_t new_parts_cnt,
7577  const rd_kafka_AdminOptions_t *options,
7578  rd_kafka_queue_t *rkqu);
7579 
7580 
7581 
7582 /*
7583  * CreatePartitions result type and methods
7584  */
7585 
7594 RD_EXPORT const rd_kafka_topic_result_t **
7596  const rd_kafka_CreatePartitions_result_t *result,
7597  size_t *cntp);
7598 
7634 
7638 
7639 
7643 RD_EXPORT const char *
7645 
7646 
7648 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
7649 
7653 RD_EXPORT const char *
7655 
7660 RD_EXPORT const char *
7662 
7666 RD_EXPORT rd_kafka_ConfigSource_t
7668 
7673 RD_EXPORT int
7675 
7681 RD_EXPORT int
7683 
7691 RD_EXPORT int
7693 
7697 RD_EXPORT int
7699 
7700 
7711 RD_EXPORT const rd_kafka_ConfigEntry_t **
7713  size_t *cntp);
7714 
7715 
7716 
7729 
7745  RD_KAFKA_RESOURCE_PATTERN_TYPE__CNT,
7747 
7753  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SET = 0,
7754  RD_KAFKA_ALTER_CONFIG_OP_TYPE_DELETE = 1,
7755  RD_KAFKA_ALTER_CONFIG_OP_TYPE_APPEND = 2,
7756  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SUBTRACT = 3,
7757  RD_KAFKA_ALTER_CONFIG_OP_TYPE__CNT,
7759 
7764  rd_kafka_ResourcePatternType_t resource_pattern_type);
7765 
7769 RD_EXPORT const char *
7771 
7773 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
7774 
7775 
7784 RD_EXPORT rd_kafka_ConfigResource_t *
7786  const char *resname);
7787 
7792 RD_EXPORT void
7794 
7795 
7801 RD_EXPORT void
7803  size_t config_cnt);
7804 
7805 
7819 RD_EXPORT rd_kafka_resp_err_t
7821  const char *name,
7822  const char *value);
7823 
7824 
7844  rd_kafka_ConfigResource_t *config,
7845  const char *name,
7847  const char *value);
7848 
7849 
7858 RD_EXPORT const rd_kafka_ConfigEntry_t **
7860  size_t *cntp);
7861 
7862 
7863 
7867 RD_EXPORT rd_kafka_ResourceType_t
7869 
7873 RD_EXPORT const char *
7875 
7879 RD_EXPORT rd_kafka_resp_err_t
7881 
7886 RD_EXPORT const char *
7888 
7889 
7890 /*
7891  * AlterConfigs - alter cluster configuration.
7892  *
7893  */
7894 
7895 
7918 RD_EXPORT
7919 void rd_kafka_AlterConfigs(rd_kafka_t *rk,
7920  rd_kafka_ConfigResource_t **configs,
7921  size_t config_cnt,
7922  const rd_kafka_AdminOptions_t *options,
7923  rd_kafka_queue_t *rkqu);
7924 
7925 
7926 /*
7927  * AlterConfigs result type and methods
7928  */
7929 
7944 RD_EXPORT const rd_kafka_ConfigResource_t **
7946  const rd_kafka_AlterConfigs_result_t *result,
7947  size_t *cntp);
7948 
7949 
7950 
7951 /*
7952  * IncrementalAlterConfigs - alter cluster configuration incrementally.
7953  *
7954  */
7955 
7956 
7977 RD_EXPORT
7979  rd_kafka_ConfigResource_t **configs,
7980  size_t config_cnt,
7981  const rd_kafka_AdminOptions_t *options,
7982  rd_kafka_queue_t *rkqu);
7983 
7984 
7985 /*
7986  * IncrementalAlterConfigs result type and methods
7987  */
7988 
8004 RD_EXPORT const rd_kafka_ConfigResource_t **
8007  size_t *cntp);
8008 
8009 
8010 
8011 /*
8012  * DescribeConfigs - retrieve cluster configuration.
8013  *
8014  */
8015 
8016 
8041 RD_EXPORT
8042 void rd_kafka_DescribeConfigs(rd_kafka_t *rk,
8043  rd_kafka_ConfigResource_t **configs,
8044  size_t config_cnt,
8045  const rd_kafka_AdminOptions_t *options,
8046  rd_kafka_queue_t *rkqu);
8047 
8048 
8049 
8050 /*
8051  * DescribeConfigs result type and methods
8052  */
8053 
8062 RD_EXPORT const rd_kafka_ConfigResource_t **
8064  const rd_kafka_DescribeConfigs_result_t *result,
8065  size_t *cntp);
8066 
8067 
8078 typedef struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t;
8079 
8097  const rd_kafka_topic_partition_list_t *before_offsets);
8098 
8103 RD_EXPORT void
8105 
8111 RD_EXPORT void
8113  size_t del_record_cnt);
8114 
8136 RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk,
8137  rd_kafka_DeleteRecords_t **del_records,
8138  size_t del_record_cnt,
8139  const rd_kafka_AdminOptions_t *options,
8140  rd_kafka_queue_t *rkqu);
8141 
8142 
8143 /*
8144  * DeleteRecords result type and methods
8145  */
8146 
8156 RD_EXPORT const rd_kafka_topic_partition_list_t *
8158  const rd_kafka_DeleteRecords_result_t *result);
8159 
8171 typedef struct rd_kafka_TopicCollection_s rd_kafka_TopicCollection_t;
8172 
8177 typedef struct rd_kafka_TopicPartitionInfo_s rd_kafka_TopicPartitionInfo_t;
8178 
8183 typedef struct rd_kafka_TopicDescription_s rd_kafka_TopicDescription_t;
8184 
8194 RD_EXPORT
8196 rd_kafka_TopicCollection_of_topic_names(const char **topics, size_t topics_cnt);
8197 
8202 RD_EXPORT void
8204 
8219 RD_EXPORT
8220 void rd_kafka_DescribeTopics(rd_kafka_t *rk,
8221  const rd_kafka_TopicCollection_t *topics,
8222  const rd_kafka_AdminOptions_t *options,
8223  rd_kafka_queue_t *rkqu);
8224 
8234 RD_EXPORT
8236  const rd_kafka_DescribeTopics_result_t *result,
8237  size_t *cntp);
8238 
8239 
8250 RD_EXPORT
8252  const rd_kafka_TopicDescription_t *topicdesc,
8253  size_t *cntp);
8254 
8255 
8263 RD_EXPORT
8265  const rd_kafka_TopicPartitionInfo_t *partition);
8266 
8267 
8278 RD_EXPORT
8280  const rd_kafka_TopicPartitionInfo_t *partition);
8281 
8293 RD_EXPORT
8294 const rd_kafka_Node_t **
8296  size_t *cntp);
8297 
8309 RD_EXPORT
8311  const rd_kafka_TopicPartitionInfo_t *partition,
8312  size_t *cntp);
8313 
8326 RD_EXPORT
8328  const rd_kafka_TopicDescription_t *topicdesc,
8329  size_t *cntp);
8330 
8341 RD_EXPORT
8342 const char *
8344 
8354 RD_EXPORT const rd_kafka_Uuid_t *rd_kafka_TopicDescription_topic_id(
8355  const rd_kafka_TopicDescription_t *topicdesc);
8356 
8364 RD_EXPORT
8366  const rd_kafka_TopicDescription_t *topicdesc);
8367 
8378 RD_EXPORT
8379 const rd_kafka_error_t *
8381 
8382 
8402 RD_EXPORT
8403 void rd_kafka_DescribeCluster(rd_kafka_t *rk,
8404  const rd_kafka_AdminOptions_t *options,
8405  rd_kafka_queue_t *rkqu);
8406 
8417 RD_EXPORT
8419  const rd_kafka_DescribeCluster_result_t *result,
8420  size_t *cntp);
8421 
8433 RD_EXPORT
8436  const rd_kafka_DescribeCluster_result_t *result,
8437  size_t *cntp);
8438 
8446 RD_EXPORT
8448  const rd_kafka_DescribeCluster_result_t *result);
8449 
8459 RD_EXPORT
8461  const rd_kafka_DescribeCluster_result_t *result);
8462 
8477 typedef struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t;
8478 
8480 typedef struct rd_kafka_ListConsumerGroupsResult_s
8482 
8493 RD_EXPORT
8494 void rd_kafka_ListConsumerGroups(rd_kafka_t *rk,
8495  const rd_kafka_AdminOptions_t *options,
8496  rd_kafka_queue_t *rkqu);
8497 
8508 RD_EXPORT
8510  const rd_kafka_ConsumerGroupListing_t *grplist);
8511 
8520 RD_EXPORT
8522  const rd_kafka_ConsumerGroupListing_t *grplist);
8523 
8531 RD_EXPORT
8533  const rd_kafka_ConsumerGroupListing_t *grplist);
8534 
8546 RD_EXPORT
8550  size_t *cntp);
8551 
8565 RD_EXPORT
8568  size_t *cntp);
8569 
8581 typedef struct rd_kafka_ConsumerGroupDescription_s
8583 
8588 typedef struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t;
8589 
8594 typedef struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t;
8595 
8611 RD_EXPORT
8613  const char **groups,
8614  size_t groups_cnt,
8615  const rd_kafka_AdminOptions_t *options,
8616  rd_kafka_queue_t *rkqu);
8617 
8629 RD_EXPORT
8633  size_t *cntp);
8634 
8635 
8646 RD_EXPORT
8648  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8649 
8660 RD_EXPORT
8662  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8663 
8671 RD_EXPORT
8673  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8674 
8675 
8686 RD_EXPORT
8688  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8689 
8702 RD_EXPORT
8705  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8706  size_t *cntp);
8707 
8715 RD_EXPORT
8717  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8718 
8729 RD_EXPORT
8731  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8732 
8740 RD_EXPORT
8742  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8743 
8756 RD_EXPORT
8758  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8759  size_t idx);
8760 
8771 RD_EXPORT
8773  const rd_kafka_MemberDescription_t *member);
8774 
8785 RD_EXPORT
8787  const rd_kafka_MemberDescription_t *member);
8788 
8799 RD_EXPORT
8801  const rd_kafka_MemberDescription_t *member);
8802 
8813 RD_EXPORT
8814 const char *
8816 
8827 RD_EXPORT
8829  const rd_kafka_MemberDescription_t *member);
8830 
8841 RD_EXPORT
8843  const rd_kafka_MemberAssignment_t *assignment);
8844 
8856 typedef struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t;
8857 
8867 RD_EXPORT
8869 
8874 RD_EXPORT
8876 
8882 RD_EXPORT void
8884  size_t del_group_cnt);
8885 
8901 RD_EXPORT
8902 void rd_kafka_DeleteGroups(rd_kafka_t *rk,
8903  rd_kafka_DeleteGroup_t **del_groups,
8904  size_t del_group_cnt,
8905  const rd_kafka_AdminOptions_t *options,
8906  rd_kafka_queue_t *rkqu);
8907 
8908 
8909 
8910 /*
8911  * DeleteGroups result type and methods
8912  */
8913 
8922 RD_EXPORT const rd_kafka_group_result_t **rd_kafka_DeleteGroups_result_groups(
8923  const rd_kafka_DeleteGroups_result_t *result,
8924  size_t *cntp);
8925 
8936 typedef struct rd_kafka_ListConsumerGroupOffsets_s
8938 
8953  const char *group_id,
8954  const rd_kafka_topic_partition_list_t *partitions);
8955 
8961  rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets);
8962 
8969  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8970  size_t list_grpoffset_cnt);
8971 
8989 RD_EXPORT
8991  rd_kafka_t *rk,
8992  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8993  size_t list_grpoffsets_cnt,
8994  const rd_kafka_AdminOptions_t *options,
8995  rd_kafka_queue_t *rkqu);
8996 
8997 
8998 
8999 /*
9000  * ListConsumerGroupOffsets result type and methods
9001  */
9002 
9014 RD_EXPORT const rd_kafka_group_result_t **
9017  size_t *cntp);
9018 
9019 
9020 
9031 typedef struct rd_kafka_AlterConsumerGroupOffsets_s
9033 
9048  const char *group_id,
9049  const rd_kafka_topic_partition_list_t *partitions);
9050 
9056  rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets);
9057 
9064  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9065  size_t alter_grpoffset_cnt);
9066 
9085 RD_EXPORT
9087  rd_kafka_t *rk,
9088  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9089  size_t alter_grpoffsets_cnt,
9090  const rd_kafka_AdminOptions_t *options,
9091  rd_kafka_queue_t *rkqu);
9092 
9093 
9094 
9095 /*
9096  * AlterConsumerGroupOffsets result type and methods
9097  */
9098 
9110 RD_EXPORT const rd_kafka_group_result_t **
9113  size_t *cntp);
9114 
9115 
9116 
9127 typedef struct rd_kafka_DeleteConsumerGroupOffsets_s
9129 
9144  const char *group,
9145  const rd_kafka_topic_partition_list_t *partitions);
9146 
9152  rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets);
9153 
9160  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9161  size_t del_grpoffset_cnt);
9162 
9181 RD_EXPORT
9183  rd_kafka_t *rk,
9184  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9185  size_t del_grpoffsets_cnt,
9186  const rd_kafka_AdminOptions_t *options,
9187  rd_kafka_queue_t *rkqu);
9188 
9189 
9190 
9191 /*
9192  * DeleteConsumerGroupOffsets result type and methods
9193  */
9194 
9203 RD_EXPORT const rd_kafka_group_result_t **
9206  size_t *cntp);
9207 
9221  /* Used to retrieve the offset with the largest timestamp of a partition
9222  * as message timestamps can be specified client side this may not match
9223  * the log end offset returned by SPEC_LATEST.
9224  */
9225  RD_KAFKA_OFFSET_SPEC_MAX_TIMESTAMP = -3,
9226  /* Used to retrieve the offset with the earliest timestamp of a
9227  partition. */
9228  RD_KAFKA_OFFSET_SPEC_EARLIEST = -2,
9229  /* Used to retrieve the offset with the latest timestamp of a partition.
9230  */
9231  RD_KAFKA_OFFSET_SPEC_LATEST = -1,
9233 
9238 typedef struct rd_kafka_ListOffsetsResultInfo_s
9240 
9244 RD_EXPORT
9247  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9248 
9252 RD_EXPORT
9254  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9255 
9260 RD_EXPORT
9263  size_t *cntp);
9264 
9289 RD_EXPORT
9290 void rd_kafka_ListOffsets(rd_kafka_t *rk,
9291  rd_kafka_topic_partition_list_t *topic_partitions,
9292  const rd_kafka_AdminOptions_t *options,
9293  rd_kafka_queue_t *rkqu);
9294 
9307  RD_KAFKA_SCRAM_MECHANISM_UNKNOWN = 0,
9308  RD_KAFKA_SCRAM_MECHANISM_SHA_256 = 1,
9309  RD_KAFKA_SCRAM_MECHANISM_SHA_512 = 2,
9310  RD_KAFKA_SCRAM_MECHANISM__CNT
9312 
9318 typedef struct rd_kafka_ScramCredentialInfo_s rd_kafka_ScramCredentialInfo_t;
9319 
9323 RD_EXPORT
9325  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9326 
9330 RD_EXPORT
9332  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9333 
9340 typedef struct rd_kafka_UserScramCredentialsDescription_s
9342 
9346 RD_EXPORT
9348  const rd_kafka_UserScramCredentialsDescription_t *description);
9349 
9353 RD_EXPORT
9355  const rd_kafka_UserScramCredentialsDescription_t *description);
9356 
9361 RD_EXPORT
9363  const rd_kafka_UserScramCredentialsDescription_t *description);
9364 
9369 RD_EXPORT
9372  const rd_kafka_UserScramCredentialsDescription_t *description,
9373  size_t idx);
9374 
9384 RD_EXPORT
9388  size_t *cntp);
9389 
9401 RD_EXPORT
9403  rd_kafka_t *rk,
9404  const char **users,
9405  size_t user_cnt,
9406  const rd_kafka_AdminOptions_t *options,
9407  rd_kafka_queue_t *rkqu);
9408 
9412 typedef struct rd_kafka_UserScramCredentialAlteration_s
9414 
9435 RD_EXPORT
9438  rd_kafka_ScramMechanism_t mechanism,
9439  int32_t iterations,
9440  const unsigned char *password,
9441  size_t password_size,
9442  const unsigned char *salt,
9443  size_t salt_size);
9444 
9454 RD_EXPORT
9457  rd_kafka_ScramMechanism_t mechanism);
9458 
9459 
9463 RD_EXPORT
9466 
9470 RD_EXPORT
9473  size_t alteration_cnt);
9474 
9478 typedef struct rd_kafka_AlterUserScramCredentials_result_response_s
9480 
9485 RD_EXPORT
9488 
9493 RD_EXPORT
9494 const rd_kafka_error_t *
9497 
9506 RD_EXPORT
9510  size_t *cntp);
9511 
9525 RD_EXPORT
9527  rd_kafka_t *rk,
9529  size_t alteration_cnt,
9530  const rd_kafka_AdminOptions_t *options,
9531  rd_kafka_queue_t *rkqu);
9532 
9545 typedef struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t;
9546 
9552 
9556 RD_EXPORT const rd_kafka_error_t *
9557 rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres);
9558 
9559 
9563 RD_EXPORT const char *
9565 
9573  1,
9576  RD_KAFKA_ACL_PERMISSION_TYPE__CNT
9578 
9582 RD_EXPORT const char *rd_kafka_AclPermissionType_name(
9583  rd_kafka_AclPermissionType_t acl_permission_type);
9584 
9603 RD_EXPORT rd_kafka_AclBinding_t *
9605  const char *name,
9606  rd_kafka_ResourcePatternType_t resource_pattern_type,
9607  const char *principal,
9608  const char *host,
9609  rd_kafka_AclOperation_t operation,
9610  rd_kafka_AclPermissionType_t permission_type,
9611  char *errstr,
9612  size_t errstr_size);
9613 
9640  rd_kafka_ResourceType_t restype,
9641  const char *name,
9642  rd_kafka_ResourcePatternType_t resource_pattern_type,
9643  const char *principal,
9644  const char *host,
9645  rd_kafka_AclOperation_t operation,
9646  rd_kafka_AclPermissionType_t permission_type,
9647  char *errstr,
9648  size_t errstr_size);
9649 
9653 RD_EXPORT rd_kafka_ResourceType_t
9655 
9661 RD_EXPORT const char *
9663 
9669 RD_EXPORT const char *
9671 
9677 RD_EXPORT const char *
9679 
9683 RD_EXPORT rd_kafka_AclOperation_t
9685 
9691 
9697 
9701 RD_EXPORT const rd_kafka_error_t *
9703 
9704 
9710 
9711 
9717 RD_EXPORT void
9719  size_t acl_bindings_cnt);
9720 
9728 RD_EXPORT const rd_kafka_acl_result_t **
9730  size_t *cntp);
9731 
9748 RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk,
9749  rd_kafka_AclBinding_t **new_acls,
9750  size_t new_acls_cnt,
9751  const rd_kafka_AdminOptions_t *options,
9752  rd_kafka_queue_t *rkqu);
9753 
9767 RD_EXPORT const rd_kafka_AclBinding_t **
9769  size_t *cntp);
9770 
9785 RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk,
9786  rd_kafka_AclBindingFilter_t *acl_filter,
9787  const rd_kafka_AdminOptions_t *options,
9788  rd_kafka_queue_t *rkqu);
9789 
9796 typedef struct rd_kafka_DeleteAcls_result_response_s
9798 
9806 RD_EXPORT const rd_kafka_DeleteAcls_result_response_t **
9808  size_t *cntp);
9809 
9814 RD_EXPORT const rd_kafka_error_t *rd_kafka_DeleteAcls_result_response_error(
9815  const rd_kafka_DeleteAcls_result_response_t *result_response);
9816 
9817 
9824 RD_EXPORT const rd_kafka_AclBinding_t **
9826  const rd_kafka_DeleteAcls_result_response_t *result_response,
9827  size_t *matching_acls_cntp);
9828 
9845 RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk,
9846  rd_kafka_AclBindingFilter_t **del_acls,
9847  size_t del_acls_cnt,
9848  const rd_kafka_AdminOptions_t *options,
9849  rd_kafka_queue_t *rkqu);
9850 
9903 RD_EXPORT
9906  const char *token_value,
9907  int64_t md_lifetime_ms,
9908  const char *md_principal_name,
9909  const char **extensions,
9910  size_t extension_size,
9911  char *errstr,
9912  size_t errstr_size);
9913 
9934 RD_EXPORT
9936  const char *errstr);
9937 
10124 RD_EXPORT
10125 rd_kafka_error_t *rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms);
10126 
10127 
10128 
10173 RD_EXPORT
10174 rd_kafka_error_t *rd_kafka_begin_transaction(rd_kafka_t *rk);
10175 
10176 
10248 RD_EXPORT
10250  rd_kafka_t *rk,
10251  const rd_kafka_topic_partition_list_t *offsets,
10252  const rd_kafka_consumer_group_metadata_t *cgmetadata,
10253  int timeout_ms);
10254 
10255 
10323 RD_EXPORT
10324 rd_kafka_error_t *rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms);
10325 
10326 
10385 RD_EXPORT
10386 rd_kafka_error_t *rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms);
10387 
10388 
10391 /* @cond NO_DOC */
10392 #ifdef __cplusplus
10393 }
10394 #endif
10395 #endif /* _RDKAFKA_H_ */
10396 /* @endcond NO_DOC */
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_new_t *on_new, void *ic_opaque)
Append an on_new() interceptor.
RD_EXPORT const 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_EXPORT int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Murmur2 partitioner (Java compatible).
struct rd_kafka_Node_s rd_kafka_Node_t
Node (broker) information.
Definition: rdkafka.h:5070
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_DeleteConsumerGroupOffsets_result_groups(const rd_kafka_DeleteConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a DeleteConsumerGroupOffsets result.
RD_EXPORT void * rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt)
Get the rkt_opaque pointer that was set in the topic configuration with rd_kafka_topic_conf_set_opaqu...
RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk, rd_kafka_AclBinding_t **new_acls, size_t new_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create acls as specified by the new_acls array of size new_topic_cnt elements.
rd_kafka_resp_err_t
Error codes.
Definition: rdkafka.h:281
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_EXPIRED
Definition: rdkafka.h:570
@ RD_KAFKA_RESP_ERR_POLICY_VIOLATION
Definition: rdkafka.h:520
@ RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafka.h:541
@ RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
Definition: rdkafka.h:407
@ RD_KAFKA_RESP_ERR_RESOURCE_NOT_FOUND
Definition: rdkafka.h:623
@ RD_KAFKA_RESP_ERR__PURGE_QUEUE
Definition: rdkafka.h:383
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafka.h:518
@ RD_KAFKA_RESP_ERR_OPERATION_NOT_ATTEMPTED
Definition: rdkafka.h:547
@ RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafka.h:534
@ RD_KAFKA_RESP_ERR__INVALID_DIFFERENT_RECORD
Definition: rdkafka.h:412
@ RD_KAFKA_RESP_ERR_INVALID_PRINCIPAL_TYPE
Definition: rdkafka.h:572
@ RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:598
@ RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION
Definition: rdkafka.h:307
@ RD_KAFKA_RESP_ERR__FS
Definition: rdkafka.h:309
@ RD_KAFKA_RESP_ERR__TIMED_OUT
Definition: rdkafka.h:317
@ RD_KAFKA_RESP_ERR_REQUEST_TIMED_OUT
Definition: rdkafka.h:437
@ RD_KAFKA_RESP_ERR_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:431
@ RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
Definition: rdkafka.h:474
@ RD_KAFKA_RESP_ERR__FATAL
Definition: rdkafka.h:387
@ RD_KAFKA_RESP_ERR_STALE_MEMBER_EPOCH
Definition: rdkafka.h:648
@ RD_KAFKA_RESP_ERR__UNDERFLOW
Definition: rdkafka.h:377
@ RD_KAFKA_RESP_ERR__BEGIN
Definition: rdkafka.h:284
@ RD_KAFKA_RESP_ERR__NOENT
Definition: rdkafka.h:375
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_OR_PART
Definition: rdkafka.h:427
@ RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafka.h:468
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafka.h:526
@ RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
Definition: rdkafka.h:449
@ RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
Definition: rdkafka.h:549
@ RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
Definition: rdkafka.h:576
@ RD_KAFKA_RESP_ERR_TOPIC_ALREADY_EXISTS
Definition: rdkafka.h:504
@ RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
Definition: rdkafka.h:456
@ RD_KAFKA_RESP_ERR_MSG_SIZE_TOO_LARGE
Definition: rdkafka.h:443
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTH_DISABLED
Definition: rdkafka.h:560
@ RD_KAFKA_RESP_ERR_DUPLICATE_RESOURCE
Definition: rdkafka.h:625
@ RD_KAFKA_RESP_ERR_INVALID_CONFIG
Definition: rdkafka.h:512
@ RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
Definition: rdkafka.h:634
@ RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
Definition: rdkafka.h:393
@ RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
Definition: rdkafka.h:632
@ RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
Definition: rdkafka.h:329
@ RD_KAFKA_RESP_ERR__ISR_INSUFF
Definition: rdkafka.h:321
@ RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafka.h:554
@ RD_KAFKA_RESP_ERR_UNKNOWN_SUBSCRIPTION_ID
Definition: rdkafka.h:651
@ RD_KAFKA_RESP_ERR_TELEMETRY_TOO_LARGE
Definition: rdkafka.h:654
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_ID_MAPPING
Definition: rdkafka.h:531
@ RD_KAFKA_RESP_ERR__IN_PROGRESS
Definition: rdkafka.h:331
@ RD_KAFKA_RESP_ERR__READ_ONLY
Definition: rdkafka.h:373
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafka.h:472
@ RD_KAFKA_RESP_ERR__UNKNOWN_BROKER
Definition: rdkafka.h:395
@ RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafka.h:478
@ RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
Definition: rdkafka.h:574
@ RD_KAFKA_RESP_ERR_SECURITY_DISABLED
Definition: rdkafka.h:545
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_ASSIGNOR
Definition: rdkafka.h:646
@ RD_KAFKA_RESP_ERR_TOPIC_DELETION_DISABLED
Definition: rdkafka.h:584
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_OWNER_MISMATCH
Definition: rdkafka.h:564
@ RD_KAFKA_RESP_ERR__INVALID_ARG
Definition: rdkafka.h:315
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_VERSION
Definition: rdkafka.h:502
@ RD_KAFKA_RESP_ERR__OUTDATED
Definition: rdkafka.h:353
@ RD_KAFKA_RESP_ERR__FAIL
Definition: rdkafka.h:292
@ RD_KAFKA_RESP_ERR_INVALID_MSG_SIZE
Definition: rdkafka.h:429
@ RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
Definition: rdkafka.h:592
@ RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
Definition: rdkafka.h:451
@ RD_KAFKA_RESP_ERR__ASSIGN_PARTITIONS
Definition: rdkafka.h:337
@ RD_KAFKA_RESP_ERR__VALUE_SERIALIZATION
Definition: rdkafka.h:365
@ RD_KAFKA_RESP_ERR_FETCH_SESSION_ID_NOT_FOUND
Definition: rdkafka.h:578
@ RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
Definition: rdkafka.h:339
@ RD_KAFKA_RESP_ERR__APPLICATION
Definition: rdkafka.h:401
@ RD_KAFKA_RESP_ERR_REPLICA_NOT_AVAILABLE
Definition: rdkafka.h:441
@ RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
Definition: rdkafka.h:335
@ RD_KAFKA_RESP_ERR_NOT_CONTROLLER
Definition: rdkafka.h:514
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_NOT_FOUND
Definition: rdkafka.h:562
@ RD_KAFKA_RESP_ERR_LISTENER_NOT_FOUND
Definition: rdkafka.h:582
@ RD_KAFKA_RESP_ERR_INVALID_GROUP_ID
Definition: rdkafka.h:480
@ RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafka.h:616
@ RD_KAFKA_RESP_ERR__GAPLESS_GUARANTEE
Definition: rdkafka.h:391
@ RD_KAFKA_RESP_ERR__INVALID_TYPE
Definition: rdkafka.h:379
@ RD_KAFKA_RESP_ERR_OFFSET_NOT_AVAILABLE
Definition: rdkafka.h:594
@ RD_KAFKA_RESP_ERR__PURGE_INFLIGHT
Definition: rdkafka.h:385
@ RD_KAFKA_RESP_ERR__STATE
Definition: rdkafka.h:343
@ RD_KAFKA_RESP_ERR_UNKNOWN_PRODUCER_ID
Definition: rdkafka.h:556
@ RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafka.h:588
@ RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafka.h:627
@ RD_KAFKA_RESP_ERR_BROKER_NOT_AVAILABLE
Definition: rdkafka.h:439
@ RD_KAFKA_RESP_ERR_INCONSISTENT_VOTER_SET
Definition: rdkafka.h:630
@ RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafka.h:605
@ RD_KAFKA_RESP_ERR__RETRY
Definition: rdkafka.h:381
@ RD_KAFKA_RESP_ERR_INVALID_REPLICATION_FACTOR
Definition: rdkafka.h:508
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_ID
Definition: rdkafka.h:638
@ RD_KAFKA_RESP_ERR_NOT_COORDINATOR
Definition: rdkafka.h:461
@ RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
Definition: rdkafka.h:333
@ RD_KAFKA_RESP_ERR_ILLEGAL_GENERATION
Definition: rdkafka.h:476
@ RD_KAFKA_RESP_ERR__BAD_MSG
Definition: rdkafka.h:286
@ RD_KAFKA_RESP_ERR__BAD_COMPRESSION
Definition: rdkafka.h:288
@ RD_KAFKA_RESP_ERR__QUEUE_FULL
Definition: rdkafka.h:319
@ RD_KAFKA_RESP_ERR__WAIT_CACHE
Definition: rdkafka.h:359
@ RD_KAFKA_RESP_ERR__RESOLVE
Definition: rdkafka.h:298
@ RD_KAFKA_RESP_ERR_PRODUCER_FENCED
Definition: rdkafka.h:621
@ RD_KAFKA_RESP_ERR__AUTHENTICATION
Definition: rdkafka.h:349
@ RD_KAFKA_RESP_ERR__NOT_IMPLEMENTED
Definition: rdkafka.h:347
@ RD_KAFKA_RESP_ERR_INVALID_RECORD
Definition: rdkafka.h:614
@ RD_KAFKA_RESP_ERR__UNKNOWN_PROTOCOL
Definition: rdkafka.h:345
@ RD_KAFKA_RESP_ERR__PARTIAL
Definition: rdkafka.h:371
@ RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
Definition: rdkafka.h:445
@ RD_KAFKA_RESP_ERR_CLUSTER_AUTHORIZATION_FAILED
Definition: rdkafka.h:494
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafka.h:568
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS
Definition: rdkafka.h:470
@ RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafka.h:600
@ RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
Definition: rdkafka.h:528
@ RD_KAFKA_RESP_ERR__DESTROY
Definition: rdkafka.h:290
@ RD_KAFKA_RESP_ERR__INCONSISTENT
Definition: rdkafka.h:389
@ RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
Definition: rdkafka.h:363
@ RD_KAFKA_RESP_ERR_UNKNOWN_MEMBER_ID
Definition: rdkafka.h:482
@ RD_KAFKA_RESP_ERR_MEMBER_ID_REQUIRED
Definition: rdkafka.h:596
@ RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafka.h:537
@ RD_KAFKA_RESP_ERR_UNRELEASED_INSTANCE_ID
Definition: rdkafka.h:643
@ RD_KAFKA_RESP_ERR__TRANSPORT
Definition: rdkafka.h:294
@ RD_KAFKA_RESP_ERR_INVALID_TIMESTAMP
Definition: rdkafka.h:496
@ RD_KAFKA_RESP_ERR_INVALID_REQUEST
Definition: rdkafka.h:516
@ RD_KAFKA_RESP_ERR_OFFSET_OUT_OF_RANGE
Definition: rdkafka.h:423
@ RD_KAFKA_RESP_ERR__LOG_TRUNCATION
Definition: rdkafka.h:409
@ RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
Definition: rdkafka.h:603
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafka.h:566
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafka.h:498
@ RD_KAFKA_RESP_ERR__CONFLICT
Definition: rdkafka.h:341
@ RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:558
@ RD_KAFKA_RESP_ERR_INVALID_COMMIT_OFFSET_SIZE
Definition: rdkafka.h:488
@ RD_KAFKA_RESP_ERR__NOOP
Definition: rdkafka.h:405
@ RD_KAFKA_RESP_ERR_THROTTLING_QUOTA_EXCEEDED
Definition: rdkafka.h:618
@ RD_KAFKA_RESP_ERR__TIMED_OUT_QUEUE
Definition: rdkafka.h:355
@ RD_KAFKA_RESP_ERR_DUPLICATE_SEQUENCE_NUMBER
Definition: rdkafka.h:524
@ RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
Definition: rdkafka.h:607
@ RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafka.h:490
@ RD_KAFKA_RESP_ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
Definition: rdkafka.h:543
@ RD_KAFKA_RESP_ERR_GROUP_SUBSCRIBED_TO_TOPIC
Definition: rdkafka.h:612
@ RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
Definition: rdkafka.h:403
@ RD_KAFKA_RESP_ERR__MSG_TIMED_OUT
Definition: rdkafka.h:300
@ RD_KAFKA_RESP_ERR_INVALID_SESSION_TIMEOUT
Definition: rdkafka.h:484
@ RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC
Definition: rdkafka.h:311
@ RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafka.h:636
@ RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafka.h:447
@ RD_KAFKA_RESP_ERR_FENCED_MEMBER_EPOCH
Definition: rdkafka.h:640
@ RD_KAFKA_RESP_ERR__NODE_UPDATE
Definition: rdkafka.h:323
@ RD_KAFKA_RESP_ERR_NO_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:609
@ RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
Definition: rdkafka.h:296
@ RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafka.h:580
@ RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
Definition: rdkafka.h:369
@ RD_KAFKA_RESP_ERR__NO_OFFSET
Definition: rdkafka.h:351
@ RD_KAFKA_RESP_ERR_INVALID_MSG
Definition: rdkafka.h:425
@ RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
Definition: rdkafka.h:552
@ RD_KAFKA_RESP_ERR__ALL_BROKERS_DOWN
Definition: rdkafka.h:313
@ RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafka.h:510
@ RD_KAFKA_RESP_ERR__FENCED
Definition: rdkafka.h:399
@ RD_KAFKA_RESP_ERR_INVALID_PARTITIONS
Definition: rdkafka.h:506
@ RD_KAFKA_RESP_ERR__NOT_CONFIGURED
Definition: rdkafka.h:397
@ RD_KAFKA_RESP_ERR__UNSUPPORTED_FEATURE
Definition: rdkafka.h:357
@ RD_KAFKA_RESP_ERR_NO_ERROR
Definition: rdkafka.h:421
@ RD_KAFKA_RESP_ERR__END
Definition: rdkafka.h:415
@ RD_KAFKA_RESP_ERR__INTR
Definition: rdkafka.h:361
@ RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
Definition: rdkafka.h:500
@ RD_KAFKA_RESP_ERR_GROUP_AUTHORIZATION_FAILED
Definition: rdkafka.h:492
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_COMPRESSION_TYPE
Definition: rdkafka.h:590
@ RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafka.h:522
@ RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
Definition: rdkafka.h:367
@ RD_KAFKA_RESP_ERR__SSL
Definition: rdkafka.h:325
@ RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
Definition: rdkafka.h:486
@ RD_KAFKA_RESP_ERR_UNKNOWN
Definition: rdkafka.h:419
@ RD_KAFKA_RESP_ERR_FENCED_LEADER_EPOCH
Definition: rdkafka.h:586
@ RD_KAFKA_RESP_ERR__WAIT_COORD
Definition: rdkafka.h:327
@ RD_KAFKA_RESP_ERR__PARTITION_EOF
Definition: rdkafka.h:305
@ RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
Definition: rdkafka.h:466
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_EXPORT void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf, void(*throttle_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int throttle_time_ms, void *opaque))
Set throttle callback.
RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic)
Destroy and free a NewTopic object previously created with rd_kafka_NewTopic_new()
RD_EXPORT rd_kafka_Uuid_t * rd_kafka_Uuid_new(int64_t most_significant_bits, int64_t least_significant_bits)
Creates a new UUID.
RD_EXPORT void rd_kafka_conf_set_socket_cb(rd_kafka_conf_t *conf, int(*socket_cb)(int domain, int type, int protocol, void *opaque))
Set socket callback.
RD_EXPORT void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs, size_t *cntp)
Returns the full list of error codes.
RD_EXPORT const rd_kafka_error_t * rd_kafka_AclBinding_error(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_store(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets)
Store offsets for next auto-commit for one or more partitions.
RD_EXPORT void rd_kafka_Uuid_destroy(rd_kafka_Uuid_t *uuid)
Destroy the provided uuid.
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupListing_state(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets state for the grplist group.
RD_EXPORT void rd_kafka_DeleteRecords_destroy_array(rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt)
Helper function to destroy all DeleteRecords objects in the del_groups array (of del_group_cnt elemen...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Atomic assignment of partitions to consume.
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:7267
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:7485
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.
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:9238
RD_EXPORT void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf, void(*dr_cb)(rd_kafka_t *rk, void *payload, size_t len, rd_kafka_resp_err_t err, void *opaque, void *msg_opaque))
RD_EXPORT const char * rd_kafka_ConfigResource_error_string(const rd_kafka_ConfigResource_t *config)
RD_EXPORT void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf, void *callback_data)
Set callback_data for OpenSSL engine.
RD_EXPORT void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque)
Sets the application's opaque pointer that will be passed to callbacks.
struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t
Definition: rdkafka.h:8078
RD_EXPORT int rd_kafka_thread_cnt(void)
Retrieve the current number of threads in use by librdkafka.
struct rd_kafka_TopicPartitionInfo_s rd_kafka_TopicPartitionInfo_t
TopicPartition represents a partition in the DescribeTopics result.
Definition: rdkafka.h:8177
RD_EXPORT const rd_kafka_CreateTopics_result_t * rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev)
Get CreateTopics result.
RD_EXPORT int32_t rd_kafka_ScramCredentialInfo_iterations(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the iterations of a given ScramCredentialInfo.
struct rd_kafka_TopicDescription_s rd_kafka_TopicDescription_t
DescribeTopics result type.
Definition: rdkafka.h:8183
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_ListConsumerGroupOffsets_result_groups(const rd_kafka_ListConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a ListConsumerGroupOffsets result.
RD_EXPORT void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata)
Release metadata memory.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_request_sent_t(rd_kafka_t *rk, int sockfd, const char *brokername, int32_t brokerid, int16_t ApiKey, int16_t ApiVersion, int32_t CorrId, size_t size, void *ic_opaque)
on_request_sent() is called when a request has been fully written to a broker TCP connections socket.
Definition: rdkafka.h:6438
RD_EXPORT void rd_kafka_topic_partition_list_destroy(rd_kafka_topic_partition_list_t *rkparlist)
Free all resources used by the list and the list itself.
RD_EXPORT const char * rd_kafka_version_str(void)
Returns the librdkafka version as string.
RD_EXPORT const rd_kafka_error_t ** rd_kafka_ListConsumerGroups_result_errors(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of errors from a ListConsumerGroups call result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_commit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_commit_t *on_commit, void *ic_opaque)
Append an on_commit() interceptor.
rd_kafka_event_t rd_kafka_DeleteRecords_result_t
Definition: rdkafka.h:5776
RD_EXPORT const rd_kafka_DescribeConfigs_result_t * rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev)
Get DescribeConfigs result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_consume(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_consume_t *on_consume, void *ic_opaque)
Append an on_consume() interceptor.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscribe(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *topics)
Subscribe to topic set using balanced consumer groups.
RD_EXPORT const 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.
struct rd_kafka_DeleteAcls_result_response_s rd_kafka_DeleteAcls_result_response_t
Definition: rdkafka.h:9796
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_partition_list_set_offset(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition, int64_t offset)
Set offset to offset for topic and partition.
RD_EXPORT rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assignment(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **partitions)
Returns the current partition assignment as set by rd_kafka_assign() or rd_kafka_incremental_assign()...
RD_EXPORT void rd_kafka_conf_set_rebalance_cb(rd_kafka_conf_t *conf, void(*rebalance_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *partitions, void *opaque))
Consumer: Set rebalance callback for use with coordinated consumer group balancing.
RD_EXPORT void rd_kafka_yield(rd_kafka_t *rk)
Cancels the current callback dispatcher (rd_kafka_poll(), rd_kafka_consume_callback(),...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_destroy(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_destroy_t *on_destroy, void *ic_opaque)
Append an on_destroy() interceptor.
RD_EXPORT const rd_kafka_error_t * rd_kafka_TopicDescription_error(const rd_kafka_TopicDescription_t *topicdesc)
Gets the error for the topicdesc topic.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_find(const rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Find element by topic and partition.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_add(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Add topic+partition to list.
RD_EXPORT void rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf, int(*resolve_cb)(const char *node, const char *service, const struct addrinfo *hints, struct addrinfo **res, void *opaque))
Set address resolution callback.
RD_EXPORT void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf, rd_kafka_topic_conf_t *tconf)
Sets the default topic configuration to use for automatically subscribed topics (e....
RD_EXPORT int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for events served through callbacks for max timeout_ms.
RD_EXPORT const rd_kafka_AclBinding_t ** rd_kafka_DeleteAcls_result_response_matching_acls(const rd_kafka_DeleteAcls_result_response_t *result_response, size_t *matching_acls_cntp)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf)
Creates a copy/duplicate of topic configuration object conf.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_dup_t(rd_kafka_conf_t *new_conf, const rd_kafka_conf_t *old_conf, size_t filter_cnt, const char **filter, void *ic_opaque)
on_conf_dup() is called from rd_kafka_conf_dup() in the order the interceptors were added and is used...
Definition: rdkafka.h:6260
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata(rd_kafka_t *rk)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token(rd_kafka_t *rk, const char *token_value, int64_t md_lifetime_ms, const char *md_principal_name, const char **extensions, size_t extension_size, char *errstr, size_t errstr_size)
Set SASL/OAUTHBEARER token and metadata.
rd_kafka_IsolationLevel_t
IsolationLevel enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:7007
RD_EXPORT size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev, const rd_kafka_message_t **rkmessages, size_t size)
Extacts size message(s) from the event into the pre-allocated array rkmessages.
RD_EXPORT int rd_kafka_ConfigEntry_is_synonym(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_new(void)
Create topic configuration object.
RD_EXPORT void rd_kafka_conf_set_offset_commit_cb(rd_kafka_conf_t *conf, void(*offset_commit_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *opaque))
Consumer: Set offset commit callback for use with consumer groups.
RD_EXPORT const char * rd_kafka_ConsumerGroupDescription_group_id(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the group id for the grpdesc group.
RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk, rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create additional partitions for the given topics, as specified by the new_parts array of size new_pa...
RD_EXPORT const rd_kafka_DescribeConsumerGroups_result_t * rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev)
Get DescribeConsumerGroups result.
RD_EXPORT const 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_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:6295
RD_EXPORT rd_kafka_DeleteTopic_t * rd_kafka_DeleteTopic_new(const char *topic)
Create a new DeleteTopic object. This object is later passed to rd_kafka_DeleteTopics().
RD_EXPORT const char * rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres)
RD_EXPORT const rd_kafka_Node_t ** rd_kafka_TopicPartitionInfo_isr(const rd_kafka_TopicPartitionInfo_t *partition, size_t *cntp)
Gets the partition in-sync replicas for partition.
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_DescribeConfigs_result_resources(const rd_kafka_DescribeConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeConfigs result.
RD_EXPORT int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_AclBindingFilter_t * rd_kafka_AclBindingFilter_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBindingFilter object. This object is later passed to rd_kafka_DescribeAcls() or rd_ka...
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_DeleteGroups_result_groups(const rd_kafka_DeleteGroups_result_t *result, size_t *cntp)
Get an array of group results from a DeleteGroups result.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy_array(rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffset_cnt)
Helper function to destroy all AlterConsumerGroupOffsets objects in the alter_grpoffsets array (of al...
rd_kafka_event_t rd_kafka_CreatePartitions_result_t
Definition: rdkafka.h:5768
RD_EXPORT void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf, void *rkt_opaque)
Sets the application's opaque pointer that will be passed to all topic callbacks as the rkt_opaque ar...
RD_EXPORT 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...
struct rd_kafka_AlterUserScramCredentials_result_response_s rd_kafka_AlterUserScramCredentials_result_response_t
Result of a single user SCRAM alteration.
Definition: rdkafka.h:9478
RD_EXPORT char * rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms)
Returns the ClusterId as reported in broker metadata.
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_EXPORT rd_kafka_error_t * rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt)
Produce and send a single message to broker.
RD_EXPORT const 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_EXPORT void rd_kafka_dump(FILE *fp, rd_kafka_t *rk)
Dumps rdkafka's internal state for handle rk to stream fp.
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_unassign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally remove partitions from the current assignment.
RD_EXPORT rd_kafka_error_t * rd_kafka_send_offsets_to_transaction(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, const rd_kafka_consumer_group_metadata_t *cgmetadata, int timeout_ms)
Sends a list of topic partition offsets to the consumer group coordinator for cgmetadata,...
RD_EXPORT const char * rd_kafka_ConsumerGroupDescription_partition_assignor(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the partition assignor for the grpdesc group.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_pause_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Pause producing or consumption for the provided list of partitions.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_config(rd_kafka_NewTopic_t *new_topic, const char *name, const char *value)
Set (broker-side) topic configuration name/value pair.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_broker_state_change(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change, void *ic_opaque)
Append an on_broker_state_change() interceptor.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_new(rd_kafka_t *rk)
Create a new message queue.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_background(rd_kafka_t *rk)
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf, size_t filter_cnt, const char **filter)
Same as rd_kafka_conf_dup() but with an array of property name prefixes to filter out (ignore) when c...
rd_kafka_event_t rd_kafka_DescribeConsumerGroups_result_t
Definition: rdkafka.h:5780
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreatePartitions_result_topics(const rd_kafka_CreatePartitions_result_t *result, size_t *cntp)
Get an array of topic results from a CreatePartitions result.
RD_EXPORT const 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_EXPORT rd_kafka_error_t * rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms)
Aborts the ongoing transaction.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk, rd_kafka_resp_err_t err, const char *reason)
Trigger a fatal error for testing purposes.
RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding)
Destroy and free an AclBinding object previously created with rd_kafka_AclBinding_new()
RD_EXPORT rd_kafka_event_t * rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for an event for max timeout_ms.
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:9340
RD_EXPORT const 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_EXPORT void rd_kafka_error_destroy(rd_kafka_error_t *error)
Free and destroy an error object.
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new(const char *group_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
rd_kafka_event_t rd_kafka_DeleteTopics_result_t
Definition: rdkafka.h:5760
RD_EXPORT rd_kafka_AclBinding_t * rd_kafka_AclBinding_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBinding object. This object is later passed to rd_kafka_CreateAcls().
RD_EXPORT void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage)
Frees resources for rkmessage and hands ownership back to rdkafka.
RD_EXPORT 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_EXPORT const rd_kafka_DeleteConsumerGroupOffsets_result_t * rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get DeleteConsumerGroupOffsets result.
rd_kafka_event_t rd_kafka_ListConsumerGroupOffsets_result_t
Definition: rdkafka.h:5788
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_dup(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_dup_t *on_conf_dup, void *ic_opaque)
Append an on_conf_dup() interceptor.
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_require_stable_offsets(rd_kafka_AdminOptions_t *options, int true_or_false)
Whether broker should return stable offsets (transaction-committed).
RD_EXPORT rd_kafka_resp_err_t rd_kafka_consumer_close(rd_kafka_t *rk)
Close the consumer.
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk)
Enable SASL OAUTHBEARER refresh callbacks on the librdkafka background thread.
RD_EXPORT const rd_kafka_ListOffsets_result_t * rd_kafka_event_ListOffsets_result(rd_kafka_event_t *rkev)
Get ListOffsets result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_committed(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Retrieve committed offsets for topics+partitions.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_send_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_send() is called from rd_kafka_produce*() (et.al) prior to the partitioner being called.
Definition: rdkafka.h:6335
RD_EXPORT const rd_kafka_ConsumerGroupListing_t ** rd_kafka_ListConsumerGroups_result_valid(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of valid list groups from a ListConsumerGroups result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs, size_t idx, const char **namep, const void **valuep, size_t *sizep)
Iterator for all headers.
RD_EXPORT const rd_kafka_message_t * rd_kafka_event_message_next(rd_kafka_event_t *rkev)
struct rd_kafka_TopicCollection_s rd_kafka_TopicCollection_t
Represents a collection of topics, to be passed to DescribeTopics.
Definition: rdkafka.h:8171
RD_EXPORT const char * rd_kafka_error_name(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally add partitions to the current assignment.
RD_EXPORT const rd_kafka_DescribeTopics_result_t * rd_kafka_event_DescribeTopics_result(rd_kafka_event_t *rkev)
Get DescribeTopics result.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Alter committed offsets for a set of partitions in a consumer group. This will succeed at the partiti...
RD_EXPORT void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk, const char **groups, size_t groups_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe groups from cluster as specified by the groups array of size groups_cnt elements.
RD_EXPORT rd_kafka_error_t * rd_kafka_begin_transaction(rd_kafka_t *rk)
Begin a new transaction.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_unsubscribe(rd_kafka_t *rk)
Unsubscribe from the current subscription set.
RD_EXPORT void rd_kafka_log_print(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin (default) log sink: print to stderr.
RD_EXPORT ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume batch of messages from queue.
rd_kafka_OffsetSpec_t
Allows to specify the desired offsets when using ListOffsets.
Definition: rdkafka.h:9220
RD_EXPORT const char * rd_kafka_topic_name(const rd_kafka_topic_t *rkt)
Returns the topic name.
RD_EXPORT const char * rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres)
RD_EXPORT const char * rd_kafka_Node_rack(const rd_kafka_Node_t *node)
Get the rack of node.
RD_EXPORT const char ** rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf, size_t *cntp)
Dump the topic configuration properties and values of conf to an array with "key",...
RD_EXPORT void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu, void(*event_cb)(rd_kafka_t *rk, void *qev_opaque), void *qev_opaque)
Enable callback event triggering for queue.
RD_EXPORT int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage, rd_kafka_timestamp_type_t *tstype)
Returns the message timestamp for a consumed message.
RD_EXPORT int64_t rd_kafka_Uuid_most_significant_bits(const rd_kafka_Uuid_t *uuid)
Gets most significant 64 bits for the given UUID.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_request_sent(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_request_sent_t *on_request_sent, void *ic_opaque)
Append an on_request_sent() interceptor.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size)
Returns the first fatal error set on this client instance, or RD_KAFKA_RESP_ERR_NO_ERROR if no fatal ...
rd_kafka_event_t rd_kafka_CreateAcls_result_t
Definition: rdkafka.h:5762
RD_EXPORT rd_kafka_resp_err_t rd_kafka_query_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high, int timeout_ms)
Query broker for low (oldest/beginning) and high (newest/end) offsets for partition.
rd_kafka_event_t rd_kafka_CreateTopics_result_t
Definition: rdkafka.h:5758
RD_EXPORT int rd_kafka_consume_callback_queue(rd_kafka_queue_t *rkqu, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consume multiple messages from queue with callback.
RD_EXPORT void rd_kafka_conf_set_open_cb(rd_kafka_conf_t *conf, int(*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque))
Set open callback.
rd_kafka_event_t rd_kafka_DeleteAcls_result_t
Definition: rdkafka.h:5766
RD_EXPORT const char * rd_kafka_ResourcePatternType_name(rd_kafka_ResourcePatternType_t resource_pattern_type)
RD_EXPORT const char * rd_kafka_TopicDescription_name(const rd_kafka_TopicDescription_t *topicdesc)
Gets the topic name for the topicdesc topic.
RD_EXPORT const char * rd_kafka_Node_host(const rd_kafka_Node_t *node)
Get the host of node.
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreateTopics_result_topics(const rd_kafka_CreateTopics_result_t *result, size_t *cntp)
Get an array of topic results from a CreateTopics result.
rd_kafka_event_t rd_kafka_AlterUserScramCredentials_result_t
Definition: rdkafka.h:5796
RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt)
Helper function to destroy all NewTopic objects in the new_topics array (of new_topic_cnt elements)....
rd_kafka_event_t rd_kafka_DescribeAcls_result_t
Definition: rdkafka.h:5764
RD_EXPORT const rd_kafka_DeleteTopics_result_t * rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev)
Get DeleteTopics result.
RD_EXPORT void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr)
Free pointer returned by librdkafka.
RD_EXPORT const char * rd_kafka_Uuid_base64str(const rd_kafka_Uuid_t *uuid)
Computes base64 encoding for the given uuid string.
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:6552
RD_EXPORT const rd_kafka_Node_t ** rd_kafka_TopicPartitionInfo_replicas(const rd_kafka_TopicPartitionInfo_t *partition, size_t *cntp)
Gets the partition replicas for partition.
RD_EXPORT const rd_kafka_error_t * rd_kafka_DeleteAcls_result_response_error(const rd_kafka_DeleteAcls_result_response_t *result_response)
rd_kafka_AclOperation_t
Apache Kafka ACL operation types. Common type for multiple Admin API functions.
Definition: rdkafka.h:7234
@ RD_KAFKA_ACL_OPERATION_ALL
Definition: rdkafka.h:7238
@ RD_KAFKA_ACL_OPERATION_ALTER
Definition: rdkafka.h:7243
@ RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
Definition: rdkafka.h:7245
@ RD_KAFKA_ACL_OPERATION_IDEMPOTENT_WRITE
Definition: rdkafka.h:7251
@ RD_KAFKA_ACL_OPERATION_DELETE
Definition: rdkafka.h:7242
@ RD_KAFKA_ACL_OPERATION_WRITE
Definition: rdkafka.h:7240
@ RD_KAFKA_ACL_OPERATION_ANY
Definition: rdkafka.h:7236
@ RD_KAFKA_ACL_OPERATION_UNKNOWN
Definition: rdkafka.h:7235
@ RD_KAFKA_ACL_OPERATION_CREATE
Definition: rdkafka.h:7241
@ RD_KAFKA_ACL_OPERATION_ALTER_CONFIGS
Definition: rdkafka.h:7249
@ RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
Definition: rdkafka.h:7247
@ RD_KAFKA_ACL_OPERATION_READ
Definition: rdkafka.h:7239
@ RD_KAFKA_ACL_OPERATION_DESCRIBE
Definition: rdkafka.h:7244
RD_EXPORT ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume up to rkmessages_size from topic rkt and partition putting a pointer to each message in the a...
RD_EXPORT int rd_kafka_event_log(rd_kafka_event_t *rkev, const char **fac, const char **str, int *level)
Extract log message from the event.
RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk)
Check whether the consumer considers the current assignment to have been lost involuntarily....
RD_EXPORT void rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf, int(*connect_cb)(int sockfd, const struct sockaddr *addr, int addrlen, const char *id, void *opaque))
Set connect callback.
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy_array(rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt)
Destroys an array of UserScramCredentialAlteration.
RD_EXPORT const rd_kafka_Uuid_t * rd_kafka_TopicDescription_topic_id(const rd_kafka_TopicDescription_t *topicdesc)
Gets the topic id for the topicdesc topic.
RD_EXPORT const 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_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf, rd_kafka_cert_type_t cert_type, rd_kafka_cert_enc_t cert_enc, const void *buffer, size_t size, char *errstr, size_t errstr_size)
Set certificate/key cert_type from the cert_enc encoded memory at buffer of size bytes.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve topic configuration value for property name.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_sasl(rd_kafka_t *rk)
RD_EXPORT int rd_kafka_consume_callback(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consumes messages from topic rkt and partition, calling the provided callback for each consumed messs...
RD_EXPORT const char * rd_kafka_rebalance_protocol(rd_kafka_t *rk)
The rebalance protocol currently in use. This will be "NONE" if the consumer has not (yet) joined a g...
RD_EXPORT int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Random partitioner.
RD_EXPORT const char * rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage)
Returns the error string for an errored rd_kafka_message_t or NULL if there was no error.
RD_EXPORT void rd_kafka_conf_set_stats_cb(rd_kafka_conf_t *conf, int(*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque))
Set statistics callback in provided conf object.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags)
Purge messages currently handled by the producer instance.
rd_kafka_event_t rd_kafka_DescribeCluster_result_t
Definition: rdkafka.h:5792
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token_failure(rd_kafka_t *rk, const char *errstr)
SASL/OAUTHBEARER token refresh failure indicator.
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres)
RD_EXPORT int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev, char *dst, size_t dstsize)
Extract log debug context from event.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_set_log_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Forward librdkafka logs (and debug) to the specified queue for serving with one of the ....
RD_EXPORT void rd_kafka_conf_set_background_event_cb(rd_kafka_conf_t *conf, void(*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque))
Generic event callback to be used with the event API to trigger callbacks for rd_kafka_event_t object...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get(const rd_kafka_headers_t *hdrs, size_t idx, const char *name, const void **valuep, size_t *sizep)
Iterator for headers matching name.
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_consumer_group_state_code(const char *name)
Returns a code for a state name.
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_MemberAssignment_partitions(const rd_kafka_MemberAssignment_t *assignment)
Gets assigned partitions of a member assignment.
RD_EXPORT void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst)
Forward/re-route queue src to dst. If dst is NULL the forwarding is removed.
RD_EXPORT const rd_kafka_ConfigEntry_t ** rd_kafka_ConfigResource_configs(const rd_kafka_ConfigResource_t *config, size_t *cntp)
Get an array of config entries from a ConfigResource object.
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy_array(rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffset_cnt)
Helper function to destroy all DeleteConsumerGroupOffsets objects in the del_grpoffsets array (of del...
rd_kafka_cert_type_t
SSL certificate type.
Definition: rdkafka.h:2483
@ RD_KAFKA_CERT_PUBLIC_KEY
Definition: rdkafka.h:2484
@ RD_KAFKA_CERT_PRIVATE_KEY
Definition: rdkafka.h:2485
@ RD_KAFKA_CERT_CA
Definition: rdkafka.h:2486
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_event_t rd_kafka_ListOffsets_result_t
Definition: rdkafka.h:5798
RD_EXPORT const char * rd_kafka_event_name(const rd_kafka_event_t *rkev)
rd_kafka_event_t rd_kafka_DescribeTopics_result_t
Definition: rdkafka.h:5790
RD_EXPORT void * rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size)
Allocate memory using the same allocator librdkafka uses.
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy(rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets)
Destroy and free a DeleteConsumerGroupOffsets object previously created with rd_kafka_DeleteConsumerG...
RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t **del_acls, size_t del_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete acls matching the filteres provided in del_acls array of size del_acls_cnt.
RD_EXPORT size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf)
Gets the default topic configuration as previously set with rd_kafka_conf_set_default_topic_conf() or...
RD_EXPORT void rd_kafka_topic_partition_list_add_range(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t start, int32_t stop)
Add range of partitions from start to stop inclusive.
RD_EXPORT rd_kafka_error_t * rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms)
Commit the current transaction (as started with rd_kafka_begin_transaction()).
RD_EXPORT const rd_kafka_Node_t * rd_kafka_ConsumerGroupDescription_coordinator(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the coordinator for the grpdesc group.
RD_EXPORT rd_kafka_t * rd_kafka_new(rd_kafka_type_t type, rd_kafka_conf_t *conf, char *errstr, size_t errstr_size)
Creates a new Kafka handle and starts its operation according to the specified type (RD_KAFKA_CONSUME...
struct rd_kafka_ListConsumerGroupsResult_s rd_kafka_ListConsumerGroupsResult_t
Definition: rdkafka.h:8480
RD_EXPORT const char * rd_kafka_ConsumerGroupListing_group_id(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets the group id for the grplist group.
RD_EXPORT rd_kafka_message_t * rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms)
Poll the consumer for messages or events.
RD_EXPORT void rd_kafka_conf_set_consume_cb(rd_kafka_conf_t *conf, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque))
Consumer: Set consume callback for use with rd_kafka_consumer_poll()
RD_EXPORT rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, int timeout_ms)
Seek consumer for topic+partition to offset which is either an absolute or logical offset.
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_copy(const rd_kafka_topic_partition_list_t *src)
Make a copy of an existing list.
RD_EXPORT const rd_kafka_Node_t * rd_kafka_TopicPartitionInfo_leader(const rd_kafka_TopicPartitionInfo_t *partition)
Gets the partition leader for partition.
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new_with_genid(const char *group_id, int32_t generation_id, const char *member_id, const char *group_instance_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
RD_EXPORT void rd_kafka_DeleteRecords_destroy(rd_kafka_DeleteRecords_t *del_records)
Destroy and free a DeleteRecords object previously created with rd_kafka_DeleteRecords_new()
RD_EXPORT rd_kafka_AdminOptions_t * rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api)
Create a new AdminOptions object.
RD_EXPORT const rd_kafka_DescribeUserScramCredentials_result_t * rd_kafka_event_DescribeUserScramCredentials_result(rd_kafka_event_t *rkev)
Get DescribeUserScramCredentials result.
RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk, rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create topics in cluster as specified by the new_topics array of size new_topic_cnt elements.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_error(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewPartitions_set_replica_assignment(rd_kafka_NewPartitions_t *new_parts, int32_t new_partition_idx, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker id) assignment for new_partition_idx to the replica set in broker_ids (of bro...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error)
rd_kafka_conf_res_t() rd_kafka_interceptor_f_on_conf_set_t(rd_kafka_conf_t *conf, const char *name, const char *val, char *errstr, size_t errstr_size, void *ic_opaque)
on_conf_set() is called from rd_kafka_*_conf_set() in the order the interceptors were added.
Definition: rdkafka.h:6230
RD_EXPORT rd_kafka_DeleteGroup_t * rd_kafka_DeleteGroup_new(const char *group)
Create a new DeleteGroup object. This object is later passed to rd_kafka_DeleteGroups().
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs, const char *name, const void **valuep, size_t *sizep)
Find last header in list hdrs matching name.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_list_groups(rd_kafka_t *rk, const char *group, const struct rd_kafka_group_list **grplistp, int timeout_ms)
List and describe client groups in cluster.
rd_kafka_resp_err_t() rd_kafka_plugin_f_conf_init_t(rd_kafka_conf_t *conf, void **plug_opaquep, char *errstr, size_t errstr_size)
Plugin's configuration initializer method called each time the library is referenced from configurati...
Definition: rdkafka.h:6144
RD_EXPORT rd_kafka_resp_err_t rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Retrieve current positions (offsets) for topics+partitions.
RD_EXPORT const char ** rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp)
Dump the configuration properties and values of conf to an array with "key", "value" pairs.
RD_EXPORT const char * rd_kafka_DescribeCluster_result_cluster_id(const rd_kafka_DescribeCluster_result_t *result)
Gets the cluster id for the result cluster.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_operation_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the broker's operation timeout, such as the timeout for CreateTopics to complete the creation of...
RD_EXPORT const rd_kafka_ConsumerGroupDescription_t ** rd_kafka_DescribeConsumerGroups_result_groups(const rd_kafka_DescribeConsumerGroups_result_t *result, size_t *cntp)
Get an array of group results from a DescribeConsumerGroups result.
RD_EXPORT const rd_kafka_conf_t * rd_kafka_conf(rd_kafka_t *rk)
RD_EXPORT void rd_kafka_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_EXPORT void rd_kafka_DeleteGroup_destroy(rd_kafka_DeleteGroup_t *del_group)
Destroy and free a DeleteGroup object previously created with rd_kafka_DeleteGroup_new()
RD_EXPORT void * rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size)
Allocate and zero memory using the same allocator librdkafka uses.
RD_EXPORT const char * rd_kafka_ConfigEntry_value(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT const char * rd_kafka_MemberDescription_host(const rd_kafka_MemberDescription_t *member)
Gets host of member.
RD_EXPORT void rd_kafka_log_syslog(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin log sink: print to syslog.
struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t
Definition: rdkafka.h:7408
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:9318
RD_EXPORT void rd_kafka_topic_conf_set_msg_order_cmp(rd_kafka_topic_conf_t *topic_conf, int(*msg_order_cmp)(const rd_kafka_message_t *a, const rd_kafka_message_t *b))
Producer: Set message queueing order comparator callback.
RD_EXPORT rd_kafka_error_t * rd_kafka_seek_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Seek consumer for partitions in partitions to the per-partition offset in the .offset field of partit...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Store offset offset + 1 for topic rkt partition partition.
rd_kafka_AlterConfigOpType_t
Incremental alter configs operations.
Definition: rdkafka.h:7752
RD_EXPORT void rd_kafka_DeleteGroup_destroy_array(rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt)
Helper function to destroy all DeleteGroup objects in the del_groups array (of del_group_cnt elements...
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_AlterConfigs_result_resources(const rd_kafka_AlterConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a AlterConfigs result.
RD_EXPORT int rd_kafka_produce_batch(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, rd_kafka_message_t *rkmessages, int message_cnt)
Produce multiple messages.
rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t
Definition: rdkafka.h:5778
RD_EXPORT const rd_kafka_MemberDescription_t * rd_kafka_ConsumerGroupDescription_member(const rd_kafka_ConsumerGroupDescription_t *grpdesc, size_t idx)
Gets a member of grpdesc group.
RD_EXPORT rd_kafka_ResourcePatternType_t rd_kafka_AclBinding_resource_pattern_type(const rd_kafka_AclBinding_t *acl)
RD_EXPORT int rd_kafka_ConfigEntry_is_sensitive(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy(rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets)
Destroy and free a ListConsumerGroupOffsets object previously created with rd_kafka_ListConsumerGroup...
RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t *acl_filter, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe acls matching the filter provided in acl_filter.
RD_EXPORT const char * rd_kafka_event_config_string(rd_kafka_event_t *rkev)
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_AlterConsumerGroupOffsets_result_groups(const rd_kafka_AlterConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a AlterConsumerGroupOffsets result.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_main(rd_kafka_t *rk)
RD_EXPORT const char * rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres)
rd_kafka_admin_op_t
Admin operation enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:6954
@ RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6977
@ RD_KAFKA_ADMIN_OP_DELETERECORDS
Definition: rdkafka.h:6961
@ RD_KAFKA_ADMIN_OP_CREATETOPICS
Definition: rdkafka.h:6956
@ RD_KAFKA_ADMIN_OP_CREATEACLS
Definition: rdkafka.h:6965
@ RD_KAFKA_ADMIN_OP_INCREMENTALALTERCONFIGS
Definition: rdkafka.h:6975
@ RD_KAFKA_ADMIN_OP_DESCRIBEACLS
Definition: rdkafka.h:6966
@ RD_KAFKA_ADMIN_OP_DESCRIBECLUSTER
Definition: rdkafka.h:6981
@ RD_KAFKA_ADMIN_OP_DESCRIBECONSUMERGROUPS
Definition: rdkafka.h:6969
@ RD_KAFKA_ADMIN_OP_DELETEGROUPS
Definition: rdkafka.h:6962
@ RD_KAFKA_ADMIN_OP_ANY
Definition: rdkafka.h:6955
@ RD_KAFKA_ADMIN_OP_DELETEACLS
Definition: rdkafka.h:6967
@ RD_KAFKA_ADMIN_OP_DELETECONSUMERGROUPOFFSETS
Definition: rdkafka.h:6964
@ RD_KAFKA_ADMIN_OP_DELETETOPICS
Definition: rdkafka.h:6957
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6971
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPS
Definition: rdkafka.h:6968
@ RD_KAFKA_ADMIN_OP_LISTOFFSETS
Definition: rdkafka.h:6982
@ RD_KAFKA_ADMIN_OP__CNT
Definition: rdkafka.h:6983
@ RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6973
@ RD_KAFKA_ADMIN_OP_ALTERCONFIGS
Definition: rdkafka.h:6959
@ RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
Definition: rdkafka.h:6958
@ RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6979
@ RD_KAFKA_ADMIN_OP_DESCRIBETOPICS
Definition: rdkafka.h:6980
@ RD_KAFKA_ADMIN_OP_DESCRIBECONFIGS
Definition: rdkafka.h:6960
struct rd_kafka_AlterConsumerGroupOffsets_s rd_kafka_AlterConsumerGroupOffsets_t
Definition: rdkafka.h:9031
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_response_received(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_response_received_t *on_response_received, void *ic_opaque)
Append an on_response_received() interceptor.
RD_EXPORT const rd_kafka_error_t * rd_kafka_ConsumerGroupDescription_error(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the error for the grpdesc group.
RD_EXPORT void rd_kafka_NewPartitions_destroy_array(rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt)
Helper function to destroy all NewPartitions objects in the new_parts array (of new_parts_cnt element...
RD_EXPORT int rd_kafka_version(void)
Returns the librdkafka version as integer.
RD_EXPORT int rd_kafka_topic_partition_list_del_by_idx(rd_kafka_topic_partition_list_t *rktparlist, int idx)
Delete partition from list by elems[] index.
RD_EXPORT size_t rd_kafka_UserScramCredentialsDescription_scramcredentialinfo_count(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the count of ScramCredentialInfos of a UserScramCredentialsDescription.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_metadata(rd_kafka_t *rk, int all_topics, rd_kafka_topic_t *only_rkt, const struct rd_kafka_metadata **metadatap, int timeout_ms)
Request Metadata from broker.
RD_EXPORT char * rd_kafka_memberid(const rd_kafka_t *rk)
Returns this client's broker-assigned group member id.
RD_EXPORT rd_kafka_error_t * rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage)
Store offset +1 for the consumed message.
RD_EXPORT void * rd_kafka_event_opaque(rd_kafka_event_t *rkev)
struct rd_kafka_UserScramCredentialAlteration_s rd_kafka_UserScramCredentialAlteration_t
A request to alter a user's SASL/SCRAM credentials.
Definition: rdkafka.h:9412
RD_EXPORT const char * rd_kafka_MemberDescription_consumer_id(const rd_kafka_MemberDescription_t *member)
Gets consumer id of member.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_exit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_exit_t *on_thread_exit, void *ic_opaque)
Append an on_thread_exit() interceptor.
RD_EXPORT int rd_kafka_consumer_closed(rd_kafka_t *rk)
RD_EXPORT const 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_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:6410
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_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs, const char *name)
Remove all headers for the given key (if any).
RD_EXPORT const rd_kafka_ConfigEntry_t ** rd_kafka_ConfigEntry_synonyms(const rd_kafka_ConfigEntry_t *entry, size_t *cntp)
RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void)
Returns the thread-local system errno.
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_default_topic_conf_dup(rd_kafka_t *rk)
Creates a copy/duplicate of rk 's default topic configuration object.
RD_EXPORT rd_kafka_DeleteConsumerGroupOffsets_t * rd_kafka_DeleteConsumerGroupOffsets_new(const char *group, const rd_kafka_topic_partition_list_t *partitions)
Create a new DeleteConsumerGroupOffsets object. This object is later passed to rd_kafka_DeleteConsume...
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_set_credentials(rd_kafka_t *rk, const char *username, const char *password)
Sets SASL credentials used for SASL PLAIN and SCRAM mechanisms by this Kafka client.
RD_EXPORT const char * rd_kafka_AclBinding_host(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup(const rd_kafka_conf_t *conf)
Creates a copy/duplicate of configuration object conf.
RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic)
Destroy and free a DeleteTopic object previously created with rd_kafka_DeleteTopic_new()
RD_EXPORT const char * rd_kafka_err2name(rd_kafka_resp_err_t err)
Returns the error code name (enum name).
rd_kafka_consumer_group_state_t
Consumer group state.
Definition: rdkafka.h:5151
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:6475
RD_EXPORT const rd_kafka_IncrementalAlterConfigs_result_t * rd_kafka_event_IncrementalAlterConfigs_result(rd_kafka_event_t *rkev)
Get IncrementalAlterConfigs result.
RD_EXPORT int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, rd_kafka_queue_t *rkqu)
Same as rd_kafka_consume_start() but re-routes incoming messages to the provided queue rkqu (which mu...
int rd_kafka_event_type_t
Event types.
Definition: rdkafka.h:5472
RD_EXPORT size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu)
struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t
ListConsumerGroups result for a single group.
Definition: rdkafka.h:8477
RD_EXPORT void rd_kafka_conf_dump_free(const char **arr, size_t cnt)
Frees a configuration dump returned from rd_kafka_conf_dump() or `rd_kafka_topic_conf_dump().
RD_EXPORT rd_kafka_message_t * rd_kafka_consume_queue(rd_kafka_queue_t *rkqu, int timeout_ms)
Consume from queue.
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_match_consumer_group_states(rd_kafka_AdminOptions_t *options, const rd_kafka_consumer_group_state_t *consumer_group_states, size_t consumer_group_states_cnt)
Set consumer groups states to query for.
struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t
Definition: rdkafka.h:7648
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_destroy(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_destroy_t *on_conf_destroy, void *ic_opaque)
Append an on_conf_destroy() interceptor.
RD_EXPORT void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt)
Loose application's topic handle refcount as previously created with rd_kafka_topic_new().
RD_EXPORT void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu)
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy_array(rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffset_cnt)
Helper function to destroy all ListConsumerGroupOffsets objects in the list_grpoffsets array (of list...
struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t
AdminOptions provides a generic mechanism for setting optional parameters for the Admin API requests.
Definition: rdkafka.h:6998
RD_EXPORT int rd_kafka_topic_partition_list_del(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Delete partition from list.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres)
Topic result provides per-topic operation result information.
RD_EXPORT int rd_kafka_error_is_fatal(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_set(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_set_t *on_conf_set, void *ic_opaque)
Append an on_conf_set() interceptor.
RD_EXPORT const rd_kafka_error_t * rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres)
Group result provides per-group operation result information.
RD_EXPORT const rd_kafka_AlterConfigs_result_t * rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev)
Get AlterConfigs result.
RD_EXPORT void rd_kafka_conf_set_oauthbearer_token_refresh_cb(rd_kafka_conf_t *conf, void(*oauthbearer_token_refresh_cb)(rd_kafka_t *rk, const char *oauthbearer_config, void *opaque))
Set SASL/OAUTHBEARER token refresh callback in provided conf object.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy(rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets)
Destroy and free a AlterConsumerGroupOffsets object previously created with rd_kafka_AlterConsumerGro...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_request_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the overall request timeout, including broker lookup, request transmission, operation time on br...
RD_EXPORT void rd_kafka_message_set_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t *hdrs)
Replace the message's current headers with a new list.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_message(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, int async)
Commit message's offset on broker for the message's partition. The committed offset is the message's ...
rd_kafka_vtype_t
Var-arg tag types.
Definition: rdkafka.h:1137
@ RD_KAFKA_VTYPE_PARTITION
Definition: rdkafka.h:1141
@ RD_KAFKA_VTYPE_HEADERS
Definition: rdkafka.h:1153
@ RD_KAFKA_VTYPE_TOPIC
Definition: rdkafka.h:1139
@ RD_KAFKA_VTYPE_MSGFLAGS
Definition: rdkafka.h:1149
@ RD_KAFKA_VTYPE_RKT
Definition: rdkafka.h:1140
@ RD_KAFKA_VTYPE_HEADER
Definition: rdkafka.h:1151
@ RD_KAFKA_VTYPE_KEY
Definition: rdkafka.h:1143
@ RD_KAFKA_VTYPE_END
Definition: rdkafka.h:1138
@ RD_KAFKA_VTYPE_OPAQUE
Definition: rdkafka.h:1144
@ RD_KAFKA_VTYPE_VALUE
Definition: rdkafka.h:1142
@ RD_KAFKA_VTYPE_TIMESTAMP
Definition: rdkafka.h:1150
struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t
Definition: rdkafka.h:7773
RD_EXPORT const char * rd_kafka_name(const rd_kafka_t *rk)
Returns Kafka handle name.
RD_EXPORT const rd_kafka_DeleteAcls_result_t * rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev)
struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t
Definition: rdkafka.h:8856
RD_EXPORT rd_kafka_resp_err_t rd_kafka_poll_set_consumer(rd_kafka_t *rk)
Redirect the main (rd_kafka_poll()) queue to the KafkaConsumer's queue (rd_kafka_consumer_poll()).
RD_EXPORT const rd_kafka_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_EXPORT const rd_kafka_MemberAssignment_t * rd_kafka_MemberDescription_assignment(const rd_kafka_MemberDescription_t *member)
Gets assignment of member.
RD_EXPORT void rd_kafka_AdminOptions_set_opaque(rd_kafka_AdminOptions_t *options, void *ev_opaque)
Set application opaque value that can be extracted from the result event using rd_kafka_event_opaque(...
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_close_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Asynchronously close the consumer.
RD_EXPORT const char * rd_kafka_AclPermissionType_name(rd_kafka_AclPermissionType_t acl_permission_type)
RD_EXPORT RD_DEPRECATED void rd_kafka_set_logger(rd_kafka_t *rk, void(*func)(const rd_kafka_t *rk, int level, const char *fac, const char *buf))
Set logger function.
RD_EXPORT rd_kafka_NewTopic_t * rd_kafka_NewTopic_new(const char *topic, int num_partitions, int replication_factor, char *errstr, size_t errstr_size)
Create a new NewTopic object. This object is later passed to rd_kafka_CreateTopics().
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_new(size_t initial_count)
Create a new headers list.
RD_EXPORT rd_kafka_AlterConsumerGroupOffsets_t * rd_kafka_AlterConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new AlterConsumerGroupOffsets object. This object is later passed to rd_kafka_AlterConsumerG...
RD_EXPORT const 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_EXPORT int rd_kafka_ConsumerGroupDescription_is_simple_consumer_group(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Is the grpdesc group a simple consumer group.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev)
RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs)
Destroy the headers list. The object and any returned value pointers are not usable after this call.
RD_EXPORT int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage)
Returns the broker id of the broker the message was produced to or fetched from.
rd_kafka_event_t rd_kafka_AlterConsumerGroupOffsets_result_t
Definition: rdkafka.h:5786
RD_EXPORT void rd_kafka_ConfigResource_destroy_array(rd_kafka_ConfigResource_t **config, size_t config_cnt)
Helper function to destroy all ConfigResource objects in the configs array (of config_cnt elements)....
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_start(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_start_t *on_thread_start, void *ic_opaque)
Append an on_thread_start() interceptor.
RD_EXPORT rd_kafka_message_t * rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms)
Consume a single message from topic rkt and partition.
RD_EXPORT size_t rd_kafka_ConsumerGroupDescription_member_count(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the members count of grpdesc group.
RD_EXPORT void rd_kafka_conf_properties_show(FILE *fp)
Prints a table to fp of all supported configuration properties, their default values as well as a des...
RD_EXPORT const rd_kafka_AclBinding_t ** rd_kafka_DescribeAcls_result_acls(const rd_kafka_DescribeAcls_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeAcls result.
RD_EXPORT const rd_kafka_CreateAcls_result_t * rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_acknowledgement_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_acknowledgement() is called to inform interceptors that a message was succesfully delivered or per...
Definition: rdkafka.h:6362
struct rd_kafka_DeleteConsumerGroupOffsets_s rd_kafka_DeleteConsumerGroupOffsets_t
Definition: rdkafka.h:9127
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_new(void)
Create configuration object.
RD_EXPORT int rd_kafka_ConfigEntry_is_read_only(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_thread_start_t(rd_kafka_t *rk, rd_kafka_thread_type_t thread_type, const char *thread_name, void *ic_opaque)
on_thread_start() is called from a newly created librdkafka-managed thread.
Definition: rdkafka.h:6505
RD_EXPORT int rd_kafka_wait_destroyed(int timeout_ms)
Wait for all rd_kafka_t objects to be destroyed.
RD_EXPORT const char * rd_kafka_consumer_group_state_name(rd_kafka_consumer_group_state_t state)
Returns a name for a state code.
rd_kafka_ConfigSource_t
Apache Kafka config sources.
Definition: rdkafka.h:7617
@ RD_KAFKA_CONFIG_SOURCE_UNKNOWN_CONFIG
Definition: rdkafka.h:7620
@ RD_KAFKA_CONFIG_SOURCE__CNT
Definition: rdkafka.h:7636
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
Definition: rdkafka.h:7624
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_TOPIC_CONFIG
Definition: rdkafka.h:7622
@ RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
Definition: rdkafka.h:7633
@ RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
Definition: rdkafka.h:7630
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_DEFAULT_BROKER_CONFIG
Definition: rdkafka.h:7627
RD_EXPORT const rd_kafka_AlterConsumerGroupOffsets_result_t * rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get AlterConsumerGroupOffsets result.
RD_EXPORT int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent partitioner.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_headers(const rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list.
RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk, rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete records (messages) in topic partitions older than the offsets provided.
RD_EXPORT rd_kafka_AclPermissionType_t rd_kafka_AclBinding_permission_type(const rd_kafka_AclBinding_t *acl)
RD_EXPORT const 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_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:6311
RD_EXPORT rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms)
Wait until all outstanding produce requests, et.al, are completed. This should typically be done prio...
RD_EXPORT 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_EXPORT int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_topic_t * rd_kafka_topic_new(rd_kafka_t *rk, const char *topic, rd_kafka_topic_conf_t *conf)
Creates a new topic handle for topic named topic.
RD_EXPORT void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist)
Release list memory.
rd_kafka_AclPermissionType_t
Apache Kafka ACL permission types.
Definition: rdkafka.h:9570
@ RD_KAFKA_ACL_PERMISSION_TYPE_DENY
Definition: rdkafka.h:9574
@ RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
Definition: rdkafka.h:9575
@ RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
Definition: rdkafka.h:9571
@ RD_KAFKA_ACL_PERMISSION_TYPE_ANY
Definition: rdkafka.h:9572
rd_kafka_thread_type_t
librdkafka internal thread type.
Definition: rdkafka.h:5409
@ RD_KAFKA_THREAD_MAIN
Definition: rdkafka.h:5410
@ RD_KAFKA_THREAD_BACKGROUND
Definition: rdkafka.h:5411
@ RD_KAFKA_THREAD_BROKER
Definition: rdkafka.h:5412
RD_EXPORT const 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_EXPORT int rd_kafka_ConsumerGroupListing_is_simple_consumer_group(const rd_kafka_ConsumerGroupListing_t *grplist)
Is the grplist group a simple consumer group.
RD_EXPORT void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags)
Destroy Kafka handle according to specified destroy flags.
RD_EXPORT rd_kafka_AclOperation_t rd_kafka_AclBinding_operation(const rd_kafka_AclBinding_t *acl)
RD_EXPORT const rd_kafka_CreatePartitions_result_t * rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev)
Get CreatePartitions result.
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List committed offsets for a set of partitions in a consumer group.
RD_EXPORT void rd_kafka_conf_set_closesocket_cb(rd_kafka_conf_t *conf, int(*closesocket_cb)(int sockfd, void *opaque))
Set close socket callback.
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random Murmur2 partitioner (Java compatible).
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics)
Returns the current topic subscription.
RD_EXPORT rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk)
Returns Kafka handle type.
RD_EXPORT void rd_kafka_topic_partition_set_leader_epoch(rd_kafka_topic_partition_t *rktpar, int32_t leader_epoch)
Sets the offset leader epoch (use -1 to clear).
rd_kafka_cert_enc_t
SSL certificate encoding.
Definition: rdkafka.h:2497
@ RD_KAFKA_CERT_ENC_PEM
Definition: rdkafka.h:2500
@ RD_KAFKA_CERT_ENC_PKCS12
Definition: rdkafka.h:2498
@ RD_KAFKA_CERT_ENC_DER
Definition: rdkafka.h:2499
RD_EXPORT const char * rd_kafka_err2str(rd_kafka_resp_err_t err)
Returns a human readable representation of a kafka error.
RD_EXPORT int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist)
Adds one or more brokers to the kafka handle's list of initial bootstrap brokers.
RD_EXPORT int rd_kafka_unittest(void)
Run librdkafka's built-in unit-tests.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, int async)
Commit offsets on broker for the provided list of partitions.
RD_EXPORT 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_EXPORT int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage)
Returns the latency for a produced message measured from the produce() call.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_acknowledgement(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement, void *ic_opaque)
Append an on_acknowledgement() interceptor.
rd_kafka_event_t rd_kafka_DeleteGroups_result_t
Definition: rdkafka.h:5782
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_EXPORT const rd_kafka_AlterUserScramCredentials_result_t * rd_kafka_event_AlterUserScramCredentials_result(rd_kafka_event_t *rkev)
Get AlterUserScramCredentials result.
rd_kafka_event_t rd_kafka_DescribeConfigs_result_t
Definition: rdkafka.h:5774
RD_EXPORT uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node)
Get the port of node.
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev)
RD_EXPORT int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms)
Returns the current ControllerId as reported in broker metadata.
RD_EXPORT int rd_kafka_ConfigEntry_is_default(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT rd_kafka_ScramMechanism_t rd_kafka_ScramCredentialInfo_mechanism(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the mechanism of a given ScramCredentialInfo.
RD_EXPORT rd_kafka_NewPartitions_t * rd_kafka_NewPartitions_new(const char *topic, size_t new_total_cnt, char *errstr, size_t errstr_size)
Create a new NewPartitions. This object is later passed to rd_kafka_CreatePartitions() to increase th...
RD_EXPORT void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events)
Enable event sourcing. events is a bitmask of RD_KAFKA_EVENT_* of events to enable for consumption by...
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve configuration value for property name.
RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox)
Converts the system errno value errnox to a rd_kafka_resp_err_t error code upon failure from the foll...
RD_EXPORT const 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_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
FNV-1a partitioner.
RD_EXPORT rd_kafka_error_t * rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms)
Initialize transactions for the producer instance.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_event_topic_partition(rd_kafka_event_t *rkev)
rd_kafka_event_t rd_kafka_DeleteConsumerGroupOffsets_result_t
Definition: rdkafka.h:5784
RD_EXPORT void rd_kafka_topic_conf_set_partitioner_cb(rd_kafka_topic_conf_t *topic_conf, int32_t(*partitioner)(const rd_kafka_topic_t *rkt, const void *keydata, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque))
Producer: Set partitioner callback in provided topic conf object.
RD_EXPORT const rd_kafka_acl_result_t ** rd_kafka_CreateAcls_result_acls(const rd_kafka_CreateAcls_result_t *result, size_t *cntp)
Get an array of acl results from a CreateAcls result.
RD_EXPORT const char * rd_kafka_ResourceType_name(rd_kafka_ResourceType_t restype)
RD_EXPORT void rd_kafka_destroy(rd_kafka_t *rk)
Destroy Kafka handle.
RD_EXPORT int rd_kafka_Node_id(const rd_kafka_Node_t *node)
Get the id of node.
RD_EXPORT void rd_kafka_conf_set_dr_msg_cb(rd_kafka_conf_t *conf, void(*dr_msg_cb)(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, void *opaque))
Producer: Set delivery report callback in provided conf object.
RD_EXPORT const rd_kafka_ListConsumerGroupOffsets_result_t * rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get ListConsumerGroupOffsets result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk,...)
Produce and send a single message to broker.
RD_EXPORT void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf)
Destroys a topic conf object.
RD_EXPORT const char * rd_kafka_MemberDescription_group_instance_id(const rd_kafka_MemberDescription_t *member)
Gets group instance id of member.
RD_EXPORT void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar)
Destroy a rd_kafka_topic_partition_t.
RD_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random FNV-1a partitioner.
RD_EXPORT void rd_kafka_DescribeConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Get configuration for the specified resources in configs.
rd_kafka_type_t
rd_kafka_t handle type.
Definition: rdkafka.h:209
@ RD_KAFKA_CONSUMER
Definition: rdkafka.h:211
@ RD_KAFKA_PRODUCER
Definition: rdkafka.h:210
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_DeleteRecords_result_offsets(const rd_kafka_DeleteRecords_result_t *result)
Get a list of topic and partition results from a DeleteRecords result. The returned objects will cont...
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a single rd_kafka_topic_conf_t value by property name.
RD_EXPORT 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_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_consumer(rd_kafka_t *rk)
RD_EXPORT void rd_kafka_set_log_level(rd_kafka_t *rk, int level)
Specifies the maximum logging level emitted by internal kafka logging and debugging.
RD_EXPORT const char * rd_kafka_AclBinding_name(const rd_kafka_AclBinding_t *acl)
RD_EXPORT void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu, int fd, const void *payload, size_t size)
Enable IO event triggering for queue.
rd_kafka_ResourcePatternType_t
Apache Kafka pattern types.
Definition: rdkafka.h:7734
@ RD_KAFKA_RESOURCE_PATTERN_LITERAL
Definition: rdkafka.h:7742
@ RD_KAFKA_RESOURCE_PATTERN_ANY
Definition: rdkafka.h:7738
@ RD_KAFKA_RESOURCE_PATTERN_MATCH
Definition: rdkafka.h:7740
@ RD_KAFKA_RESOURCE_PATTERN_PREFIXED
Definition: rdkafka.h:7744
@ RD_KAFKA_RESOURCE_PATTERN_UNKNOWN
Definition: rdkafka.h:7736
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupDescription_state(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets state for the grpdesc group.
RD_EXPORT int rd_kafka_error_is_retriable(const rd_kafka_error_t *error)
rd_kafka_ResourceType_t
Apache Kafka resource types.
Definition: rdkafka.h:7721
@ RD_KAFKA_RESOURCE_BROKER
Definition: rdkafka.h:7726
@ RD_KAFKA_RESOURCE__CNT
Definition: rdkafka.h:7727
@ RD_KAFKA_RESOURCE_TOPIC
Definition: rdkafka.h:7724
@ RD_KAFKA_RESOURCE_GROUP
Definition: rdkafka.h:7725
@ RD_KAFKA_RESOURCE_UNKNOWN
Definition: rdkafka.h:7722
@ RD_KAFKA_RESOURCE_ANY
Definition: rdkafka.h:7723
struct rd_kafka_ConsumerGroupDescription_s rd_kafka_ConsumerGroupDescription_t
DescribeConsumerGroups result type.
Definition: rdkafka.h:8581
RD_EXPORT void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf, void(*error_cb)(rd_kafka_t *rk, int err, const char *reason, void *opaque))
Set error callback in provided conf object.
RD_EXPORT int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition)
Stop consuming messages for topic rkt and partition, purging all messages currently in the local queu...
RD_EXPORT void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable)
Enable/disable creation of a queue specific to SASL events and callbacks.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_resume_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Resume producing consumption for the provided list of partitions.
RD_EXPORT void rd_kafka_ConfigResource_destroy(rd_kafka_ConfigResource_t *config)
Destroy and free a ConfigResource object previously created with rd_kafka_ConfigResource_new()
RD_EXPORT void rd_kafka_DeleteTopic_destroy_array(rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt)
Helper function to destroy all DeleteTopic objects in the del_topics array (of del_topic_cnt elements...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high)
Get last known low (oldest/beginning) and high (newest/end) offsets for partition.
RD_EXPORT const 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_EXPORT int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt, int32_t partition)
Check if partition is available (has a leader broker).
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_read(rd_kafka_consumer_group_metadata_t **cgmdp, const void *buffer, size_t size)
Reads serialized consumer group metadata and returns a consumer group metadata object....
RD_EXPORT void rd_kafka_ListConsumerGroups(rd_kafka_t *rk, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List the consumer groups available in the cluster.
RD_EXPORT const rd_kafka_DeleteGroups_result_t * rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev)
Get DeleteGroups result.
rd_kafka_msg_status_t
Message persistence status can be used by the application to find out if a produced message was persi...
Definition: rdkafka.h:1623
@ RD_KAFKA_MSG_STATUS_POSSIBLY_PERSISTED
Definition: rdkafka.h:1632
@ RD_KAFKA_MSG_STATUS_PERSISTED
Definition: rdkafka.h:1637
@ RD_KAFKA_MSG_STATUS_NOT_PERSISTED
Definition: rdkafka.h:1627
RD_EXPORT int rd_kafka_outq_len(rd_kafka_t *rk)
Returns the current out queue length.
RD_EXPORT int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms)
Polls the provided kafka handle for events.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_partition(rd_kafka_t *rk, const char *topic, int32_t partition)
RD_EXPORT const char * rd_kafka_ConfigEntry_name(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_event_t rd_kafka_AlterConfigs_result_t
Definition: rdkafka.h:5770
RD_EXPORT const char * rd_kafka_event_error_string(rd_kafka_event_t *rkev)
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1746
@ RD_KAFKA_CONF_OK
Definition: rdkafka.h:1751
@ RD_KAFKA_CONF_INVALID
Definition: rdkafka.h:1748
@ RD_KAFKA_CONF_UNKNOWN
Definition: rdkafka.h:1747
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs, const char *name, ssize_t name_size, const void *value, ssize_t value_size)
Add header with name name and value val (copied) of size size (not including null-terminator).
RD_EXPORT int32_t rd_kafka_topic_partition_get_leader_epoch(const rd_kafka_topic_partition_t *rktpar)
RD_EXPORT const char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
RD_EXPORT const rd_kafka_DeleteRecords_result_t * rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_AclBinding_principal(const rd_kafka_AclBinding_t *acl)
RD_EXPORT int32_t rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage)
RD_EXPORT rd_kafka_Uuid_t * rd_kafka_Uuid_copy(const rd_kafka_Uuid_t *uuid)
Copies the given UUID.
RD_EXPORT const rd_kafka_DescribeAcls_result_t * rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_DeleteRecords_t * rd_kafka_DeleteRecords_new(const rd_kafka_topic_partition_list_t *before_offsets)
Create a new DeleteRecords object. This object is later passed to rd_kafka_DeleteRecords().
RD_EXPORT const 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_EXPORT void rd_kafka_NewPartitions_destroy(rd_kafka_NewPartitions_t *new_parts)
Destroy and free a NewPartitions object previously created with rd_kafka_NewPartitions_new()
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_replica_assignment(rd_kafka_NewTopic_t *new_topic, int32_t partition, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker) assignment for partition to the replica set in broker_ids (of broker_id_cnt ...
RD_EXPORT void rd_kafka_AlterConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Update the configuration for the specified resources. Updates are not transactional so they may succe...
RD_EXPORT const char * rd_kafka_get_debug_contexts(void)
Retrieve supported debug contexts for use with the "debug" configuration property....
RD_EXPORT rd_kafka_ConfigResource_t * rd_kafka_ConfigResource_new(rd_kafka_ResourceType_t restype, const char *resname)
Create new ConfigResource object.
RD_EXPORT const 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_EXPORT void rd_kafka_AdminOptions_destroy(rd_kafka_AdminOptions_t *options)
Destroy a AdminOptions object.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert_verify_cb(rd_kafka_conf_t *conf, int(*ssl_cert_verify_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int *x509_error, int depth, const char *buf, size_t size, char *errstr, size_t errstr_size, void *opaque))
Sets the verification callback of the broker certificate.
RD_EXPORT 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_EXPORT int rd_kafka_consume_start(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Start consuming messages for topic rkt and partition at offset offset which may either be an absolute...
RD_EXPORT int rd_kafka_produce(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, void *payload, size_t len, const void *key, size_t keylen, void *msg_opaque)
Produce and send a single message to broker.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_for_times(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets, int timeout_ms)
Look up the offsets for the given partitions by timestamp.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_consume_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_consume() is called just prior to passing the message to the application in rd_kafka_consumer_poll...
Definition: rdkafka.h:6384
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_broker(rd_kafka_AdminOptions_t *options, int32_t broker_id, char *errstr, size_t errstr_size)
Override what broker the Admin request will be sent to.
RD_EXPORT void rd_kafka_DeleteTopics(rd_kafka_t *rk, rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete topics from cluster as specified by the topics array of size topic_cnt elements.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_last_error(void)
Returns the last error code generated by a legacy API call in the current thread.
RD_EXPORT const char * rd_kafka_UserScramCredentialsDescription_user(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the username of a UserScramCredentialsDescription.
RD_EXPORT const rd_kafka_error_t * rd_kafka_UserScramCredentialsDescription_error(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the error associated with a UserScramCredentialsDescription.
RD_EXPORT const char * rd_kafka_event_stats(rd_kafka_event_t *rkev)
Extract stats from the event.
struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t
ACL Binding is used to create access control lists.
Definition: rdkafka.h:9545
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:6531
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_validate_only(rd_kafka_AdminOptions_t *options, int true_or_false, char *errstr, size_t errstr_size)
Tell broker to only validate the request, without performing the requested operation (create topics,...
RD_EXPORT 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_EXPORT const rd_kafka_ListConsumerGroups_result_t * rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev)
Get ListConsumerGroups result.
rd_kafka_event_t rd_kafka_IncrementalAlterConfigs_result_t
Definition: rdkafka.h:5772
RD_EXPORT const char * rd_kafka_error_string(const rd_kafka_error_t *error)
RD_EXPORT const rd_kafka_DescribeCluster_result_t * rd_kafka_event_DescribeCluster_result(rd_kafka_event_t *rkev)
Get DescribeCluster result.
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_EXPORT const rd_kafka_error_t * rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres)
RD_EXPORT int rd_kafka_TopicDescription_is_internal(const rd_kafka_TopicDescription_t *topicdesc)
Gets if the topicdesc topic is internal.
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_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_EXPORT int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random partitioner.
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_write(const rd_kafka_consumer_group_metadata_t *cgmd, void **bufferp, size_t *sizep)
Serialize the consumer group metadata to a binary format. This is mainly for client binding use and n...
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_DeleteTopics_result_topics(const rd_kafka_DeleteTopics_result_t *result, size_t *cntp)
Get an array of topic results from a DeleteTopics result.
RD_EXPORT void rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *)
Frees the consumer group metadata object as returned by rd_kafka_consumer_group_metadata().
struct rd_kafka_ListConsumerGroupOffsets_s rd_kafka_ListConsumerGroupOffsets_t
Definition: rdkafka.h:8936
RD_EXPORT void * rd_kafka_opaque(const rd_kafka_t *rk)
Retrieves the opaque pointer previously set with rd_kafka_conf_set_opaque()
RD_EXPORT void rd_kafka_event_destroy(rd_kafka_event_t *rkev)
Destroy an event.
RD_EXPORT const rd_kafka_DeleteAcls_result_response_t ** rd_kafka_DeleteAcls_result_responses(const rd_kafka_DeleteAcls_result_t *result, size_t *cntp)
Get an array of DeleteAcls result responses from a DeleteAcls result.
struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t
Member description included in ConsumerGroupDescription.
Definition: rdkafka.h:8588
RD_EXPORT rd_kafka_msg_status_t rd_kafka_message_status(const rd_kafka_message_t *rkmessage)
Returns the message's persistence status in the topic log.
RD_EXPORT const 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_event_t rd_kafka_DescribeUserScramCredentials_result_t
Definition: rdkafka.h:5794
RD_EXPORT void rd_kafka_AclBinding_destroy_array(rd_kafka_AclBinding_t **acl_bindings, size_t acl_bindings_cnt)
Helper function to destroy all AclBinding objects in the acl_bindings array (of acl_bindings_cnt elem...
RD_EXPORT void rd_kafka_conf_destroy(rd_kafka_conf_t *conf)
Destroys a conf object.
RD_EXPORT void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu)
Cancels the current rd_kafka_queue_poll() on rkqu.
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy(rd_kafka_UserScramCredentialAlteration_t *alteration)
Destroys a UserScramCredentialAlteration given its pointer.
RD_EXPORT const char * rd_kafka_MemberDescription_client_id(const rd_kafka_MemberDescription_t *member)
Gets client id of member.
RD_EXPORT void rd_kafka_topic_partition_list_sort(rd_kafka_topic_partition_list_t *rktparlist, int(*cmp)(const void *a, const void *b, void *cmp_opaque), void *cmp_opaque)
Sort list using comparator cmp.
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_AclBinding_restype(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_send(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_send_t *on_send, void *ic_opaque)
Append an on_send() interceptor.
RD_EXPORT const char * rd_kafka_ConfigResource_name(const rd_kafka_ConfigResource_t *config)
RD_EXPORT const int rd_kafka_TopicPartitionInfo_partition(const rd_kafka_TopicPartitionInfo_t *partition)
Gets the partition id for partition.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_queue(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, rd_kafka_queue_t *rkqu, void(*cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *commit_opaque), void *commit_opaque)
Commit offsets on broker for the provided list of partitions.
rd_kafka_timestamp_type_t
Definition: rdkafka.h:220
@ RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafka.h:221
@ RD_KAFKA_TIMESTAMP_CREATE_TIME
Definition: rdkafka.h:222
@ RD_KAFKA_TIMESTAMP_LOG_APPEND_TIME
Definition: rdkafka.h:223
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_detach_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list and detach the list from the message making the application the owner of ...
RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs)
Returns the number of header key/value pairs.
RD_EXPORT void rd_kafka_DeleteGroups(rd_kafka_t *rk, rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete groups from cluster as specified by the del_groups array of size del_group_cnt elements.
RD_EXPORT rd_kafka_ListConsumerGroupOffsets_t * rd_kafka_ListConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new ListConsumerGroupOffsets object. This object is later passed to rd_kafka_ListConsumerGro...
rd_kafka_AclBinding_t rd_kafka_AclBindingFilter_t
ACL Binding filter is used to filter access control lists.
Definition: rdkafka.h:9551
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete committed offsets for a set of partitions in a consumer group. This will succeed at the partit...
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_ConfigResource_type(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_new(int size)
Create a new list/vector Topic+Partition container.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_set_config(rd_kafka_ConfigResource_t *config, const char *name, const char *value)
Set configuration name value pair.
rd_kafka_ScramMechanism_t
Apache Kafka ScramMechanism values.
Definition: rdkafka.h:9306
RD_EXPORT rd_kafka_error_t * rd_kafka_error_new(rd_kafka_resp_err_t code, const char *fmt,...) RD_FORMAT(printf
Create a new error object with error code and optional human readable error string in fmt.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_destroy_t(void *ic_opaque)
on_conf_destroy() is called from rd_kafka_*_conf_destroy() in the order the interceptors were added.
Definition: rdkafka.h:6274
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.
struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t
Member assignment included in MemberDescription.
Definition: rdkafka.h:8594
RD_EXPORT const char * rd_kafka_AclOperation_name(rd_kafka_AclOperation_t acl_operation)
RD_EXPORT rd_kafka_ConfigSource_t rd_kafka_ConfigEntry_source(const rd_kafka_ConfigEntry_t *entry)
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:664
const char * name
Definition: rdkafka.h:666
rd_kafka_resp_err_t code
Definition: rdkafka.h:665
const char * desc
Definition: rdkafka.h:667
Group information.
Definition: rdkafka.h:5164
struct rd_kafka_metadata_broker broker
Definition: rdkafka.h:5165
int member_cnt
Definition: rdkafka.h:5172
char * state
Definition: rdkafka.h:5168
char * group
Definition: rdkafka.h:5166
struct rd_kafka_group_member_info * members
Definition: rdkafka.h:5171
char * protocol
Definition: rdkafka.h:5170
char * protocol_type
Definition: rdkafka.h:5169
rd_kafka_resp_err_t err
Definition: rdkafka.h:5167
List of groups.
Definition: rdkafka.h:5180
int group_cnt
Definition: rdkafka.h:5182
struct rd_kafka_group_info * groups
Definition: rdkafka.h:5181
Group member information.
Definition: rdkafka.h:5134
char * member_id
Definition: rdkafka.h:5135
int member_assignment_size
Definition: rdkafka.h:5143
int member_metadata_size
Definition: rdkafka.h:5140
void * member_metadata
Definition: rdkafka.h:5138
char * client_host
Definition: rdkafka.h:5137
void * member_assignment
Definition: rdkafka.h:5141
char * client_id
Definition: rdkafka.h:5136
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1459
size_t key_len
Definition: rdkafka.h:1472
size_t len
Definition: rdkafka.h:1467
void * _private
Definition: rdkafka.h:1482
void * key
Definition: rdkafka.h:1470
int64_t offset
Definition: rdkafka.h:1474
void * payload
Definition: rdkafka.h:1463
int32_t partition
Definition: rdkafka.h:1462
rd_kafka_topic_t * rkt
Definition: rdkafka.h:1461
rd_kafka_resp_err_t err
Definition: rdkafka.h:1460
Broker information.
Definition: rdkafka.h:4989
int32_t id
Definition: rdkafka.h:4990
int port
Definition: rdkafka.h:4992
char * host
Definition: rdkafka.h:4991
Partition information.
Definition: rdkafka.h:4998
int32_t leader
Definition: rdkafka.h:5001
int32_t * isrs
Definition: rdkafka.h:5005
int replica_cnt
Definition: rdkafka.h:5002
rd_kafka_resp_err_t err
Definition: rdkafka.h:5000
int isr_cnt
Definition: rdkafka.h:5004
int32_t id
Definition: rdkafka.h:4999
int32_t * replicas
Definition: rdkafka.h:5003
Metadata container.
Definition: rdkafka.h:5022
int32_t orig_broker_id
Definition: rdkafka.h:5029
char * orig_broker_name
Definition: rdkafka.h:5030
int broker_cnt
Definition: rdkafka.h:5023
struct rd_kafka_metadata_topic * topics
Definition: rdkafka.h:5027
int topic_cnt
Definition: rdkafka.h:5026
struct rd_kafka_metadata_broker * brokers
Definition: rdkafka.h:5024
Topic information.
Definition: rdkafka.h:5011
char * topic
Definition: rdkafka.h:5012
struct rd_kafka_metadata_partition * partitions
Definition: rdkafka.h:5014
rd_kafka_resp_err_t err
Definition: rdkafka.h:5015
int partition_cnt
Definition: rdkafka.h:5013
A growable list of Topic+Partitions.
Definition: rdkafka.h:970
int cnt
Definition: rdkafka.h:971
int size
Definition: rdkafka.h:972
rd_kafka_topic_partition_t * elems
Definition: rdkafka.h:973
Topic+Partition place holder.
Definition: rdkafka.h:919
int64_t offset
Definition: rdkafka.h:922
size_t metadata_size
Definition: rdkafka.h:924
void * opaque
Definition: rdkafka.h:925
char * topic
Definition: rdkafka.h:920
void * metadata
Definition: rdkafka.h:923
int32_t partition
Definition: rdkafka.h:921
rd_kafka_resp_err_t err
Definition: rdkafka.h:926
void * _private
Definition: rdkafka.h:927
VTYPE + argument container for use with rd_kafka_produce_va()
Definition: rdkafka.h:1163
rd_kafka_vtype_t vtype
Definition: rdkafka.h:1164