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 0x020300ff
171 
180 RD_EXPORT
181 int rd_kafka_version(void);
182 
188 RD_EXPORT
189 const char *rd_kafka_version_str(void);
190 
209 typedef enum rd_kafka_type_t {
213 
214 
225 
226 
227 
234 RD_EXPORT
235 const char *rd_kafka_get_debug_contexts(void);
236 
244 #define RD_KAFKA_DEBUG_CONTEXTS \
245  "all,generic,broker,topic,metadata,feature,queue,msg,protocol,cgrp," \
246  "security,fetch,interceptor,plugin,consumer,admin,eos,mock,assignor," \
247  "conf"
248 
249 
250 /* @cond NO_DOC */
251 /* Private types to provide ABI compatibility */
252 typedef struct rd_kafka_s rd_kafka_t;
253 typedef struct rd_kafka_topic_s rd_kafka_topic_t;
254 typedef struct rd_kafka_conf_s rd_kafka_conf_t;
255 typedef struct rd_kafka_topic_conf_s rd_kafka_topic_conf_t;
256 typedef struct rd_kafka_queue_s rd_kafka_queue_t;
257 typedef struct rd_kafka_op_s rd_kafka_event_t;
258 typedef struct rd_kafka_topic_result_s rd_kafka_topic_result_t;
259 typedef struct rd_kafka_consumer_group_metadata_s
260  rd_kafka_consumer_group_metadata_t;
261 typedef struct rd_kafka_error_s rd_kafka_error_t;
262 typedef struct rd_kafka_headers_s rd_kafka_headers_t;
263 typedef struct rd_kafka_group_result_s rd_kafka_group_result_t;
264 typedef struct rd_kafka_acl_result_s rd_kafka_acl_result_t;
265 typedef struct rd_kafka_Uuid_s rd_kafka_Uuid_t;
266 /* @endcond */
267 
268 
281 typedef enum {
282  /* Internal errors to rdkafka: */
410 
413 
414  /* Kafka broker errors: */
430 #define RD_KAFKA_RESP_ERR_NOT_LEADER_OR_FOLLOWER \
431  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION
432  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION = 6,
450 #define RD_KAFKA_RESP_ERR_GROUP_LOAD_IN_PROGRESS \
451  RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
452 
455 #define RD_KAFKA_RESP_ERR_GROUP_COORDINATOR_NOT_AVAILABLE \
456  RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
457 
460 #define RD_KAFKA_RESP_ERR_NOT_COORDINATOR_FOR_GROUP \
461  RD_KAFKA_RESP_ERR_NOT_COORDINATOR
462 
634 
635  RD_KAFKA_RESP_ERR_END_ALL,
637 
638 
646  const char *name;
647  const char *desc;
648 };
649 
650 
654 RD_EXPORT
655 void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs,
656  size_t *cntp);
657 
658 
659 
665 RD_EXPORT
666 const char *rd_kafka_err2str(rd_kafka_resp_err_t err);
667 
668 
669 
675 RD_EXPORT
676 const char *rd_kafka_err2name(rd_kafka_resp_err_t err);
677 
678 
704 RD_EXPORT
706 
707 
732 RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox);
733 
734 
747 RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void);
748 
749 
750 
777 RD_EXPORT
779 rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size);
780 
781 
799 RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk,
801  const char *reason);
802 
803 
808 RD_EXPORT
809 rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error);
810 
819 RD_EXPORT
820 const char *rd_kafka_error_name(const rd_kafka_error_t *error);
821 
828 RD_EXPORT
829 const char *rd_kafka_error_string(const rd_kafka_error_t *error);
830 
831 
836 RD_EXPORT
837 int rd_kafka_error_is_fatal(const rd_kafka_error_t *error);
838 
839 
844 RD_EXPORT
845 int rd_kafka_error_is_retriable(const rd_kafka_error_t *error);
846 
847 
858 RD_EXPORT
859 int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error);
860 
866 RD_EXPORT
867 void rd_kafka_error_destroy(rd_kafka_error_t *error);
868 
869 
878 RD_EXPORT
879 rd_kafka_error_t *rd_kafka_error_new(rd_kafka_resp_err_t code,
880  const char *fmt,
881  ...) RD_FORMAT(printf, 2, 3);
882 
883 
900  char *topic;
901  int32_t partition;
902  int64_t offset;
903  void *metadata;
904  size_t metadata_size;
905  void *opaque;
907  void *_private;
911 } rd_kafka_topic_partition_t;
912 
917 RD_EXPORT
918 void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar);
919 
920 
929 RD_EXPORT
931  rd_kafka_topic_partition_t *rktpar,
932  int32_t leader_epoch);
933 
942 RD_EXPORT
944  const rd_kafka_topic_partition_t *rktpar);
945 
951  int cnt;
952  int size;
953  rd_kafka_topic_partition_t *elems;
954 } rd_kafka_topic_partition_list_t;
955 
970 RD_EXPORT
971 rd_kafka_topic_partition_list_t *rd_kafka_topic_partition_list_new(int size);
972 
976 RD_EXPORT
978  rd_kafka_topic_partition_list_t *rkparlist);
979 
989 RD_EXPORT
990 rd_kafka_topic_partition_t *
991 rd_kafka_topic_partition_list_add(rd_kafka_topic_partition_list_t *rktparlist,
992  const char *topic,
993  int32_t partition);
994 
995 
1004 RD_EXPORT
1006  rd_kafka_topic_partition_list_t *rktparlist,
1007  const char *topic,
1008  int32_t start,
1009  int32_t stop);
1010 
1011 
1012 
1024 RD_EXPORT
1026  rd_kafka_topic_partition_list_t *rktparlist,
1027  const char *topic,
1028  int32_t partition);
1029 
1030 
1038 RD_EXPORT
1040  rd_kafka_topic_partition_list_t *rktparlist,
1041  int idx);
1042 
1043 
1051 RD_EXPORT
1052 rd_kafka_topic_partition_list_t *
1053 rd_kafka_topic_partition_list_copy(const rd_kafka_topic_partition_list_t *src);
1054 
1055 
1056 
1064 RD_EXPORT
1066  rd_kafka_topic_partition_list_t *rktparlist,
1067  const char *topic,
1068  int32_t partition,
1069  int64_t offset);
1070 
1071 
1072 
1078 RD_EXPORT
1079 rd_kafka_topic_partition_t *rd_kafka_topic_partition_list_find(
1080  const rd_kafka_topic_partition_list_t *rktparlist,
1081  const char *topic,
1082  int32_t partition);
1083 
1084 
1094 RD_EXPORT void rd_kafka_topic_partition_list_sort(
1095  rd_kafka_topic_partition_list_t *rktparlist,
1096  int (*cmp)(const void *a, const void *b, void *cmp_opaque),
1097  void *cmp_opaque);
1098 
1099 
1117 typedef enum rd_kafka_vtype_t {
1135 
1136 
1143 typedef struct rd_kafka_vu_s {
1146  union {
1147  const char *cstr;
1148  rd_kafka_topic_t *rkt;
1149  int i;
1150  int32_t i32;
1151  int64_t i64;
1152  struct {
1153  void *ptr;
1154  size_t size;
1155  } mem;
1156  struct {
1157  const char *name;
1158  const void *val;
1159  ssize_t size;
1160  } header;
1161  rd_kafka_headers_t *headers;
1162  void *ptr;
1163  char _pad[64];
1164  } u;
1165 } rd_kafka_vu_t;
1166 
1175 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
1176 
1182 #define RD_KAFKA_V_TOPIC(topic) \
1183  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
1184  (const char *)topic
1185 
1190 #define RD_KAFKA_V_RKT(rkt) \
1191  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
1192  (rd_kafka_topic_t *)rkt
1193 
1198 #define RD_KAFKA_V_PARTITION(partition) \
1199  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
1200  (int32_t)partition
1201 
1206 #define RD_KAFKA_V_VALUE(VALUE, LEN) \
1207  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
1208  (void *)VALUE, (size_t)LEN
1209 
1214 #define RD_KAFKA_V_KEY(KEY, LEN) \
1215  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
1216  (void *)KEY, (size_t)LEN
1217 
1224 #define RD_KAFKA_V_OPAQUE(msg_opaque) \
1225  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, msg_opaque), \
1226  (void *)msg_opaque
1227 
1233 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
1234  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), (int)msgflags
1235 
1241 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
1242  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
1243  (int64_t)timestamp
1244 
1252 #define RD_KAFKA_V_HEADER(NAME, VALUE, LEN) \
1253  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
1254  const void *, VALUE, ssize_t, LEN), \
1255  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
1256 
1268 #define RD_KAFKA_V_HEADERS(HDRS) \
1269  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
1270  (rd_kafka_headers_t *)HDRS
1271 
1272 
1303 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new(size_t initial_count);
1304 
1309 RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs);
1310 
1314 RD_EXPORT rd_kafka_headers_t *
1315 rd_kafka_headers_copy(const rd_kafka_headers_t *src);
1316 
1334 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs,
1335  const char *name,
1336  ssize_t name_size,
1337  const void *value,
1338  ssize_t value_size);
1339 
1347 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs,
1348  const char *name);
1349 
1350 
1368 RD_EXPORT rd_kafka_resp_err_t
1369 rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs,
1370  const char *name,
1371  const void **valuep,
1372  size_t *sizep);
1373 
1387 RD_EXPORT rd_kafka_resp_err_t
1388 rd_kafka_header_get(const rd_kafka_headers_t *hdrs,
1389  size_t idx,
1390  const char *name,
1391  const void **valuep,
1392  size_t *sizep);
1393 
1394 
1402 RD_EXPORT rd_kafka_resp_err_t
1403 rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs,
1404  size_t idx,
1405  const char **namep,
1406  const void **valuep,
1407  size_t *sizep);
1408 
1409 
1410 
1423 // FIXME: This doesn't show up in docs for some reason
1424 // "Compound rd_kafka_message_t is not documented."
1425 
1439 typedef struct rd_kafka_message_s {
1441  rd_kafka_topic_t *rkt;
1442  int32_t partition;
1443  void *payload;
1447  size_t len;
1450  void *key;
1452  size_t key_len;
1454  int64_t offset;
1462  void *_private;
1469 } rd_kafka_message_t;
1470 
1471 
1475 RD_EXPORT
1476 void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage);
1477 
1478 
1479 
1486 RD_EXPORT
1487 const char *rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage);
1488 
1489 
1501 RD_EXPORT
1502 int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage,
1503  rd_kafka_timestamp_type_t *tstype);
1504 
1505 
1506 
1513 RD_EXPORT
1514 int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage);
1515 
1516 
1523 RD_EXPORT
1524 int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage);
1525 
1526 
1543 RD_EXPORT rd_kafka_resp_err_t
1544 rd_kafka_message_headers(const rd_kafka_message_t *rkmessage,
1545  rd_kafka_headers_t **hdrsp);
1546 
1558 RD_EXPORT rd_kafka_resp_err_t
1559 rd_kafka_message_detach_headers(rd_kafka_message_t *rkmessage,
1560  rd_kafka_headers_t **hdrsp);
1561 
1562 
1575 RD_EXPORT
1576 void rd_kafka_message_set_headers(rd_kafka_message_t *rkmessage,
1577  rd_kafka_headers_t *hdrs);
1578 
1579 
1585 RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs);
1586 
1587 
1593 typedef enum {
1598 
1603 
1609 
1610 
1617 RD_EXPORT rd_kafka_msg_status_t
1618 rd_kafka_message_status(const rd_kafka_message_t *rkmessage);
1619 
1620 
1628 RD_EXPORT int32_t
1629 rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage);
1630 
1631 
1648 RD_EXPORT const char *rd_kafka_Uuid_base64str(const rd_kafka_Uuid_t *uuid);
1649 
1657 RD_EXPORT int64_t
1658 rd_kafka_Uuid_least_significant_bits(const rd_kafka_Uuid_t *uuid);
1659 
1660 
1668 RD_EXPORT int64_t
1669 rd_kafka_Uuid_most_significant_bits(const rd_kafka_Uuid_t *uuid);
1670 
1671 
1681 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_new(int64_t most_significant_bits,
1682  int64_t least_significant_bits);
1683 
1692 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_copy(const rd_kafka_Uuid_t *uuid);
1693 
1699 RD_EXPORT void rd_kafka_Uuid_destroy(rd_kafka_Uuid_t *uuid);
1700 
1716 typedef enum {
1723 
1724 
1758 RD_EXPORT
1759 rd_kafka_conf_t *rd_kafka_conf_new(void);
1760 
1761 
1765 RD_EXPORT
1766 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1767 
1768 
1775 RD_EXPORT
1776 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1777 
1778 
1783 RD_EXPORT
1784 rd_kafka_conf_t *rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf,
1785  size_t filter_cnt,
1786  const char **filter);
1787 
1788 
1789 
1798 RD_EXPORT
1799 const rd_kafka_conf_t *rd_kafka_conf(rd_kafka_t *rk);
1800 
1801 
1821 RD_EXPORT
1822 rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf,
1823  const char *name,
1824  const char *value,
1825  char *errstr,
1826  size_t errstr_size);
1827 
1828 
1834 RD_EXPORT
1835 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1836 
1837 
1874  rd_kafka_conf_t *conf,
1875  void (*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque));
1876 
1877 
1881 RD_EXPORT
1882 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1883  void (*dr_cb)(rd_kafka_t *rk,
1884  void *payload,
1885  size_t len,
1886  rd_kafka_resp_err_t err,
1887  void *opaque,
1888  void *msg_opaque));
1889 
1920 RD_EXPORT
1922  rd_kafka_conf_t *conf,
1923  void (*dr_msg_cb)(rd_kafka_t *rk,
1924  const rd_kafka_message_t *rkmessage,
1925  void *opaque));
1926 
1927 
1935 RD_EXPORT
1937  rd_kafka_conf_t *conf,
1938  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque));
1939 
2044 RD_EXPORT
2046  rd_kafka_conf_t *conf,
2047  void (*rebalance_cb)(rd_kafka_t *rk,
2048  rd_kafka_resp_err_t err,
2049  rd_kafka_topic_partition_list_t *partitions,
2050  void *opaque));
2051 
2052 
2053 
2071 RD_EXPORT
2073  rd_kafka_conf_t *conf,
2074  void (*offset_commit_cb)(rd_kafka_t *rk,
2075  rd_kafka_resp_err_t err,
2076  rd_kafka_topic_partition_list_t *offsets,
2077  void *opaque));
2078 
2079 
2102 RD_EXPORT
2103 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
2104  void (*error_cb)(rd_kafka_t *rk,
2105  int err,
2106  const char *reason,
2107  void *opaque));
2108 
2126 RD_EXPORT
2127 void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf,
2128  void (*throttle_cb)(rd_kafka_t *rk,
2129  const char *broker_name,
2130  int32_t broker_id,
2131  int throttle_time_ms,
2132  void *opaque));
2133 
2134 
2151 RD_EXPORT
2152 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
2153  void (*log_cb)(const rd_kafka_t *rk,
2154  int level,
2155  const char *fac,
2156  const char *buf));
2157 
2158 
2181 RD_EXPORT
2183  rd_kafka_conf_t *conf,
2184  int (*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque));
2185 
2235 RD_EXPORT
2237  rd_kafka_conf_t *conf,
2238  void (*oauthbearer_token_refresh_cb)(rd_kafka_t *rk,
2239  const char *oauthbearer_config,
2240  void *opaque));
2241 
2269 RD_EXPORT
2270 void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable);
2271 
2272 
2290 RD_EXPORT
2292  rd_kafka_conf_t *conf,
2293  int (*socket_cb)(int domain, int type, int protocol, void *opaque));
2294 
2295 
2296 
2312 RD_EXPORT void
2313 rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf,
2314  int (*connect_cb)(int sockfd,
2315  const struct sockaddr *addr,
2316  int addrlen,
2317  const char *id,
2318  void *opaque));
2319 
2330 RD_EXPORT void rd_kafka_conf_set_closesocket_cb(
2331  rd_kafka_conf_t *conf,
2332  int (*closesocket_cb)(int sockfd, void *opaque));
2333 
2334 
2335 
2336 #ifndef _WIN32
2337 
2354 RD_EXPORT
2356  rd_kafka_conf_t *conf,
2357  int (*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque));
2358 #endif
2359 
2361 struct addrinfo;
2362 
2382 RD_EXPORT void
2383 rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf,
2384  int (*resolve_cb)(const char *node,
2385  const char *service,
2386  const struct addrinfo *hints,
2387  struct addrinfo **res,
2388  void *opaque));
2389 
2431 RD_EXPORT
2433  rd_kafka_conf_t *conf,
2434  int (*ssl_cert_verify_cb)(rd_kafka_t *rk,
2435  const char *broker_name,
2436  int32_t broker_id,
2437  int *x509_error,
2438  int depth,
2439  const char *buf,
2440  size_t size,
2441  char *errstr,
2442  size_t errstr_size,
2443  void *opaque));
2444 
2445 
2453 typedef enum rd_kafka_cert_type_t {
2457  RD_KAFKA_CERT__CNT,
2459 
2467 typedef enum rd_kafka_cert_enc_t {
2471  RD_KAFKA_CERT_ENC__CNT,
2473 
2474 
2517 RD_EXPORT rd_kafka_conf_res_t
2518 rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf,
2519  rd_kafka_cert_type_t cert_type,
2520  rd_kafka_cert_enc_t cert_enc,
2521  const void *buffer,
2522  size_t size,
2523  char *errstr,
2524  size_t errstr_size);
2525 
2526 
2541 RD_EXPORT
2542 void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf,
2543  void *callback_data);
2544 
2545 
2551 RD_EXPORT
2552 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
2553 
2558 RD_EXPORT
2559 void *rd_kafka_opaque(const rd_kafka_t *rk);
2560 
2561 
2562 
2576 RD_EXPORT
2577 void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf,
2578  rd_kafka_topic_conf_t *tconf);
2579 
2592 RD_EXPORT rd_kafka_topic_conf_t *
2593 rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf);
2594 
2595 
2615 RD_EXPORT
2616 rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf,
2617  const char *name,
2618  char *dest,
2619  size_t *dest_size);
2620 
2621 
2627 RD_EXPORT
2628 rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf,
2629  const char *name,
2630  char *dest,
2631  size_t *dest_size);
2632 
2633 
2642 RD_EXPORT
2643 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
2644 
2645 
2654 RD_EXPORT
2655 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
2656  size_t *cntp);
2657 
2662 RD_EXPORT
2663 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
2664 
2672 RD_EXPORT
2673 void rd_kafka_conf_properties_show(FILE *fp);
2674 
2691 RD_EXPORT
2692 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
2693 
2694 
2698 RD_EXPORT
2699 rd_kafka_topic_conf_t *
2700 rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf);
2701 
2706 RD_EXPORT
2707 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup(rd_kafka_t *rk);
2708 
2709 
2713 RD_EXPORT
2714 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
2715 
2716 
2725 RD_EXPORT
2726 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
2727  const char *name,
2728  const char *value,
2729  char *errstr,
2730  size_t errstr_size);
2731 
2738 RD_EXPORT
2739 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf,
2740  void *rkt_opaque);
2741 
2742 
2762 RD_EXPORT
2764  rd_kafka_topic_conf_t *topic_conf,
2765  int32_t (*partitioner)(const rd_kafka_topic_t *rkt,
2766  const void *keydata,
2767  size_t keylen,
2768  int32_t partition_cnt,
2769  void *rkt_opaque,
2770  void *msg_opaque));
2771 
2772 
2799  rd_kafka_topic_conf_t *topic_conf,
2800  int (*msg_order_cmp)(const rd_kafka_message_t *a,
2801  const rd_kafka_message_t *b));
2802 
2803 
2811 RD_EXPORT
2812 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
2813  int32_t partition);
2814 
2815 
2816 /*******************************************************************
2817  * *
2818  * Partitioners provided by rdkafka *
2819  * *
2820  *******************************************************************/
2821 
2835 RD_EXPORT
2836 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
2837  const void *key,
2838  size_t keylen,
2839  int32_t partition_cnt,
2840  void *rkt_opaque,
2841  void *msg_opaque);
2842 
2856 RD_EXPORT
2857 int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt,
2858  const void *key,
2859  size_t keylen,
2860  int32_t partition_cnt,
2861  void *rkt_opaque,
2862  void *msg_opaque);
2863 
2879 RD_EXPORT
2880 int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt,
2881  const void *key,
2882  size_t keylen,
2883  int32_t partition_cnt,
2884  void *rkt_opaque,
2885  void *msg_opaque);
2886 
2887 
2901 RD_EXPORT
2902 int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt,
2903  const void *key,
2904  size_t keylen,
2905  int32_t partition_cnt,
2906  void *rkt_opaque,
2907  void *msg_opaque);
2908 
2923 RD_EXPORT
2924 int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt,
2925  const void *key,
2926  size_t keylen,
2927  int32_t partition_cnt,
2928  void *rkt_opaque,
2929  void *msg_opaque);
2930 
2931 
2945 RD_EXPORT
2946 int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt,
2947  const void *key,
2948  size_t keylen,
2949  int32_t partition_cnt,
2950  void *rkt_opaque,
2951  void *msg_opaque);
2952 
2953 
2968 RD_EXPORT
2969 int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt,
2970  const void *key,
2971  size_t keylen,
2972  int32_t partition_cnt,
2973  void *rkt_opaque,
2974  void *msg_opaque);
2975 
2976 
3016 RD_EXPORT
3017 rd_kafka_t *rd_kafka_new(rd_kafka_type_t type,
3018  rd_kafka_conf_t *conf,
3019  char *errstr,
3020  size_t errstr_size);
3021 
3022 
3037 RD_EXPORT
3038 void rd_kafka_destroy(rd_kafka_t *rk);
3039 
3040 
3045 RD_EXPORT
3046 void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags);
3047 
3062 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
3063 
3064 
3065 
3069 RD_EXPORT
3070 const char *rd_kafka_name(const rd_kafka_t *rk);
3071 
3072 
3076 RD_EXPORT
3077 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
3078 
3079 
3090 RD_EXPORT
3091 char *rd_kafka_memberid(const rd_kafka_t *rk);
3092 
3093 
3094 
3113 RD_EXPORT
3114 char *rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms);
3115 
3116 
3132 RD_EXPORT
3133 int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms);
3134 
3135 
3157 RD_EXPORT
3158 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk,
3159  const char *topic,
3160  rd_kafka_topic_conf_t *conf);
3161 
3162 
3163 
3172 RD_EXPORT
3173 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
3174 
3175 
3179 RD_EXPORT
3180 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
3181 
3182 
3187 RD_EXPORT
3188 void *rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt);
3189 
3190 
3197 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
3198 
3199 
3227 RD_EXPORT
3228 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
3229 
3230 
3241 RD_EXPORT
3242 void rd_kafka_yield(rd_kafka_t *rk);
3243 
3244 
3245 
3253 RD_EXPORT rd_kafka_resp_err_t
3254 rd_kafka_pause_partitions(rd_kafka_t *rk,
3255  rd_kafka_topic_partition_list_t *partitions);
3256 
3257 
3258 
3266 RD_EXPORT rd_kafka_resp_err_t
3267 rd_kafka_resume_partitions(rd_kafka_t *rk,
3268  rd_kafka_topic_partition_list_t *partitions);
3269 
3270 
3271 
3280 RD_EXPORT rd_kafka_resp_err_t
3281 rd_kafka_query_watermark_offsets(rd_kafka_t *rk,
3282  const char *topic,
3283  int32_t partition,
3284  int64_t *low,
3285  int64_t *high,
3286  int timeout_ms);
3287 
3288 
3305 RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk,
3306  const char *topic,
3307  int32_t partition,
3308  int64_t *low,
3309  int64_t *high);
3310 
3311 
3312 
3338 RD_EXPORT rd_kafka_resp_err_t
3339 rd_kafka_offsets_for_times(rd_kafka_t *rk,
3340  rd_kafka_topic_partition_list_t *offsets,
3341  int timeout_ms);
3342 
3343 
3344 
3358 RD_EXPORT
3359 void *rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size);
3360 
3361 
3362 
3376 RD_EXPORT
3377 void *rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size);
3378 
3379 
3380 
3397 RD_EXPORT
3398 void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr);
3399 
3400 
3422 RD_EXPORT
3423 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
3424 
3428 RD_EXPORT
3429 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
3430 
3431 
3438 RD_EXPORT
3439 rd_kafka_queue_t *rd_kafka_queue_get_main(rd_kafka_t *rk);
3440 
3441 
3442 
3452 RD_EXPORT
3453 rd_kafka_queue_t *rd_kafka_queue_get_sasl(rd_kafka_t *rk);
3454 
3455 
3471 RD_EXPORT
3472 rd_kafka_error_t *rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk);
3473 
3474 
3489 RD_EXPORT
3490 rd_kafka_error_t *rd_kafka_sasl_set_credentials(rd_kafka_t *rk,
3491  const char *username,
3492  const char *password);
3493 
3509 RD_EXPORT
3510 rd_kafka_queue_t *rd_kafka_queue_get_consumer(rd_kafka_t *rk);
3511 
3522 RD_EXPORT
3523 rd_kafka_queue_t *rd_kafka_queue_get_partition(rd_kafka_t *rk,
3524  const char *topic,
3525  int32_t partition);
3526 
3549 RD_EXPORT
3550 rd_kafka_queue_t *rd_kafka_queue_get_background(rd_kafka_t *rk);
3551 
3552 
3563 RD_EXPORT
3564 void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
3565 
3584 RD_EXPORT
3586  rd_kafka_queue_t *rkqu);
3587 
3588 
3592 RD_EXPORT
3593 size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu);
3594 
3595 
3613 RD_EXPORT
3614 void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu,
3615  int fd,
3616  const void *payload,
3617  size_t size);
3618 
3635 RD_EXPORT
3636 void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu,
3637  void (*event_cb)(rd_kafka_t *rk,
3638  void *qev_opaque),
3639  void *qev_opaque);
3640 
3641 
3649 RD_EXPORT
3650 void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu);
3651 
3652 
3663 #define RD_KAFKA_OFFSET_BEGINNING \
3664  -2
3666 #define RD_KAFKA_OFFSET_END \
3667  -1
3669 #define RD_KAFKA_OFFSET_STORED \
3670  -1000
3672 #define RD_KAFKA_OFFSET_INVALID -1001
3676 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
3677 
3684 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
3685 
3719 RD_EXPORT
3720 int rd_kafka_consume_start(rd_kafka_topic_t *rkt,
3721  int32_t partition,
3722  int64_t offset);
3723 
3738 RD_EXPORT
3739 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt,
3740  int32_t partition,
3741  int64_t offset,
3742  rd_kafka_queue_t *rkqu);
3743 
3757 RD_EXPORT
3758 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
3759 
3760 
3761 
3790 RD_EXPORT
3791 rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt,
3792  int32_t partition,
3793  int64_t offset,
3794  int timeout_ms);
3795 
3796 
3797 
3829 RD_EXPORT rd_kafka_error_t *
3830 rd_kafka_seek_partitions(rd_kafka_t *rk,
3831  rd_kafka_topic_partition_list_t *partitions,
3832  int timeout_ms);
3833 
3834 
3859 RD_EXPORT
3860 rd_kafka_message_t *
3861 rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms);
3862 
3863 
3864 
3890 RD_EXPORT
3891 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt,
3892  int32_t partition,
3893  int timeout_ms,
3894  rd_kafka_message_t **rkmessages,
3895  size_t rkmessages_size);
3896 
3897 
3898 
3932 RD_EXPORT
3933 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt,
3934  int32_t partition,
3935  int timeout_ms,
3936  void (*consume_cb)(rd_kafka_message_t *rkmessage,
3937  void *commit_opaque),
3938  void *commit_opaque);
3939 
3940 
3959 RD_EXPORT
3960 rd_kafka_message_t *rd_kafka_consume_queue(rd_kafka_queue_t *rkqu,
3961  int timeout_ms);
3962 
3968 RD_EXPORT
3969 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
3970  int timeout_ms,
3971  rd_kafka_message_t **rkmessages,
3972  size_t rkmessages_size);
3973 
3983 RD_EXPORT
3985  rd_kafka_queue_t *rkqu,
3986  int timeout_ms,
3987  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque),
3988  void *commit_opaque);
3989 
3990 
4030 RD_EXPORT
4032 rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset);
4033 
4034 
4067 RD_EXPORT rd_kafka_resp_err_t
4068 rd_kafka_offsets_store(rd_kafka_t *rk,
4069  rd_kafka_topic_partition_list_t *offsets);
4070 
4071 
4092 RD_EXPORT
4093 rd_kafka_error_t *rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage);
4094 
4148 RD_EXPORT rd_kafka_resp_err_t
4149 rd_kafka_subscribe(rd_kafka_t *rk,
4150  const rd_kafka_topic_partition_list_t *topics);
4151 
4152 
4156 RD_EXPORT
4158 
4159 
4169 RD_EXPORT rd_kafka_resp_err_t
4170 rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics);
4171 
4172 
4173 
4203 RD_EXPORT
4204 rd_kafka_message_t *rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms);
4205 
4223 RD_EXPORT
4225 
4226 
4244 RD_EXPORT
4245 rd_kafka_error_t *rd_kafka_consumer_close_queue(rd_kafka_t *rk,
4246  rd_kafka_queue_t *rkqu);
4247 
4248 
4257 RD_EXPORT
4258 int rd_kafka_consumer_closed(rd_kafka_t *rk);
4259 
4260 
4278 RD_EXPORT rd_kafka_error_t *
4279 rd_kafka_incremental_assign(rd_kafka_t *rk,
4280  const rd_kafka_topic_partition_list_t *partitions);
4281 
4282 
4300 RD_EXPORT rd_kafka_error_t *rd_kafka_incremental_unassign(
4301  rd_kafka_t *rk,
4302  const rd_kafka_topic_partition_list_t *partitions);
4303 
4304 
4318 RD_EXPORT
4319 const char *rd_kafka_rebalance_protocol(rd_kafka_t *rk);
4320 
4321 
4342 RD_EXPORT rd_kafka_resp_err_t
4343 rd_kafka_assign(rd_kafka_t *rk,
4344  const rd_kafka_topic_partition_list_t *partitions);
4345 
4362 RD_EXPORT rd_kafka_resp_err_t
4363 rd_kafka_assignment(rd_kafka_t *rk,
4364  rd_kafka_topic_partition_list_t **partitions);
4365 
4366 
4383 RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk);
4384 
4385 
4406 RD_EXPORT rd_kafka_resp_err_t
4407 rd_kafka_commit(rd_kafka_t *rk,
4408  const rd_kafka_topic_partition_list_t *offsets,
4409  int async);
4410 
4411 
4418 RD_EXPORT rd_kafka_resp_err_t
4419 rd_kafka_commit_message(rd_kafka_t *rk,
4420  const rd_kafka_message_t *rkmessage,
4421  int async);
4422 
4423 
4446 RD_EXPORT rd_kafka_resp_err_t
4447 rd_kafka_commit_queue(rd_kafka_t *rk,
4448  const rd_kafka_topic_partition_list_t *offsets,
4449  rd_kafka_queue_t *rkqu,
4450  void (*cb)(rd_kafka_t *rk,
4451  rd_kafka_resp_err_t err,
4452  rd_kafka_topic_partition_list_t *offsets,
4453  void *commit_opaque),
4454  void *commit_opaque);
4455 
4456 
4474 RD_EXPORT rd_kafka_resp_err_t
4475 rd_kafka_committed(rd_kafka_t *rk,
4476  rd_kafka_topic_partition_list_t *partitions,
4477  int timeout_ms);
4478 
4479 
4480 
4497 RD_EXPORT rd_kafka_resp_err_t
4498 rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions);
4499 
4500 
4501 
4513 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4514 rd_kafka_consumer_group_metadata(rd_kafka_t *rk);
4515 
4516 
4526 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4527 rd_kafka_consumer_group_metadata_new(const char *group_id);
4528 
4529 
4542 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4544  int32_t generation_id,
4545  const char *member_id,
4546  const char *group_instance_id);
4547 
4548 
4553 RD_EXPORT void
4554 rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *);
4555 
4556 
4577 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_write(
4578  const rd_kafka_consumer_group_metadata_t *cgmd,
4579  void **bufferp,
4580  size_t *sizep);
4581 
4602 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_read(
4603  rd_kafka_consumer_group_metadata_t **cgmdp,
4604  const void *buffer,
4605  size_t size);
4606 
4622 #define RD_KAFKA_MSG_F_FREE \
4623  0x1
4625 #define RD_KAFKA_MSG_F_COPY \
4626  0x2
4628 #define RD_KAFKA_MSG_F_BLOCK \
4629  0x4
4640 #define RD_KAFKA_MSG_F_PARTITION \
4641  0x8
4746 RD_EXPORT
4747 int rd_kafka_produce(rd_kafka_topic_t *rkt,
4748  int32_t partition,
4749  int msgflags,
4750  void *payload,
4751  size_t len,
4752  const void *key,
4753  size_t keylen,
4754  void *msg_opaque);
4755 
4756 
4770 RD_EXPORT
4771 rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk, ...);
4772 
4773 
4785 RD_EXPORT
4786 rd_kafka_error_t *
4787 rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt);
4788 
4789 
4819 RD_EXPORT
4820 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt,
4821  int32_t partition,
4822  int msgflags,
4823  rd_kafka_message_t *rkmessages,
4824  int message_cnt);
4825 
4826 
4827 
4852 RD_EXPORT
4853 rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms);
4854 
4855 
4856 
4887 RD_EXPORT
4888 rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags);
4889 
4890 
4898 #define RD_KAFKA_PURGE_F_QUEUE 0x1
4899 
4907 #define RD_KAFKA_PURGE_F_INFLIGHT 0x2
4908 
4909 
4913 #define RD_KAFKA_PURGE_F_NON_BLOCKING 0x4
4914 
4915 
4931  int32_t id;
4932  char *host;
4933  int port;
4934 } rd_kafka_metadata_broker_t;
4935 
4940  int32_t id;
4942  int32_t leader;
4944  int32_t *replicas;
4945  int isr_cnt;
4946  int32_t *isrs;
4947 } rd_kafka_metadata_partition_t;
4948 
4952 typedef struct rd_kafka_metadata_topic {
4953  char *topic;
4957 } rd_kafka_metadata_topic_t;
4958 
4959 
4963 typedef struct rd_kafka_metadata {
4970  int32_t orig_broker_id;
4972 } rd_kafka_metadata_t;
4973 
4994 RD_EXPORT
4996 rd_kafka_metadata(rd_kafka_t *rk,
4997  int all_topics,
4998  rd_kafka_topic_t *only_rkt,
4999  const struct rd_kafka_metadata **metadatap,
5000  int timeout_ms);
5001 
5005 RD_EXPORT
5006 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
5007 
5011 typedef struct rd_kafka_Node_s rd_kafka_Node_t;
5012 
5020 RD_EXPORT
5021 int rd_kafka_Node_id(const rd_kafka_Node_t *node);
5022 
5033 RD_EXPORT
5034 const char *rd_kafka_Node_host(const rd_kafka_Node_t *node);
5035 
5043 RD_EXPORT
5044 uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node);
5045 
5053 RD_EXPORT
5054 const char *rd_kafka_Node_rack(const rd_kafka_Node_t *node);
5055 
5076  char *member_id;
5077  char *client_id;
5078  char *client_host;
5085 };
5086 
5092 typedef enum {
5093  RD_KAFKA_CONSUMER_GROUP_STATE_UNKNOWN = 0,
5094  RD_KAFKA_CONSUMER_GROUP_STATE_PREPARING_REBALANCE = 1,
5095  RD_KAFKA_CONSUMER_GROUP_STATE_COMPLETING_REBALANCE = 2,
5096  RD_KAFKA_CONSUMER_GROUP_STATE_STABLE = 3,
5097  RD_KAFKA_CONSUMER_GROUP_STATE_DEAD = 4,
5098  RD_KAFKA_CONSUMER_GROUP_STATE_EMPTY = 5,
5099  RD_KAFKA_CONSUMER_GROUP_STATE__CNT
5101 
5107  char *group;
5109  char *state;
5111  char *protocol;
5114 };
5115 
5124 };
5125 
5126 
5157 RD_EXPORT
5159 rd_kafka_list_groups(rd_kafka_t *rk,
5160  const char *group,
5161  const struct rd_kafka_group_list **grplistp,
5162  int timeout_ms);
5163 
5171 RD_EXPORT
5172 const char *
5174 
5182 RD_EXPORT
5184 rd_kafka_consumer_group_state_code(const char *name);
5185 
5189 RD_EXPORT
5190 void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist);
5191 
5192 
5236 RD_EXPORT
5237 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
5238 
5239 
5240 
5253 RD_EXPORT RD_DEPRECATED void
5254 rd_kafka_set_logger(rd_kafka_t *rk,
5255  void (*func)(const rd_kafka_t *rk,
5256  int level,
5257  const char *fac,
5258  const char *buf));
5259 
5260 
5270 RD_EXPORT
5271 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
5272 
5273 
5277 RD_EXPORT
5278 void rd_kafka_log_print(const rd_kafka_t *rk,
5279  int level,
5280  const char *fac,
5281  const char *buf);
5282 
5283 
5289 RD_EXPORT
5290 void rd_kafka_log_syslog(const rd_kafka_t *rk,
5291  int level,
5292  const char *fac,
5293  const char *buf);
5294 
5295 
5318 RD_EXPORT
5319 int rd_kafka_outq_len(rd_kafka_t *rk);
5320 
5321 
5322 
5329 RD_EXPORT
5330 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
5331 
5332 
5333 
5339 RD_EXPORT
5340 int rd_kafka_thread_cnt(void);
5341 
5342 
5355 
5356 
5365 RD_EXPORT
5366 int rd_kafka_wait_destroyed(int timeout_ms);
5367 
5368 
5374 RD_EXPORT
5375 int rd_kafka_unittest(void);
5376 
5377 
5394 RD_EXPORT
5396 
5397 
5414 #define RD_KAFKA_EVENT_NONE 0x0
5415 #define RD_KAFKA_EVENT_DR 0x1
5416 #define RD_KAFKA_EVENT_FETCH 0x2
5417 #define RD_KAFKA_EVENT_LOG 0x4
5418 #define RD_KAFKA_EVENT_ERROR 0x8
5419 #define RD_KAFKA_EVENT_REBALANCE 0x10
5420 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
5421 #define RD_KAFKA_EVENT_STATS 0x40
5422 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
5423 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
5424 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT \
5425  102
5426 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
5427 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT \
5428  104
5429 #define RD_KAFKA_EVENT_DELETERECORDS_RESULT 105
5430 #define RD_KAFKA_EVENT_DELETEGROUPS_RESULT 106
5432 #define RD_KAFKA_EVENT_DELETECONSUMERGROUPOFFSETS_RESULT 107
5433 
5434 #define RD_KAFKA_EVENT_OAUTHBEARER_TOKEN_REFRESH 0x100
5435 #define RD_KAFKA_EVENT_BACKGROUND 0x200
5436 #define RD_KAFKA_EVENT_CREATEACLS_RESULT 0x400
5437 #define RD_KAFKA_EVENT_DESCRIBEACLS_RESULT 0x800
5438 #define RD_KAFKA_EVENT_DELETEACLS_RESULT 0x1000
5440 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPS_RESULT 0x2000
5441 
5442 #define RD_KAFKA_EVENT_DESCRIBECONSUMERGROUPS_RESULT 0x4000
5443 
5444 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPOFFSETS_RESULT 0x8000
5445 
5446 #define RD_KAFKA_EVENT_ALTERCONSUMERGROUPOFFSETS_RESULT 0x10000
5447 
5448 #define RD_KAFKA_EVENT_INCREMENTALALTERCONFIGS_RESULT 0x20000
5449 
5450 #define RD_KAFKA_EVENT_DESCRIBEUSERSCRAMCREDENTIALS_RESULT 0x40000
5451 
5452 #define RD_KAFKA_EVENT_ALTERUSERSCRAMCREDENTIALS_RESULT 0x80000
5453 
5454 #define RD_KAFKA_EVENT_DESCRIBETOPICS_RESULT 0x100000
5455 
5456 #define RD_KAFKA_EVENT_DESCRIBECLUSTER_RESULT 0x200000
5457 
5458 #define RD_KAFKA_EVENT_LISTOFFSETS_RESULT 0x400000
5459 
5466 RD_EXPORT
5467 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev);
5468 
5475 RD_EXPORT
5476 const char *rd_kafka_event_name(const rd_kafka_event_t *rkev);
5477 
5478 
5488 RD_EXPORT
5489 void rd_kafka_event_destroy(rd_kafka_event_t *rkev);
5490 
5491 
5507 RD_EXPORT
5508 const rd_kafka_message_t *rd_kafka_event_message_next(rd_kafka_event_t *rkev);
5509 
5510 
5524 RD_EXPORT
5525 size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev,
5526  const rd_kafka_message_t **rkmessages,
5527  size_t size);
5528 
5529 
5537 RD_EXPORT
5538 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev);
5539 
5540 
5552 RD_EXPORT
5553 const char *rd_kafka_event_config_string(rd_kafka_event_t *rkev);
5554 
5555 
5564 RD_EXPORT
5565 rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev);
5566 
5567 
5576 RD_EXPORT
5577 const char *rd_kafka_event_error_string(rd_kafka_event_t *rkev);
5578 
5579 
5588 RD_EXPORT
5589 int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev);
5590 
5591 
5618 RD_EXPORT
5619 void *rd_kafka_event_opaque(rd_kafka_event_t *rkev);
5620 
5621 
5630 RD_EXPORT
5631 int rd_kafka_event_log(rd_kafka_event_t *rkev,
5632  const char **fac,
5633  const char **str,
5634  int *level);
5635 
5636 
5648 RD_EXPORT
5649 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev,
5650  char *dst,
5651  size_t dstsize);
5652 
5653 
5666 RD_EXPORT
5667 const char *rd_kafka_event_stats(rd_kafka_event_t *rkev);
5668 
5669 
5680 RD_EXPORT rd_kafka_topic_partition_list_t *
5681 rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev);
5682 
5683 
5694 RD_EXPORT rd_kafka_topic_partition_t *
5695 rd_kafka_event_topic_partition(rd_kafka_event_t *rkev);
5696 
5697 
5699 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
5701 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
5703 typedef rd_kafka_event_t rd_kafka_CreateAcls_result_t;
5705 typedef rd_kafka_event_t rd_kafka_DescribeAcls_result_t;
5707 typedef rd_kafka_event_t rd_kafka_DeleteAcls_result_t;
5709 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
5711 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
5715 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
5717 typedef rd_kafka_event_t rd_kafka_DeleteRecords_result_t;
5719 typedef rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t;
5723 typedef rd_kafka_event_t rd_kafka_DeleteGroups_result_t;
5731 typedef rd_kafka_event_t rd_kafka_DescribeTopics_result_t;
5733 typedef rd_kafka_event_t rd_kafka_DescribeCluster_result_t;
5739 typedef rd_kafka_event_t rd_kafka_ListOffsets_result_t;
5740 
5750 RD_EXPORT const rd_kafka_CreateTopics_result_t *
5751 rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev);
5752 
5762 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
5763 rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev);
5764 
5774 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
5775 rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev);
5776 
5786 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
5787 rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev);
5788 
5799 rd_kafka_event_IncrementalAlterConfigs_result(rd_kafka_event_t *rkev);
5800 
5810 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
5811 rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev);
5812 
5820 RD_EXPORT const rd_kafka_DeleteRecords_result_t *
5821 rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev);
5822 
5835 RD_EXPORT const rd_kafka_ListConsumerGroups_result_t *
5836 rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev);
5837 
5851 rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev);
5852 
5865 RD_EXPORT const rd_kafka_DescribeTopics_result_t *
5866 rd_kafka_event_DescribeTopics_result(rd_kafka_event_t *rkev);
5867 
5880 RD_EXPORT const rd_kafka_DescribeCluster_result_t *
5881 rd_kafka_event_DescribeCluster_result(rd_kafka_event_t *rkev);
5891 RD_EXPORT const rd_kafka_DeleteGroups_result_t *
5892 rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev);
5893 
5904 rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
5905 
5913 RD_EXPORT const rd_kafka_CreateAcls_result_t *
5914 rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev);
5915 
5923 RD_EXPORT const rd_kafka_DescribeAcls_result_t *
5924 rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev);
5925 
5933 RD_EXPORT const rd_kafka_DeleteAcls_result_t *
5934 rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev);
5935 
5949 rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
5950 
5964 rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
5965 
5978 RD_EXPORT const rd_kafka_ListOffsets_result_t *
5979 rd_kafka_event_ListOffsets_result(rd_kafka_event_t *rkev);
5980 
5981 
5996 
6010 rd_kafka_event_AlterUserScramCredentials_result(rd_kafka_event_t *rkev);
6011 
6021 RD_EXPORT
6022 rd_kafka_event_t *rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms);
6023 
6038 RD_EXPORT
6039 int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms);
6040 
6041 
6086  rd_kafka_conf_t *conf,
6087  void **plug_opaquep,
6088  char *errstr,
6089  size_t errstr_size);
6090 
6172  rd_kafka_conf_t *conf,
6173  const char *name,
6174  const char *val,
6175  char *errstr,
6176  size_t errstr_size,
6177  void *ic_opaque);
6178 
6179 
6202  rd_kafka_conf_t *new_conf,
6203  const rd_kafka_conf_t *old_conf,
6204  size_t filter_cnt,
6205  const char **filter,
6206  void *ic_opaque);
6207 
6208 
6216  void *ic_opaque);
6217 
6218 
6237  rd_kafka_t *rk,
6238  const rd_kafka_conf_t *conf,
6239  void *ic_opaque,
6240  char *errstr,
6241  size_t errstr_size);
6242 
6243 
6251 typedef rd_kafka_resp_err_t(
6252  rd_kafka_interceptor_f_on_destroy_t)(rd_kafka_t *rk, void *ic_opaque);
6253 
6254 
6255 
6277  rd_kafka_t *rk,
6278  rd_kafka_message_t *rkmessage,
6279  void *ic_opaque);
6280 
6304  rd_kafka_t *rk,
6305  rd_kafka_message_t *rkmessage,
6306  void *ic_opaque);
6307 
6308 
6326  rd_kafka_t *rk,
6327  rd_kafka_message_t *rkmessage,
6328  void *ic_opaque);
6329 
6352  rd_kafka_t *rk,
6353  const rd_kafka_topic_partition_list_t *offsets,
6355  void *ic_opaque);
6356 
6357 
6380  rd_kafka_t *rk,
6381  int sockfd,
6382  const char *brokername,
6383  int32_t brokerid,
6384  int16_t ApiKey,
6385  int16_t ApiVersion,
6386  int32_t CorrId,
6387  size_t size,
6388  void *ic_opaque);
6389 
6390 
6417  rd_kafka_t *rk,
6418  int sockfd,
6419  const char *brokername,
6420  int32_t brokerid,
6421  int16_t ApiKey,
6422  int16_t ApiVersion,
6423  int32_t CorrId,
6424  size_t size,
6425  int64_t rtt,
6427  void *ic_opaque);
6428 
6429 
6447  rd_kafka_t *rk,
6448  rd_kafka_thread_type_t thread_type,
6449  const char *thread_name,
6450  void *ic_opaque);
6451 
6452 
6473  rd_kafka_t *rk,
6474  rd_kafka_thread_type_t thread_type,
6475  const char *thread_name,
6476  void *ic_opaque);
6477 
6478 
6494  rd_kafka_t *rk,
6495  int32_t broker_id,
6496  const char *secproto,
6497  const char *name,
6498  int port,
6499  const char *state,
6500  void *ic_opaque);
6501 
6502 
6516  rd_kafka_conf_t *conf,
6517  const char *ic_name,
6519  void *ic_opaque);
6520 
6521 
6535  rd_kafka_conf_t *conf,
6536  const char *ic_name,
6538  void *ic_opaque);
6539 
6554  rd_kafka_conf_t *conf,
6555  const char *ic_name,
6557  void *ic_opaque);
6558 
6559 
6581 RD_EXPORT rd_kafka_resp_err_t
6582 rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf,
6583  const char *ic_name,
6585  void *ic_opaque);
6586 
6587 
6588 
6602  rd_kafka_t *rk,
6603  const char *ic_name,
6605  void *ic_opaque);
6606 
6607 
6620 RD_EXPORT rd_kafka_resp_err_t
6621 rd_kafka_interceptor_add_on_send(rd_kafka_t *rk,
6622  const char *ic_name,
6624  void *ic_opaque);
6625 
6639  rd_kafka_t *rk,
6640  const char *ic_name,
6641  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
6642  void *ic_opaque);
6643 
6644 
6658  rd_kafka_t *rk,
6659  const char *ic_name,
6661  void *ic_opaque);
6662 
6663 
6677  rd_kafka_t *rk,
6678  const char *ic_name,
6680  void *ic_opaque);
6681 
6682 
6696  rd_kafka_t *rk,
6697  const char *ic_name,
6699  void *ic_opaque);
6700 
6701 
6715  rd_kafka_t *rk,
6716  const char *ic_name,
6717  rd_kafka_interceptor_f_on_response_received_t *on_response_received,
6718  void *ic_opaque);
6719 
6720 
6734  rd_kafka_t *rk,
6735  const char *ic_name,
6737  void *ic_opaque);
6738 
6739 
6753  rd_kafka_t *rk,
6754  const char *ic_name,
6756  void *ic_opaque);
6757 
6758 
6771 RD_EXPORT
6773  rd_kafka_t *rk,
6774  const char *ic_name,
6775  rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change,
6776  void *ic_opaque);
6777 
6778 
6779 
6800 RD_EXPORT rd_kafka_resp_err_t
6801 rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres);
6802 
6809 RD_EXPORT const char *
6810 rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres);
6811 
6817 RD_EXPORT const char *
6818 rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres);
6819 
6829 RD_EXPORT const rd_kafka_error_t *
6830 rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres);
6831 
6837 RD_EXPORT const char *
6838 rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres);
6839 
6845 RD_EXPORT const rd_kafka_topic_partition_list_t *
6846 rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres);
6847 
6848 
6895 typedef enum rd_kafka_admin_op_t {
6926 
6939 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
6940 
6949  RD_KAFKA_ISOLATION_LEVEL_READ_UNCOMMITTED = 0,
6950  RD_KAFKA_ISOLATION_LEVEL_READ_COMMITTED = 1
6952 
6972 RD_EXPORT rd_kafka_AdminOptions_t *
6973 rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api);
6974 
6975 
6980 
6981 
6999 RD_EXPORT rd_kafka_resp_err_t
7001  int timeout_ms,
7002  char *errstr,
7003  size_t errstr_size);
7004 
7005 
7032 RD_EXPORT rd_kafka_resp_err_t
7034  int timeout_ms,
7035  char *errstr,
7036  size_t errstr_size);
7037 
7038 
7057 RD_EXPORT rd_kafka_resp_err_t
7059  int true_or_false,
7060  char *errstr,
7061  size_t errstr_size);
7062 
7063 
7091 RD_EXPORT rd_kafka_resp_err_t
7093  int32_t broker_id,
7094  char *errstr,
7095  size_t errstr_size);
7096 
7097 
7110 RD_EXPORT
7112  rd_kafka_AdminOptions_t *options,
7113  int true_or_false);
7114 
7129 RD_EXPORT
7131  rd_kafka_AdminOptions_t *options,
7132  int true_or_false);
7133 
7146 RD_EXPORT
7148  rd_kafka_AdminOptions_t *options,
7149  const rd_kafka_consumer_group_state_t *consumer_group_states,
7150  size_t consumer_group_states_cnt);
7151 
7155 RD_EXPORT
7156 rd_kafka_error_t *
7159 
7164 RD_EXPORT void
7166  void *ev_opaque);
7167 
7168 
7169 
7178  1,
7187  9,
7189  10,
7191  11,
7193  12,
7194  RD_KAFKA_ACL_OPERATION__CNT
7196 
7208 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
7209 
7231 RD_EXPORT rd_kafka_NewTopic_t *rd_kafka_NewTopic_new(const char *topic,
7232  int num_partitions,
7233  int replication_factor,
7234  char *errstr,
7235  size_t errstr_size);
7236 
7241 RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic);
7242 
7243 
7249 RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics,
7250  size_t new_topic_cnt);
7251 
7252 
7274 RD_EXPORT rd_kafka_resp_err_t
7276  int32_t partition,
7277  int32_t *broker_ids,
7278  size_t broker_id_cnt,
7279  char *errstr,
7280  size_t errstr_size);
7281 
7294 RD_EXPORT rd_kafka_resp_err_t
7296  const char *name,
7297  const char *value);
7298 
7299 
7318 RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk,
7319  rd_kafka_NewTopic_t **new_topics,
7320  size_t new_topic_cnt,
7321  const rd_kafka_AdminOptions_t *options,
7322  rd_kafka_queue_t *rkqu);
7323 
7324 
7325 /*
7326  * CreateTopics result type and methods
7327  */
7328 
7337 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_CreateTopics_result_topics(
7338  const rd_kafka_CreateTopics_result_t *result,
7339  size_t *cntp);
7340 
7341 
7342 
7343 /*
7344  * DeleteTopics - delete topics from cluster
7345  *
7346  */
7347 
7349 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
7350 
7360 RD_EXPORT rd_kafka_DeleteTopic_t *rd_kafka_DeleteTopic_new(const char *topic);
7361 
7366 RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic);
7367 
7373 RD_EXPORT void
7375  size_t del_topic_cnt);
7376 
7390 RD_EXPORT
7391 void rd_kafka_DeleteTopics(rd_kafka_t *rk,
7392  rd_kafka_DeleteTopic_t **del_topics,
7393  size_t del_topic_cnt,
7394  const rd_kafka_AdminOptions_t *options,
7395  rd_kafka_queue_t *rkqu);
7396 
7397 
7398 
7399 /*
7400  * DeleteTopics result type and methods
7401  */
7402 
7411 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_DeleteTopics_result_topics(
7412  const rd_kafka_DeleteTopics_result_t *result,
7413  size_t *cntp);
7414 
7415 
7426 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
7427 
7444 RD_EXPORT rd_kafka_NewPartitions_t *
7445 rd_kafka_NewPartitions_new(const char *topic,
7446  size_t new_total_cnt,
7447  char *errstr,
7448  size_t errstr_size);
7449 
7454 RD_EXPORT void
7456 
7462 RD_EXPORT void
7464  size_t new_parts_cnt);
7465 
7489  rd_kafka_NewPartitions_t *new_parts,
7490  int32_t new_partition_idx,
7491  int32_t *broker_ids,
7492  size_t broker_id_cnt,
7493  char *errstr,
7494  size_t errstr_size);
7495 
7496 
7515 RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk,
7516  rd_kafka_NewPartitions_t **new_parts,
7517  size_t new_parts_cnt,
7518  const rd_kafka_AdminOptions_t *options,
7519  rd_kafka_queue_t *rkqu);
7520 
7521 
7522 
7523 /*
7524  * CreatePartitions result type and methods
7525  */
7526 
7535 RD_EXPORT const rd_kafka_topic_result_t **
7537  const rd_kafka_CreatePartitions_result_t *result,
7538  size_t *cntp);
7539 
7575 
7579 
7580 
7584 RD_EXPORT const char *
7586 
7587 
7589 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
7590 
7594 RD_EXPORT const char *
7596 
7601 RD_EXPORT const char *
7603 
7607 RD_EXPORT rd_kafka_ConfigSource_t
7609 
7614 RD_EXPORT int
7616 
7622 RD_EXPORT int
7624 
7632 RD_EXPORT int
7634 
7638 RD_EXPORT int
7640 
7641 
7652 RD_EXPORT const rd_kafka_ConfigEntry_t **
7654  size_t *cntp);
7655 
7656 
7657 
7670 
7686  RD_KAFKA_RESOURCE_PATTERN_TYPE__CNT,
7688 
7694  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SET = 0,
7695  RD_KAFKA_ALTER_CONFIG_OP_TYPE_DELETE = 1,
7696  RD_KAFKA_ALTER_CONFIG_OP_TYPE_APPEND = 2,
7697  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SUBTRACT = 3,
7698  RD_KAFKA_ALTER_CONFIG_OP_TYPE__CNT,
7700 
7704 RD_EXPORT const char *rd_kafka_ResourcePatternType_name(
7705  rd_kafka_ResourcePatternType_t resource_pattern_type);
7706 
7710 RD_EXPORT const char *
7712 
7714 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
7715 
7716 
7725 RD_EXPORT rd_kafka_ConfigResource_t *
7727  const char *resname);
7728 
7733 RD_EXPORT void
7735 
7736 
7742 RD_EXPORT void
7744  size_t config_cnt);
7745 
7746 
7760 RD_EXPORT rd_kafka_resp_err_t
7762  const char *name,
7763  const char *value);
7764 
7765 
7784 RD_EXPORT rd_kafka_error_t *rd_kafka_ConfigResource_add_incremental_config(
7785  rd_kafka_ConfigResource_t *config,
7786  const char *name,
7788  const char *value);
7789 
7790 
7799 RD_EXPORT const rd_kafka_ConfigEntry_t **
7801  size_t *cntp);
7802 
7803 
7804 
7808 RD_EXPORT rd_kafka_ResourceType_t
7810 
7814 RD_EXPORT const char *
7816 
7820 RD_EXPORT rd_kafka_resp_err_t
7822 
7827 RD_EXPORT const char *
7829 
7830 
7831 /*
7832  * AlterConfigs - alter cluster configuration.
7833  *
7834  */
7835 
7836 
7859 RD_EXPORT
7860 void rd_kafka_AlterConfigs(rd_kafka_t *rk,
7861  rd_kafka_ConfigResource_t **configs,
7862  size_t config_cnt,
7863  const rd_kafka_AdminOptions_t *options,
7864  rd_kafka_queue_t *rkqu);
7865 
7866 
7867 /*
7868  * AlterConfigs result type and methods
7869  */
7870 
7885 RD_EXPORT const rd_kafka_ConfigResource_t **
7887  const rd_kafka_AlterConfigs_result_t *result,
7888  size_t *cntp);
7889 
7890 
7891 
7892 /*
7893  * IncrementalAlterConfigs - alter cluster configuration incrementally.
7894  *
7895  */
7896 
7897 
7918 RD_EXPORT
7919 void rd_kafka_IncrementalAlterConfigs(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  * IncrementalAlterConfigs result type and methods
7928  */
7929 
7945 RD_EXPORT const rd_kafka_ConfigResource_t **
7948  size_t *cntp);
7949 
7950 
7951 
7952 /*
7953  * DescribeConfigs - retrieve cluster configuration.
7954  *
7955  */
7956 
7957 
7982 RD_EXPORT
7983 void rd_kafka_DescribeConfigs(rd_kafka_t *rk,
7984  rd_kafka_ConfigResource_t **configs,
7985  size_t config_cnt,
7986  const rd_kafka_AdminOptions_t *options,
7987  rd_kafka_queue_t *rkqu);
7988 
7989 
7990 
7991 /*
7992  * DescribeConfigs result type and methods
7993  */
7994 
8003 RD_EXPORT const rd_kafka_ConfigResource_t **
8005  const rd_kafka_DescribeConfigs_result_t *result,
8006  size_t *cntp);
8007 
8008 
8019 typedef struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t;
8020 
8038  const rd_kafka_topic_partition_list_t *before_offsets);
8039 
8044 RD_EXPORT void
8046 
8052 RD_EXPORT void
8054  size_t del_record_cnt);
8055 
8077 RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk,
8078  rd_kafka_DeleteRecords_t **del_records,
8079  size_t del_record_cnt,
8080  const rd_kafka_AdminOptions_t *options,
8081  rd_kafka_queue_t *rkqu);
8082 
8083 
8084 /*
8085  * DeleteRecords result type and methods
8086  */
8087 
8097 RD_EXPORT const rd_kafka_topic_partition_list_t *
8099  const rd_kafka_DeleteRecords_result_t *result);
8100 
8112 typedef struct rd_kafka_TopicCollection_s rd_kafka_TopicCollection_t;
8113 
8118 typedef struct rd_kafka_TopicPartitionInfo_s rd_kafka_TopicPartitionInfo_t;
8119 
8124 typedef struct rd_kafka_TopicDescription_s rd_kafka_TopicDescription_t;
8125 
8135 RD_EXPORT
8137 rd_kafka_TopicCollection_of_topic_names(const char **topics, size_t topics_cnt);
8138 
8143 RD_EXPORT void
8145 
8160 RD_EXPORT
8161 void rd_kafka_DescribeTopics(rd_kafka_t *rk,
8162  const rd_kafka_TopicCollection_t *topics,
8163  const rd_kafka_AdminOptions_t *options,
8164  rd_kafka_queue_t *rkqu);
8165 
8175 RD_EXPORT
8177  const rd_kafka_DescribeTopics_result_t *result,
8178  size_t *cntp);
8179 
8180 
8191 RD_EXPORT
8193  const rd_kafka_TopicDescription_t *topicdesc,
8194  size_t *cntp);
8195 
8196 
8204 RD_EXPORT
8206  const rd_kafka_TopicPartitionInfo_t *partition);
8207 
8208 
8219 RD_EXPORT
8221  const rd_kafka_TopicPartitionInfo_t *partition);
8222 
8234 RD_EXPORT
8235 const rd_kafka_Node_t **
8237  size_t *cntp);
8238 
8250 RD_EXPORT
8252  const rd_kafka_TopicPartitionInfo_t *partition,
8253  size_t *cntp);
8254 
8267 RD_EXPORT
8269  const rd_kafka_TopicDescription_t *topicdesc,
8270  size_t *cntp);
8271 
8282 RD_EXPORT
8283 const char *
8285 
8295 RD_EXPORT const rd_kafka_Uuid_t *rd_kafka_TopicDescription_topic_id(
8296  const rd_kafka_TopicDescription_t *topicdesc);
8297 
8305 RD_EXPORT
8307  const rd_kafka_TopicDescription_t *topicdesc);
8308 
8319 RD_EXPORT
8320 const rd_kafka_error_t *
8322 
8323 
8343 RD_EXPORT
8344 void rd_kafka_DescribeCluster(rd_kafka_t *rk,
8345  const rd_kafka_AdminOptions_t *options,
8346  rd_kafka_queue_t *rkqu);
8347 
8358 RD_EXPORT
8360  const rd_kafka_DescribeCluster_result_t *result,
8361  size_t *cntp);
8362 
8374 RD_EXPORT
8377  const rd_kafka_DescribeCluster_result_t *result,
8378  size_t *cntp);
8379 
8387 RD_EXPORT
8389  const rd_kafka_DescribeCluster_result_t *result);
8390 
8400 RD_EXPORT
8402  const rd_kafka_DescribeCluster_result_t *result);
8403 
8418 typedef struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t;
8419 
8421 typedef struct rd_kafka_ListConsumerGroupsResult_s
8423 
8434 RD_EXPORT
8435 void rd_kafka_ListConsumerGroups(rd_kafka_t *rk,
8436  const rd_kafka_AdminOptions_t *options,
8437  rd_kafka_queue_t *rkqu);
8438 
8449 RD_EXPORT
8451  const rd_kafka_ConsumerGroupListing_t *grplist);
8452 
8461 RD_EXPORT
8463  const rd_kafka_ConsumerGroupListing_t *grplist);
8464 
8472 RD_EXPORT
8474  const rd_kafka_ConsumerGroupListing_t *grplist);
8475 
8487 RD_EXPORT
8491  size_t *cntp);
8492 
8506 RD_EXPORT
8507 const rd_kafka_error_t **rd_kafka_ListConsumerGroups_result_errors(
8509  size_t *cntp);
8510 
8522 typedef struct rd_kafka_ConsumerGroupDescription_s
8524 
8529 typedef struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t;
8530 
8535 typedef struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t;
8536 
8552 RD_EXPORT
8553 void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk,
8554  const char **groups,
8555  size_t groups_cnt,
8556  const rd_kafka_AdminOptions_t *options,
8557  rd_kafka_queue_t *rkqu);
8558 
8570 RD_EXPORT
8574  size_t *cntp);
8575 
8576 
8587 RD_EXPORT
8589  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8590 
8601 RD_EXPORT
8602 const rd_kafka_error_t *rd_kafka_ConsumerGroupDescription_error(
8603  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8604 
8612 RD_EXPORT
8614  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8615 
8616 
8627 RD_EXPORT
8629  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8630 
8643 RD_EXPORT
8646  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8647  size_t *cntp);
8648 
8656 RD_EXPORT
8658  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8659 
8670 RD_EXPORT
8672  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8673 
8681 RD_EXPORT
8683  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8684 
8697 RD_EXPORT
8699  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8700  size_t idx);
8701 
8712 RD_EXPORT
8714  const rd_kafka_MemberDescription_t *member);
8715 
8726 RD_EXPORT
8728  const rd_kafka_MemberDescription_t *member);
8729 
8740 RD_EXPORT
8742  const rd_kafka_MemberDescription_t *member);
8743 
8754 RD_EXPORT
8755 const char *
8757 
8768 RD_EXPORT
8770  const rd_kafka_MemberDescription_t *member);
8771 
8782 RD_EXPORT
8783 const rd_kafka_topic_partition_list_t *rd_kafka_MemberAssignment_partitions(
8784  const rd_kafka_MemberAssignment_t *assignment);
8785 
8797 typedef struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t;
8798 
8808 RD_EXPORT
8810 
8815 RD_EXPORT
8817 
8823 RD_EXPORT void
8825  size_t del_group_cnt);
8826 
8842 RD_EXPORT
8843 void rd_kafka_DeleteGroups(rd_kafka_t *rk,
8844  rd_kafka_DeleteGroup_t **del_groups,
8845  size_t del_group_cnt,
8846  const rd_kafka_AdminOptions_t *options,
8847  rd_kafka_queue_t *rkqu);
8848 
8849 
8850 
8851 /*
8852  * DeleteGroups result type and methods
8853  */
8854 
8863 RD_EXPORT const rd_kafka_group_result_t **rd_kafka_DeleteGroups_result_groups(
8864  const rd_kafka_DeleteGroups_result_t *result,
8865  size_t *cntp);
8866 
8877 typedef struct rd_kafka_ListConsumerGroupOffsets_s
8879 
8894  const char *group_id,
8895  const rd_kafka_topic_partition_list_t *partitions);
8896 
8902  rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets);
8903 
8910  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8911  size_t list_grpoffset_cnt);
8912 
8930 RD_EXPORT
8932  rd_kafka_t *rk,
8933  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8934  size_t list_grpoffsets_cnt,
8935  const rd_kafka_AdminOptions_t *options,
8936  rd_kafka_queue_t *rkqu);
8937 
8938 
8939 
8940 /*
8941  * ListConsumerGroupOffsets result type and methods
8942  */
8943 
8955 RD_EXPORT const rd_kafka_group_result_t **
8958  size_t *cntp);
8959 
8960 
8961 
8972 typedef struct rd_kafka_AlterConsumerGroupOffsets_s
8974 
8989  const char *group_id,
8990  const rd_kafka_topic_partition_list_t *partitions);
8991 
8997  rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets);
8998 
9005  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9006  size_t alter_grpoffset_cnt);
9007 
9026 RD_EXPORT
9028  rd_kafka_t *rk,
9029  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9030  size_t alter_grpoffsets_cnt,
9031  const rd_kafka_AdminOptions_t *options,
9032  rd_kafka_queue_t *rkqu);
9033 
9034 
9035 
9036 /*
9037  * AlterConsumerGroupOffsets result type and methods
9038  */
9039 
9051 RD_EXPORT const rd_kafka_group_result_t **
9054  size_t *cntp);
9055 
9056 
9057 
9068 typedef struct rd_kafka_DeleteConsumerGroupOffsets_s
9070 
9085  const char *group,
9086  const rd_kafka_topic_partition_list_t *partitions);
9087 
9093  rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets);
9094 
9101  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9102  size_t del_grpoffset_cnt);
9103 
9122 RD_EXPORT
9124  rd_kafka_t *rk,
9125  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9126  size_t del_grpoffsets_cnt,
9127  const rd_kafka_AdminOptions_t *options,
9128  rd_kafka_queue_t *rkqu);
9129 
9130 
9131 
9132 /*
9133  * DeleteConsumerGroupOffsets result type and methods
9134  */
9135 
9144 RD_EXPORT const rd_kafka_group_result_t **
9147  size_t *cntp);
9148 
9162  /* Used to retrieve the offset with the largest timestamp of a partition
9163  * as message timestamps can be specified client side this may not match
9164  * the log end offset returned by SPEC_LATEST.
9165  */
9166  RD_KAFKA_OFFSET_SPEC_MAX_TIMESTAMP = -3,
9167  /* Used to retrieve the offset with the earliest timestamp of a
9168  partition. */
9169  RD_KAFKA_OFFSET_SPEC_EARLIEST = -2,
9170  /* Used to retrieve the offset with the latest timestamp of a partition.
9171  */
9172  RD_KAFKA_OFFSET_SPEC_LATEST = -1,
9174 
9179 typedef struct rd_kafka_ListOffsetsResultInfo_s
9181 
9185 RD_EXPORT
9186 const rd_kafka_topic_partition_t *
9188  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9189 
9193 RD_EXPORT
9195  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9196 
9201 RD_EXPORT
9204  size_t *cntp);
9205 
9230 RD_EXPORT
9231 void rd_kafka_ListOffsets(rd_kafka_t *rk,
9232  rd_kafka_topic_partition_list_t *topic_partitions,
9233  const rd_kafka_AdminOptions_t *options,
9234  rd_kafka_queue_t *rkqu);
9235 
9248  RD_KAFKA_SCRAM_MECHANISM_UNKNOWN = 0,
9249  RD_KAFKA_SCRAM_MECHANISM_SHA_256 = 1,
9250  RD_KAFKA_SCRAM_MECHANISM_SHA_512 = 2,
9251  RD_KAFKA_SCRAM_MECHANISM__CNT
9253 
9259 typedef struct rd_kafka_ScramCredentialInfo_s rd_kafka_ScramCredentialInfo_t;
9260 
9264 RD_EXPORT
9266  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9267 
9271 RD_EXPORT
9273  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9274 
9281 typedef struct rd_kafka_UserScramCredentialsDescription_s
9283 
9287 RD_EXPORT
9289  const rd_kafka_UserScramCredentialsDescription_t *description);
9290 
9294 RD_EXPORT
9295 const rd_kafka_error_t *rd_kafka_UserScramCredentialsDescription_error(
9296  const rd_kafka_UserScramCredentialsDescription_t *description);
9297 
9302 RD_EXPORT
9304  const rd_kafka_UserScramCredentialsDescription_t *description);
9305 
9310 RD_EXPORT
9313  const rd_kafka_UserScramCredentialsDescription_t *description,
9314  size_t idx);
9315 
9325 RD_EXPORT
9329  size_t *cntp);
9330 
9342 RD_EXPORT
9344  rd_kafka_t *rk,
9345  const char **users,
9346  size_t user_cnt,
9347  const rd_kafka_AdminOptions_t *options,
9348  rd_kafka_queue_t *rkqu);
9349 
9353 typedef struct rd_kafka_UserScramCredentialAlteration_s
9355 
9376 RD_EXPORT
9378 rd_kafka_UserScramCredentialUpsertion_new(const char *username,
9379  rd_kafka_ScramMechanism_t mechanism,
9380  int32_t iterations,
9381  const unsigned char *password,
9382  size_t password_size,
9383  const unsigned char *salt,
9384  size_t salt_size);
9385 
9395 RD_EXPORT
9397 rd_kafka_UserScramCredentialDeletion_new(const char *username,
9398  rd_kafka_ScramMechanism_t mechanism);
9399 
9400 
9404 RD_EXPORT
9407 
9411 RD_EXPORT
9414  size_t alteration_cnt);
9415 
9419 typedef struct rd_kafka_AlterUserScramCredentials_result_response_s
9421 
9426 RD_EXPORT
9429 
9434 RD_EXPORT
9435 const rd_kafka_error_t *
9438 
9447 RD_EXPORT
9451  size_t *cntp);
9452 
9466 RD_EXPORT
9468  rd_kafka_t *rk,
9470  size_t alteration_cnt,
9471  const rd_kafka_AdminOptions_t *options,
9472  rd_kafka_queue_t *rkqu);
9473 
9486 typedef struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t;
9487 
9493 
9497 RD_EXPORT const rd_kafka_error_t *
9498 rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres);
9499 
9500 
9504 RD_EXPORT const char *
9506 
9514  1,
9517  RD_KAFKA_ACL_PERMISSION_TYPE__CNT
9519 
9523 RD_EXPORT const char *rd_kafka_AclPermissionType_name(
9524  rd_kafka_AclPermissionType_t acl_permission_type);
9525 
9544 RD_EXPORT rd_kafka_AclBinding_t *
9546  const char *name,
9547  rd_kafka_ResourcePatternType_t resource_pattern_type,
9548  const char *principal,
9549  const char *host,
9550  rd_kafka_AclOperation_t operation,
9551  rd_kafka_AclPermissionType_t permission_type,
9552  char *errstr,
9553  size_t errstr_size);
9554 
9581  rd_kafka_ResourceType_t restype,
9582  const char *name,
9583  rd_kafka_ResourcePatternType_t resource_pattern_type,
9584  const char *principal,
9585  const char *host,
9586  rd_kafka_AclOperation_t operation,
9587  rd_kafka_AclPermissionType_t permission_type,
9588  char *errstr,
9589  size_t errstr_size);
9590 
9594 RD_EXPORT rd_kafka_ResourceType_t
9596 
9602 RD_EXPORT const char *
9604 
9610 RD_EXPORT const char *
9612 
9618 RD_EXPORT const char *
9620 
9624 RD_EXPORT rd_kafka_AclOperation_t
9626 
9632 
9638 
9642 RD_EXPORT const rd_kafka_error_t *
9644 
9645 
9650 RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding);
9651 
9652 
9658 RD_EXPORT void
9660  size_t acl_bindings_cnt);
9661 
9669 RD_EXPORT const rd_kafka_acl_result_t **
9671  size_t *cntp);
9672 
9689 RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk,
9690  rd_kafka_AclBinding_t **new_acls,
9691  size_t new_acls_cnt,
9692  const rd_kafka_AdminOptions_t *options,
9693  rd_kafka_queue_t *rkqu);
9694 
9708 RD_EXPORT const rd_kafka_AclBinding_t **
9710  size_t *cntp);
9711 
9726 RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk,
9727  rd_kafka_AclBindingFilter_t *acl_filter,
9728  const rd_kafka_AdminOptions_t *options,
9729  rd_kafka_queue_t *rkqu);
9730 
9737 typedef struct rd_kafka_DeleteAcls_result_response_s
9739 
9747 RD_EXPORT const rd_kafka_DeleteAcls_result_response_t **
9749  size_t *cntp);
9750 
9755 RD_EXPORT const rd_kafka_error_t *rd_kafka_DeleteAcls_result_response_error(
9756  const rd_kafka_DeleteAcls_result_response_t *result_response);
9757 
9758 
9765 RD_EXPORT const rd_kafka_AclBinding_t **
9767  const rd_kafka_DeleteAcls_result_response_t *result_response,
9768  size_t *matching_acls_cntp);
9769 
9786 RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk,
9787  rd_kafka_AclBindingFilter_t **del_acls,
9788  size_t del_acls_cnt,
9789  const rd_kafka_AdminOptions_t *options,
9790  rd_kafka_queue_t *rkqu);
9791 
9844 RD_EXPORT
9846 rd_kafka_oauthbearer_set_token(rd_kafka_t *rk,
9847  const char *token_value,
9848  int64_t md_lifetime_ms,
9849  const char *md_principal_name,
9850  const char **extensions,
9851  size_t extension_size,
9852  char *errstr,
9853  size_t errstr_size);
9854 
9875 RD_EXPORT
9877  const char *errstr);
9878 
10065 RD_EXPORT
10066 rd_kafka_error_t *rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms);
10067 
10068 
10069 
10114 RD_EXPORT
10115 rd_kafka_error_t *rd_kafka_begin_transaction(rd_kafka_t *rk);
10116 
10117 
10189 RD_EXPORT
10190 rd_kafka_error_t *rd_kafka_send_offsets_to_transaction(
10191  rd_kafka_t *rk,
10192  const rd_kafka_topic_partition_list_t *offsets,
10193  const rd_kafka_consumer_group_metadata_t *cgmetadata,
10194  int timeout_ms);
10195 
10196 
10264 RD_EXPORT
10265 rd_kafka_error_t *rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms);
10266 
10267 
10326 RD_EXPORT
10327 rd_kafka_error_t *rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms);
10328 
10329 
10332 /* @cond NO_DOC */
10333 #ifdef __cplusplus
10334 }
10335 #endif
10336 #endif /* _RDKAFKA_H_ */
10337 /* @endcond NO_DOC */
rd_kafka_produceva
RD_EXPORT rd_kafka_error_t * rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt)
Produce and send a single message to broker.
rd_kafka_get_err_descs
RD_EXPORT void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs, size_t *cntp)
Returns the full list of error codes.
rd_kafka_topic_result_error_string
const RD_EXPORT char * rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres)
rd_kafka_admin_op_t
rd_kafka_admin_op_t
Admin operation enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:6895
rd_kafka_conf_set_dr_msg_cb
RD_EXPORT void rd_kafka_conf_set_dr_msg_cb(rd_kafka_conf_t *conf, void(*dr_msg_cb)(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, void *opaque))
Producer: Set delivery report callback in provided conf object.
rd_kafka_new
RD_EXPORT rd_kafka_t * rd_kafka_new(rd_kafka_type_t type, rd_kafka_conf_t *conf, char *errstr, size_t errstr_size)
Creates a new Kafka handle and starts its operation according to the specified type (RD_KAFKA_CONSUME...
RD_KAFKA_RESP_ERR__NOOP
@ RD_KAFKA_RESP_ERR__NOOP
Definition: rdkafka.h:405
rd_kafka_ConsumerGroupDescription_t
struct rd_kafka_ConsumerGroupDescription_s rd_kafka_ConsumerGroupDescription_t
DescribeConsumerGroups result type.
Definition: rdkafka.h:8522
rd_kafka_DeleteTopic_destroy_array
RD_EXPORT void rd_kafka_DeleteTopic_destroy_array(rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt)
Helper function to destroy all DeleteTopic objects in the del_topics array (of del_topic_cnt elements...
rd_kafka_header_add
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs, const char *name, ssize_t name_size, const void *value, ssize_t value_size)
Add header with name name and value val (copied) of size size (not including null-terminator).
rd_kafka_interceptor_f_on_broker_state_change_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_broker_state_change_t(rd_kafka_t *rk, int32_t broker_id, const char *secproto, const char *name, int port, const char *state, void *ic_opaque)
on_broker_state_change() is called just after a broker has been created or its state has been changed...
Definition: rdkafka.h:6493
rd_kafka_subscription
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics)
Returns the current topic subscription.
rd_kafka_ConfigResource_add_incremental_config
RD_EXPORT rd_kafka_error_t * rd_kafka_ConfigResource_add_incremental_config(rd_kafka_ConfigResource_t *config, const char *name, rd_kafka_AlterConfigOpType_t op_type, const char *value)
Add the value of the configuration entry for a subsequent incremental alter config operation....
rd_kafka_AdminOptions_new
RD_EXPORT rd_kafka_AdminOptions_t * rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api)
Create a new AdminOptions object.
rd_kafka_thread_cnt
RD_EXPORT int rd_kafka_thread_cnt(void)
Retrieve the current number of threads in use by librdkafka.
rd_kafka_set_log_queue
RD_EXPORT rd_kafka_resp_err_t rd_kafka_set_log_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Forward librdkafka logs (and debug) to the specified queue for serving with one of the ....
rd_kafka_conf_set_engine_callback_data
RD_EXPORT void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf, void *callback_data)
Set callback_data for OpenSSL engine.
rd_kafka_metadata::orig_broker_id
int32_t orig_broker_id
Definition: rdkafka.h:4970
rd_kafka_DeleteConsumerGroupOffsets_destroy
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy(rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets)
Destroy and free a DeleteConsumerGroupOffsets object previously created with rd_kafka_DeleteConsumerG...
rd_kafka_unittest
RD_EXPORT int rd_kafka_unittest(void)
Run librdkafka's built-in unit-tests.
RD_KAFKA_RESP_ERR__UNDERFLOW
@ RD_KAFKA_RESP_ERR__UNDERFLOW
Definition: rdkafka.h:377
rd_kafka_purge
RD_EXPORT rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags)
Purge messages currently handled by the producer instance.
rd_kafka_DeleteRecords
RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk, rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete records (messages) in topic partitions older than the offsets provided.
rd_kafka_ListOffsets_result_infos
const RD_EXPORT rd_kafka_ListOffsetsResultInfo_t ** rd_kafka_ListOffsets_result_infos(const rd_kafka_ListOffsets_result_t *result, size_t *cntp)
Returns the array of ListOffsetsResultInfo in result and populates the size of the array in cntp.
RD_KAFKA_RESP_ERR__GAPLESS_GUARANTEE
@ RD_KAFKA_RESP_ERR__GAPLESS_GUARANTEE
Definition: rdkafka.h:391
rd_kafka_consumer_close
RD_EXPORT rd_kafka_resp_err_t rd_kafka_consumer_close(rd_kafka_t *rk)
Close the consumer.
rd_kafka_DeleteAcls_result_responses
const RD_EXPORT rd_kafka_DeleteAcls_result_response_t ** rd_kafka_DeleteAcls_result_responses(const rd_kafka_DeleteAcls_result_t *result, size_t *cntp)
Get an array of DeleteAcls result responses from a DeleteAcls result.
RD_KAFKA_CONFIG_SOURCE_DYNAMIC_DEFAULT_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_DEFAULT_BROKER_CONFIG
Definition: rdkafka.h:7568
rd_kafka_AdminOptions_set_match_consumer_group_states
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_match_consumer_group_states(rd_kafka_AdminOptions_t *options, const rd_kafka_consumer_group_state_t *consumer_group_states, size_t consumer_group_states_cnt)
Set consumer groups states to query for.
rd_kafka_msg_partitioner_fnv1a
RD_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
FNV-1a partitioner.
rd_kafka_opaque
RD_EXPORT void * rd_kafka_opaque(const rd_kafka_t *rk)
Retrieves the opaque pointer previously set with rd_kafka_conf_set_opaque()
RD_KAFKA_RESP_ERR__WAIT_CACHE
@ RD_KAFKA_RESP_ERR__WAIT_CACHE
Definition: rdkafka.h:359
rd_kafka_conf_dump_free
RD_EXPORT void rd_kafka_conf_dump_free(const char **arr, size_t cnt)
Frees a configuration dump returned from rd_kafka_conf_dump() or `rd_kafka_topic_conf_dump().
rd_kafka_interceptor_f_on_destroy_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_destroy_t(rd_kafka_t *rk, void *ic_opaque)
on_destroy() is called from rd_kafka_destroy() or (rd_kafka_new() if rd_kafka_new() fails during init...
Definition: rdkafka.h:6252
RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafka.h:495
rd_kafka_topic_conf_set_partitioner_cb
RD_EXPORT void rd_kafka_topic_conf_set_partitioner_cb(rd_kafka_topic_conf_t *topic_conf, int32_t(*partitioner)(const rd_kafka_topic_t *rkt, const void *keydata, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque))
Producer: Set partitioner callback in provided topic conf object.
rd_kafka_queue_get_consumer
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_consumer(rd_kafka_t *rk)
rd_kafka_topic_partition_s::offset
int64_t offset
Definition: rdkafka.h:902
rd_kafka_AclPermissionType_name
const RD_EXPORT char * rd_kafka_AclPermissionType_name(rd_kafka_AclPermissionType_t acl_permission_type)
rd_kafka_message_destroy
RD_EXPORT void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage)
Frees resources for rkmessage and hands ownership back to rdkafka.
rd_kafka_last_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_last_error(void)
Returns the last error code generated by a legacy API call in the current thread.
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_EXPIRED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_EXPIRED
Definition: rdkafka.h:567
rd_kafka_event_stats
const RD_EXPORT char * rd_kafka_event_stats(rd_kafka_event_t *rkev)
Extract stats from the event.
rd_kafka_DeleteTopics_result_t
rd_kafka_event_t rd_kafka_DeleteTopics_result_t
Definition: rdkafka.h:5701
RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
@ RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
Definition: rdkafka.h:629
RD_KAFKA_RESP_ERR__TRANSPORT
@ RD_KAFKA_RESP_ERR__TRANSPORT
Definition: rdkafka.h:294
rd_kafka_CreateTopics_result_topics
const RD_EXPORT rd_kafka_topic_result_t ** rd_kafka_CreateTopics_result_topics(const rd_kafka_CreateTopics_result_t *result, size_t *cntp)
Get an array of topic results from a CreateTopics result.
rd_kafka_default_topic_conf_dup
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_default_topic_conf_dup(rd_kafka_t *rk)
Creates a copy/duplicate of rk 's default topic configuration object.
rd_kafka_group_info::members
struct rd_kafka_group_member_info * members
Definition: rdkafka.h:5112
rd_kafka_topic_partition_list_new
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_new(int size)
Create a new list/vector Topic+Partition container.
rd_kafka_log_print
RD_EXPORT void rd_kafka_log_print(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin (default) log sink: print to stderr.
RD_KAFKA_RESP_ERR_THROTTLING_QUOTA_EXCEEDED
@ RD_KAFKA_RESP_ERR_THROTTLING_QUOTA_EXCEEDED
Definition: rdkafka.h:615
RD_KAFKA_RESOURCE_PATTERN_LITERAL
@ RD_KAFKA_RESOURCE_PATTERN_LITERAL
Definition: rdkafka.h:7683
RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
@ RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafka.h:613
rd_kafka_group_member_info::client_host
char * client_host
Definition: rdkafka.h:5078
RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
@ RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
Definition: rdkafka.h:631
RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
Definition: rdkafka.h:7571
rd_kafka_controllerid
RD_EXPORT int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms)
Returns the current ControllerId as reported in broker metadata.
rd_kafka_topic_partition_list_sort
RD_EXPORT void rd_kafka_topic_partition_list_sort(rd_kafka_topic_partition_list_t *rktparlist, int(*cmp)(const void *a, const void *b, void *cmp_opaque), void *cmp_opaque)
Sort list using comparator cmp.
rd_kafka_ListConsumerGroupOffsets_result_t
rd_kafka_event_t rd_kafka_ListConsumerGroupOffsets_result_t
Definition: rdkafka.h:5729
rd_kafka_topic_conf_get
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve topic configuration value for property name.
rd_kafka_message_status
RD_EXPORT rd_kafka_msg_status_t rd_kafka_message_status(const rd_kafka_message_t *rkmessage)
Returns the message's persistence status in the topic log.
rd_kafka_error_new
RD_EXPORT rd_kafka_error_t * rd_kafka_error_new(rd_kafka_resp_err_t code, const char *fmt,...) RD_FORMAT(printf
Create a new error object with error code and optional human readable error string in fmt.
RD_KAFKA_RESP_ERR_INVALID_PARTITIONS
@ RD_KAFKA_RESP_ERR_INVALID_PARTITIONS
Definition: rdkafka.h:503
rd_kafka_event_DeleteConsumerGroupOffsets_result
const RD_EXPORT rd_kafka_DeleteConsumerGroupOffsets_result_t * rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get DeleteConsumerGroupOffsets result.
rd_kafka_ConsumerGroupDescription_state
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupDescription_state(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets state for the grpdesc group.
rd_kafka_topic_partition_s::opaque
void * opaque
Definition: rdkafka.h:905
rd_kafka_ConfigEntry_is_read_only
RD_EXPORT int rd_kafka_ConfigEntry_is_read_only(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_message_s
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1439
RD_KAFKA_RESP_ERR__INVALID_ARG
@ RD_KAFKA_RESP_ERR__INVALID_ARG
Definition: rdkafka.h:315
rd_kafka_conf_enable_sasl_queue
RD_EXPORT void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable)
Enable/disable creation of a queue specific to SASL events and callbacks.
rd_kafka_group_info::protocol_type
char * protocol_type
Definition: rdkafka.h:5110
rd_kafka_AlterConfigOpType_t
rd_kafka_AlterConfigOpType_t
Incremental alter configs operations.
Definition: rdkafka.h:7693
rd_kafka_UserScramCredentialAlteration_destroy
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy(rd_kafka_UserScramCredentialAlteration_t *alteration)
Destroys a UserScramCredentialAlteration given its pointer.
rd_kafka_CreatePartitions_result_topics
const RD_EXPORT rd_kafka_topic_result_t ** rd_kafka_CreatePartitions_result_topics(const rd_kafka_CreatePartitions_result_t *result, size_t *cntp)
Get an array of topic results from a CreatePartitions result.
RD_KAFKA_ACL_OPERATION_DELETE
@ RD_KAFKA_ACL_OPERATION_DELETE
Definition: rdkafka.h:7183
rd_kafka_mem_malloc
RD_EXPORT void * rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size)
Allocate memory using the same allocator librdkafka uses.
rd_kafka_AclBindingFilter_new
RD_EXPORT rd_kafka_AclBindingFilter_t * rd_kafka_AclBindingFilter_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBindingFilter object. This object is later passed to rd_kafka_DescribeAcls() or rd_ka...
RD_KAFKA_ADMIN_OP_INCREMENTALALTERCONFIGS
@ RD_KAFKA_ADMIN_OP_INCREMENTALALTERCONFIGS
Definition: rdkafka.h:6916
rd_kafka_metadata_broker::host
char * host
Definition: rdkafka.h:4932
rd_kafka_NewTopic_new
RD_EXPORT rd_kafka_NewTopic_t * rd_kafka_NewTopic_new(const char *topic, int num_partitions, int replication_factor, char *errstr, size_t errstr_size)
Create a new NewTopic object. This object is later passed to rd_kafka_CreateTopics().
RD_KAFKA_RESP_ERR__UNKNOWN_PROTOCOL
@ RD_KAFKA_RESP_ERR__UNKNOWN_PROTOCOL
Definition: rdkafka.h:345
rd_kafka_ListConsumerGroups
RD_EXPORT void rd_kafka_ListConsumerGroups(rd_kafka_t *rk, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List the consumer groups available in the cluster.
rd_kafka_conf_set_ssl_cert_verify_cb
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert_verify_cb(rd_kafka_conf_t *conf, int(*ssl_cert_verify_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int *x509_error, int depth, const char *buf, size_t size, char *errstr, size_t errstr_size, void *opaque))
Sets the verification callback of the broker certificate.
rd_kafka_event_type
RD_EXPORT rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev)
rd_kafka_ConsumerGroupListing_is_simple_consumer_group
RD_EXPORT int rd_kafka_ConsumerGroupListing_is_simple_consumer_group(const rd_kafka_ConsumerGroupListing_t *grplist)
Is the grplist group a simple consumer group.
rd_kafka_topic_partition_list_set_offset
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_partition_list_set_offset(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition, int64_t offset)
Set offset to offset for topic and partition.
rd_kafka_AdminOptions_set_validate_only
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_validate_only(rd_kafka_AdminOptions_t *options, int true_or_false, char *errstr, size_t errstr_size)
Tell broker to only validate the request, without performing the requested operation (create topics,...
rd_kafka_interceptor_f_on_conf_destroy_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_destroy_t(void *ic_opaque)
on_conf_destroy() is called from rd_kafka_*_conf_destroy() in the order the interceptors were added.
Definition: rdkafka.h:6215
rd_kafka_conf_set_default_topic_conf
RD_EXPORT void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf, rd_kafka_topic_conf_t *tconf)
Sets the default topic configuration to use for automatically subscribed topics (e....
RD_KAFKA_CERT_CA
@ RD_KAFKA_CERT_CA
Definition: rdkafka.h:2456
rd_kafka_ConsumerGroupDescription_partition_assignor
const RD_EXPORT char * rd_kafka_ConsumerGroupDescription_partition_assignor(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the partition assignor for the grpdesc group.
rd_kafka_topic_conf_set
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a single rd_kafka_topic_conf_t value by property name.
rd_kafka_DescribeUserScramCredentials_result_t
rd_kafka_event_t rd_kafka_DescribeUserScramCredentials_result_t
Definition: rdkafka.h:5735
rd_kafka_group_info::member_cnt
int member_cnt
Definition: rdkafka.h:5113
RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
@ RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
Definition: rdkafka.h:403
RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
@ RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
Definition: rdkafka.h:604
RD_KAFKA_ADMIN_OP_DELETERECORDS
@ RD_KAFKA_ADMIN_OP_DELETERECORDS
Definition: rdkafka.h:6902
rd_kafka_conf_set
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a configuration property.
rd_kafka_TopicPartitionInfo_t
struct rd_kafka_TopicPartitionInfo_s rd_kafka_TopicPartitionInfo_t
TopicPartition represents a partition in the DescribeTopics result.
Definition: rdkafka.h:8118
rd_kafka_ConsumerGroupListing_t
struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t
ListConsumerGroups result for a single group.
Definition: rdkafka.h:8418
rd_kafka_group_info::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:5108
rd_kafka_topic_partition_s::topic
char * topic
Definition: rdkafka.h:900
RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
@ RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6920
rd_kafka_message_leader_epoch
RD_EXPORT int32_t rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage)
rd_kafka_offset_store
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Store offset offset + 1 for topic rkt partition partition.
rd_kafka_metadata::topic_cnt
int topic_cnt
Definition: rdkafka.h:4967
rd_kafka_group_list::groups
struct rd_kafka_group_info * groups
Definition: rdkafka.h:5122
RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
@ RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
Definition: rdkafka.h:335
rd_kafka_metadata_partition::leader
int32_t leader
Definition: rdkafka.h:4942
RD_KAFKA_RESP_ERR__FS
@ RD_KAFKA_RESP_ERR__FS
Definition: rdkafka.h:309
rd_kafka_consume_start_queue
RD_EXPORT int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, rd_kafka_queue_t *rkqu)
Same as rd_kafka_consume_start() but re-routes incoming messages to the provided queue rkqu (which mu...
rd_kafka_DeleteConsumerGroupOffsets_t
struct rd_kafka_DeleteConsumerGroupOffsets_s rd_kafka_DeleteConsumerGroupOffsets_t
Definition: rdkafka.h:9068
rd_kafka_event_CreateTopics_result
const RD_EXPORT rd_kafka_CreateTopics_result_t * rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev)
Get CreateTopics result.
rd_kafka_AlterConfigs_result_t
rd_kafka_event_t rd_kafka_AlterConfigs_result_t
Definition: rdkafka.h:5711
RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafka.h:469
rd_kafka_group_info::group
char * group
Definition: rdkafka.h:5107
rd_kafka_message_s::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:1440
rd_kafka_produce
RD_EXPORT int rd_kafka_produce(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, void *payload, size_t len, const void *key, size_t keylen, void *msg_opaque)
Produce and send a single message to broker.
rd_kafka_group_member_info
Group member information.
Definition: rdkafka.h:5075
rd_kafka_MemberDescription_assignment
const RD_EXPORT rd_kafka_MemberAssignment_t * rd_kafka_MemberDescription_assignment(const rd_kafka_MemberDescription_t *member)
Gets assignment of member.
rd_kafka_interceptor_add_on_broker_state_change
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_broker_state_change(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change, void *ic_opaque)
Append an on_broker_state_change() interceptor.
rd_kafka_DescribeConsumerGroups
RD_EXPORT void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk, const char **groups, size_t groups_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe groups from cluster as specified by the groups array of size groups_cnt elements.
RD_KAFKA_RESP_ERR__TIMED_OUT
@ RD_KAFKA_RESP_ERR__TIMED_OUT
Definition: rdkafka.h:317
RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
@ RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafka.h:531
RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
@ RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafka.h:538
RD_KAFKA_RESP_ERR__RETRY
@ RD_KAFKA_RESP_ERR__RETRY
Definition: rdkafka.h:381
rd_kafka_header_cnt
RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs)
Returns the number of header key/value pairs.
RD_KAFKA_ACL_OPERATION_READ
@ RD_KAFKA_ACL_OPERATION_READ
Definition: rdkafka.h:7180
rd_kafka_poll_set_consumer
RD_EXPORT rd_kafka_resp_err_t rd_kafka_poll_set_consumer(rd_kafka_t *rk)
Redirect the main (rd_kafka_poll()) queue to the KafkaConsumer's queue (rd_kafka_consumer_poll()).
rd_kafka_DescribeConfigs
RD_EXPORT void rd_kafka_DescribeConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Get configuration for the specified resources in configs.
RD_KAFKA_ADMIN_OP_DESCRIBECONFIGS
@ RD_KAFKA_ADMIN_OP_DESCRIBECONFIGS
Definition: rdkafka.h:6901
rd_kafka_ConfigResource_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_error(const rd_kafka_ConfigResource_t *config)
rd_kafka_CreatePartitions
RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk, rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create additional partitions for the given topics, as specified by the new_parts array of size new_pa...
RD_KAFKA_RESP_ERR_INVALID_CONFIG
@ RD_KAFKA_RESP_ERR_INVALID_CONFIG
Definition: rdkafka.h:509
rd_kafka_consume_batch
RD_EXPORT ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume up to rkmessages_size from topic rkt and partition putting a pointer to each message in the a...
RD_KAFKA_CONFIG_SOURCE_UNKNOWN_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_UNKNOWN_CONFIG
Definition: rdkafka.h:7561
RD_KAFKA_VTYPE_PARTITION
@ RD_KAFKA_VTYPE_PARTITION
Definition: rdkafka.h:1121
RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
@ RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafka.h:551
RD_KAFKA_RESP_ERR__INCONSISTENT
@ RD_KAFKA_RESP_ERR__INCONSISTENT
Definition: rdkafka.h:389
rd_kafka_DeleteTopics_result_topics
const RD_EXPORT rd_kafka_topic_result_t ** rd_kafka_DeleteTopics_result_topics(const rd_kafka_DeleteTopics_result_t *result, size_t *cntp)
Get an array of topic results from a DeleteTopics result.
rd_kafka_event_DescribeAcls_result
const RD_EXPORT rd_kafka_DescribeAcls_result_t * rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR_PRODUCER_FENCED
@ RD_KAFKA_RESP_ERR_PRODUCER_FENCED
Definition: rdkafka.h:618
rd_kafka_rebalance_protocol
const RD_EXPORT char * rd_kafka_rebalance_protocol(rd_kafka_t *rk)
The rebalance protocol currently in use. This will be "NONE" if the consumer has not (yet) joined a g...
RD_KAFKA_MSG_STATUS_NOT_PERSISTED
@ RD_KAFKA_MSG_STATUS_NOT_PERSISTED
Definition: rdkafka.h:1597
RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
@ RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
Definition: rdkafka.h:571
RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
@ RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
Definition: rdkafka.h:497
rd_kafka_event_CreateAcls_result
const RD_EXPORT rd_kafka_CreateAcls_result_t * rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev)
rd_kafka_interceptor_f_on_request_sent_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_request_sent_t(rd_kafka_t *rk, int sockfd, const char *brokername, int32_t brokerid, int16_t ApiKey, int16_t ApiVersion, int32_t CorrId, size_t size, void *ic_opaque)
on_request_sent() is called when a request has been fully written to a broker TCP connections socket.
Definition: rdkafka.h:6379
RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
@ RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
Definition: rdkafka.h:546
rd_kafka_msg_partitioner_consistent
RD_EXPORT int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent partitioner.
rd_kafka_event_ListConsumerGroupOffsets_result
const RD_EXPORT rd_kafka_ListConsumerGroupOffsets_result_t * rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get ListConsumerGroupOffsets result.
rd_kafka_topic_partition_s
Topic+Partition place holder.
Definition: rdkafka.h:899
rd_kafka_AdminOptions_set_require_stable_offsets
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_require_stable_offsets(rd_kafka_AdminOptions_t *options, int true_or_false)
Whether broker should return stable offsets (transaction-committed).
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_OWNER_MISMATCH
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_OWNER_MISMATCH
Definition: rdkafka.h:561
rd_kafka_err_desc::desc
const char * desc
Definition: rdkafka.h:647
rd_kafka_conf_res_t
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1716
rd_kafka_event_topic_partition
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_event_topic_partition(rd_kafka_event_t *rkev)
rd_kafka_set_log_level
RD_EXPORT void rd_kafka_set_log_level(rd_kafka_t *rk, int level)
Specifies the maximum logging level emitted by internal kafka logging and debugging.
RD_KAFKA_RESP_ERR__ASSIGN_PARTITIONS
@ RD_KAFKA_RESP_ERR__ASSIGN_PARTITIONS
Definition: rdkafka.h:337
rd_kafka_queue_forward
RD_EXPORT void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst)
Forward/re-route queue src to dst. If dst is NULL the forwarding is removed.
rd_kafka_DeleteRecords_destroy_array
RD_EXPORT void rd_kafka_DeleteRecords_destroy_array(rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt)
Helper function to destroy all DeleteRecords objects in the del_groups array (of del_group_cnt elemen...
RD_KAFKA_RESP_ERR_MSG_SIZE_TOO_LARGE
@ RD_KAFKA_RESP_ERR_MSG_SIZE_TOO_LARGE
Definition: rdkafka.h:440
rd_kafka_ConfigSource_t
rd_kafka_ConfigSource_t
Apache Kafka config sources.
Definition: rdkafka.h:7558
rd_kafka_metadata_broker::id
int32_t id
Definition: rdkafka.h:4931
RD_KAFKA_RESP_ERR__INTR
@ RD_KAFKA_RESP_ERR__INTR
Definition: rdkafka.h:361
rd_kafka_ConsumerGroupDescription_authorized_operations
const RD_EXPORT rd_kafka_AclOperation_t * rd_kafka_ConsumerGroupDescription_authorized_operations(const rd_kafka_ConsumerGroupDescription_t *grpdesc, size_t *cntp)
Gets the authorized ACL operations for the grpdesc group.
rd_kafka_interceptor_add_on_consume
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_consume(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_consume_t *on_consume, void *ic_opaque)
Append an on_consume() interceptor.
rd_kafka_consumer_closed
RD_EXPORT int rd_kafka_consumer_closed(rd_kafka_t *rk)
rd_kafka_AdminOptions_set_request_timeout
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_request_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the overall request timeout, including broker lookup, request transmission, operation time on br...
rd_kafka_DescribeTopics
RD_EXPORT void rd_kafka_DescribeTopics(rd_kafka_t *rk, const rd_kafka_TopicCollection_t *topics, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe topics as specified by the topics array of size topics_cnt elements.
rd_kafka_consume_start
RD_EXPORT int rd_kafka_consume_start(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Start consuming messages for topic rkt and partition at offset offset which may either be an absolute...
rd_kafka_AlterConsumerGroupOffsets_t
struct rd_kafka_AlterConsumerGroupOffsets_s rd_kafka_AlterConsumerGroupOffsets_t
Definition: rdkafka.h:8972
rd_kafka_topic_name
const RD_EXPORT char * rd_kafka_topic_name(const rd_kafka_topic_t *rkt)
Returns the topic name.
rd_kafka_message_errstr
const RD_EXPORT char * rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage)
Returns the error string for an errored rd_kafka_message_t or NULL if there was no error.
rd_kafka_conf_set_consume_cb
RD_EXPORT void rd_kafka_conf_set_consume_cb(rd_kafka_conf_t *conf, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque))
Consumer: Set consume callback for use with rd_kafka_consumer_poll()
rd_kafka_MemberDescription_t
struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t
Member description included in ConsumerGroupDescription.
Definition: rdkafka.h:8529
rd_kafka_message_s::key_len
size_t key_len
Definition: rdkafka.h:1452
rd_kafka_CreateAcls_result_t
rd_kafka_event_t rd_kafka_CreateAcls_result_t
Definition: rdkafka.h:5703
RD_KAFKA_RESP_ERR_INVALID_MSG
@ RD_KAFKA_RESP_ERR_INVALID_MSG
Definition: rdkafka.h:422
rd_kafka_AclBinding_name
const RD_EXPORT char * rd_kafka_AclBinding_name(const rd_kafka_AclBinding_t *acl)
rd_kafka_name
const RD_EXPORT char * rd_kafka_name(const rd_kafka_t *rk)
Returns Kafka handle name.
rd_kafka_AlterConfigs
RD_EXPORT void rd_kafka_AlterConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Update the configuration for the specified resources. Updates are not transactional so they may succe...
rd_kafka_abort_transaction
RD_EXPORT rd_kafka_error_t * rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms)
Aborts the ongoing transaction.
rd_kafka_message_headers
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_headers(const rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list.
rd_kafka_DescribeCluster_result_controller
const RD_EXPORT rd_kafka_Node_t * rd_kafka_DescribeCluster_result_controller(const rd_kafka_DescribeCluster_result_t *result)
Gets the current controller for the result cluster.
rd_kafka_DeleteTopics
RD_EXPORT void rd_kafka_DeleteTopics(rd_kafka_t *rk, rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete topics from cluster as specified by the topics array of size topic_cnt elements.
rd_kafka_metadata::orig_broker_name
char * orig_broker_name
Definition: rdkafka.h:4971
rd_kafka_interceptor_f_on_consume_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_consume_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_consume() is called just prior to passing the message to the application in rd_kafka_consumer_poll...
Definition: rdkafka.h:6325
RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
@ RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
Definition: rdkafka.h:9516
rd_kafka_AlterConsumerGroupOffsets_result_t
rd_kafka_event_t rd_kafka_AlterConsumerGroupOffsets_result_t
Definition: rdkafka.h:5727
rd_kafka_ListConsumerGroupsResult_t
struct rd_kafka_ListConsumerGroupsResult_s rd_kafka_ListConsumerGroupsResult_t
Definition: rdkafka.h:8421
RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
@ RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
Definition: rdkafka.h:446
rd_kafka_interceptor_add_on_commit
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_commit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_commit_t *on_commit, void *ic_opaque)
Append an on_commit() interceptor.
rd_kafka_event_log
RD_EXPORT int rd_kafka_event_log(rd_kafka_event_t *rkev, const char **fac, const char **str, int *level)
Extract log message from the event.
rd_kafka_ConsumerGroupDescription_group_id
const RD_EXPORT char * rd_kafka_ConsumerGroupDescription_group_id(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the group id for the grpdesc group.
RD_KAFKA_RESOURCE_PATTERN_ANY
@ RD_KAFKA_RESOURCE_PATTERN_ANY
Definition: rdkafka.h:7679
rd_kafka_topic_result_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres)
Topic result provides per-topic operation result information.
rd_kafka_ListOffsetsResultInfo_timestamp
RD_EXPORT int64_t rd_kafka_ListOffsetsResultInfo_timestamp(const rd_kafka_ListOffsetsResultInfo_t *result_info)
Returns the timestamp corresponding to the offset in result_info.
RD_KAFKA_MSG_STATUS_PERSISTED
@ RD_KAFKA_MSG_STATUS_PERSISTED
Definition: rdkafka.h:1607
rd_kafka_group_member_info::member_metadata_size
int member_metadata_size
Definition: rdkafka.h:5081
rd_kafka_conf_set_error_cb
RD_EXPORT void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf, void(*error_cb)(rd_kafka_t *rk, int err, const char *reason, void *opaque))
Set error callback in provided conf object.
rd_kafka_ConfigEntry_source
RD_EXPORT rd_kafka_ConfigSource_t rd_kafka_ConfigEntry_source(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_topic_conf_dump
const RD_EXPORT char ** rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf, size_t *cntp)
Dump the topic configuration properties and values of conf to an array with "key",...
RD_KAFKA_RESP_ERR__READ_ONLY
@ RD_KAFKA_RESP_ERR__READ_ONLY
Definition: rdkafka.h:373
RD_KAFKA_ADMIN_OP_DELETEGROUPS
@ RD_KAFKA_ADMIN_OP_DELETEGROUPS
Definition: rdkafka.h:6903
rd_kafka_conf_set_oauthbearer_token_refresh_cb
RD_EXPORT void rd_kafka_conf_set_oauthbearer_token_refresh_cb(rd_kafka_conf_t *conf, void(*oauthbearer_token_refresh_cb)(rd_kafka_t *rk, const char *oauthbearer_config, void *opaque))
Set SASL/OAUTHBEARER token refresh callback in provided conf object.
rd_kafka_event_name
const RD_EXPORT char * rd_kafka_event_name(const rd_kafka_event_t *rkev)
rd_kafka_queue_get_background
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_background(rd_kafka_t *rk)
RD_KAFKA_RESP_ERR_INVALID_SESSION_TIMEOUT
@ RD_KAFKA_RESP_ERR_INVALID_SESSION_TIMEOUT
Definition: rdkafka.h:481
rd_kafka_AlterUserScramCredentials_result_t
rd_kafka_event_t rd_kafka_AlterUserScramCredentials_result_t
Definition: rdkafka.h:5737
rd_kafka_headers_new
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_new(size_t initial_count)
Create a new headers list.
rd_kafka_metadata_topic::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:4956
rd_kafka_group_member_info::member_id
char * member_id
Definition: rdkafka.h:5076
rd_kafka_AlterConsumerGroupOffsets_destroy
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy(rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets)
Destroy and free a AlterConsumerGroupOffsets object previously created with rd_kafka_AlterConsumerGro...
rd_kafka_err_desc::name
const char * name
Definition: rdkafka.h:646
rd_kafka_Uuid_destroy
RD_EXPORT void rd_kafka_Uuid_destroy(rd_kafka_Uuid_t *uuid)
Destroy the provided uuid.
RD_KAFKA_VTYPE_END
@ RD_KAFKA_VTYPE_END
Definition: rdkafka.h:1118
rd_kafka_AlterUserScramCredentials_result_responses
const RD_EXPORT rd_kafka_AlterUserScramCredentials_result_response_t ** rd_kafka_AlterUserScramCredentials_result_responses(const rd_kafka_AlterUserScramCredentials_result_t *result, size_t *cntp)
Get an array of responses from a AlterUserScramCredentials result.
rd_kafka_ConfigResource_configs
const RD_EXPORT rd_kafka_ConfigEntry_t ** rd_kafka_ConfigResource_configs(const rd_kafka_ConfigResource_t *config, size_t *cntp)
Get an array of config entries from a ConfigResource object.
rd_kafka_ConfigResource_set_config
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_set_config(rd_kafka_ConfigResource_t *config, const char *name, const char *value)
Set configuration name value pair.
rd_kafka_vu_s::vtype
rd_kafka_vtype_t vtype
Definition: rdkafka.h:1144
rd_kafka_mem_calloc
RD_EXPORT void * rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size)
Allocate and zero memory using the same allocator librdkafka uses.
rd_kafka_commit_transaction
RD_EXPORT rd_kafka_error_t * rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms)
Commit the current transaction (as started with rd_kafka_begin_transaction()).
rd_kafka_ConfigEntry_synonyms
const RD_EXPORT rd_kafka_ConfigEntry_t ** rd_kafka_ConfigEntry_synonyms(const rd_kafka_ConfigEntry_t *entry, size_t *cntp)
RD_KAFKA_ADMIN_OP_CREATEACLS
@ RD_KAFKA_ADMIN_OP_CREATEACLS
Definition: rdkafka.h:6906
RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:595
rd_kafka_CreateTopics_result_t
rd_kafka_event_t rd_kafka_CreateTopics_result_t
Definition: rdkafka.h:5699
RD_KAFKA_RESP_ERR__FATAL
@ RD_KAFKA_RESP_ERR__FATAL
Definition: rdkafka.h:387
rd_kafka_CreateAcls_result_acls
const RD_EXPORT rd_kafka_acl_result_t ** rd_kafka_CreateAcls_result_acls(const rd_kafka_CreateAcls_result_t *result, size_t *cntp)
Get an array of acl results from a CreateAcls result.
RD_KAFKA_ADMIN_OP_LISTOFFSETS
@ RD_KAFKA_ADMIN_OP_LISTOFFSETS
Definition: rdkafka.h:6923
RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
@ RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
Definition: rdkafka.h:367
rd_kafka_oauthbearer_set_token_failure
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token_failure(rd_kafka_t *rk, const char *errstr)
SASL/OAUTHBEARER token refresh failure indicator.
rd_kafka_ListConsumerGroups_result_errors
const RD_EXPORT rd_kafka_error_t ** rd_kafka_ListConsumerGroups_result_errors(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of errors from a ListConsumerGroups call result.
rd_kafka_destroy_flags
RD_EXPORT void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags)
Destroy Kafka handle according to specified destroy flags.
rd_kafka_interceptor_add_on_acknowledgement
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_acknowledgement(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement, void *ic_opaque)
Append an on_acknowledgement() interceptor.
rd_kafka_incremental_assign
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally add partitions to the current assignment.
rd_kafka_TopicDescription_topic_id
const RD_EXPORT rd_kafka_Uuid_t * rd_kafka_TopicDescription_topic_id(const rd_kafka_TopicDescription_t *topicdesc)
Gets the topic id for the topicdesc topic.
rd_kafka_conf_set_throttle_cb
RD_EXPORT void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf, void(*throttle_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int throttle_time_ms, void *opaque))
Set throttle callback.
RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPOFFSETS
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6912
RD_KAFKA_RESP_ERR_SECURITY_DISABLED
@ RD_KAFKA_RESP_ERR_SECURITY_DISABLED
Definition: rdkafka.h:542
rd_kafka_vtype_t
rd_kafka_vtype_t
Var-arg tag types.
Definition: rdkafka.h:1117
rd_kafka_CreatePartitions_result_t
rd_kafka_event_t rd_kafka_CreatePartitions_result_t
Definition: rdkafka.h:5709
rd_kafka_DescribeUserScramCredentials
RD_EXPORT void rd_kafka_DescribeUserScramCredentials(rd_kafka_t *rk, const char **users, size_t user_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe SASL/SCRAM credentials. This operation is supported by brokers with version 2....
RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS
Definition: rdkafka.h:467
rd_kafka_log_syslog
RD_EXPORT void rd_kafka_log_syslog(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin log sink: print to syslog.
RD_KAFKA_ADMIN_OP_DELETECONSUMERGROUPOFFSETS
@ RD_KAFKA_ADMIN_OP_DELETECONSUMERGROUPOFFSETS
Definition: rdkafka.h:6905
rd_kafka_ScramCredentialInfo_iterations
RD_EXPORT int32_t rd_kafka_ScramCredentialInfo_iterations(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the iterations of a given ScramCredentialInfo.
rd_kafka_NewTopic_destroy
RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic)
Destroy and free a NewTopic object previously created with rd_kafka_NewTopic_new()
RD_KAFKA_RESP_ERR__NOT_IMPLEMENTED
@ RD_KAFKA_RESP_ERR__NOT_IMPLEMENTED
Definition: rdkafka.h:347
rd_kafka_ListConsumerGroupOffsets_result_groups
const RD_EXPORT rd_kafka_group_result_t ** rd_kafka_ListConsumerGroupOffsets_result_groups(const rd_kafka_ListConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a ListConsumerGroupOffsets result.
rd_kafka_dump
RD_EXPORT void rd_kafka_dump(FILE *fp, rd_kafka_t *rk)
Dumps rdkafka's internal state for handle rk to stream fp.
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_NOT_FOUND
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_NOT_FOUND
Definition: rdkafka.h:559
RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
Definition: rdkafka.h:589
rd_kafka_conf_dump
const RD_EXPORT char ** rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp)
Dump the configuration properties and values of conf to an array with "key", "value" pairs.
rd_kafka_ConsumerGroupListing_state
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupListing_state(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets state for the grplist group.
rd_kafka_DescribeCluster_result_nodes
const RD_EXPORT rd_kafka_Node_t ** rd_kafka_DescribeCluster_result_nodes(const rd_kafka_DescribeCluster_result_t *result, size_t *cntp)
Gets the broker nodes for the result cluster.
rd_kafka_error_name
const RD_EXPORT char * rd_kafka_error_name(const rd_kafka_error_t *error)
RD_KAFKA_RESOURCE_PATTERN_UNKNOWN
@ RD_KAFKA_RESOURCE_PATTERN_UNKNOWN
Definition: rdkafka.h:7677
RD_KAFKA_RESP_ERR__QUEUE_FULL
@ RD_KAFKA_RESP_ERR__QUEUE_FULL
Definition: rdkafka.h:319
RD_KAFKA_VTYPE_KEY
@ RD_KAFKA_VTYPE_KEY
Definition: rdkafka.h:1123
rd_kafka_consumer_group_metadata_new
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new(const char *group_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
rd_kafka_outq_len
RD_EXPORT int rd_kafka_outq_len(rd_kafka_t *rk)
Returns the current out queue length.
RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
Definition: rdkafka.h:448
rd_kafka_Uuid_most_significant_bits
RD_EXPORT int64_t rd_kafka_Uuid_most_significant_bits(const rd_kafka_Uuid_t *uuid)
Gets most significant 64 bits for the given UUID.
rd_kafka_ResourceType_t
rd_kafka_ResourceType_t
Apache Kafka resource types.
Definition: rdkafka.h:7662
rd_kafka_AclBinding_restype
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_AclBinding_restype(const rd_kafka_AclBinding_t *acl)
rd_kafka_AlterUserScramCredentials_result_response_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_AlterUserScramCredentials_result_response_error(const rd_kafka_AlterUserScramCredentials_result_response_t *response)
Returns the error of a rd_kafka_AlterUserScramCredentials_result_response.
RD_KAFKA_CONF_OK
@ RD_KAFKA_CONF_OK
Definition: rdkafka.h:1721
rd_kafka_conf_set_log_cb
RD_EXPORT void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf, void(*log_cb)(const rd_kafka_t *rk, int level, const char *fac, const char *buf))
Set logger callback.
rd_kafka_group_result_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres)
Group result provides per-group operation result information.
RD_KAFKA_ACL_PERMISSION_TYPE_DENY
@ RD_KAFKA_ACL_PERMISSION_TYPE_DENY
Definition: rdkafka.h:9515
rd_kafka_CreateTopics
RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk, rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create topics in cluster as specified by the new_topics array of size new_topic_cnt elements.
rd_kafka_DeleteRecords_new
RD_EXPORT rd_kafka_DeleteRecords_t * rd_kafka_DeleteRecords_new(const rd_kafka_topic_partition_list_t *before_offsets)
Create a new DeleteRecords object. This object is later passed to rd_kafka_DeleteRecords().
rd_kafka_msg_partitioner_consistent_random
RD_EXPORT int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random partitioner.
RD_KAFKA_RESP_ERR__PURGE_INFLIGHT
@ RD_KAFKA_RESP_ERR__PURGE_INFLIGHT
Definition: rdkafka.h:385
RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
@ RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
Definition: rdkafka.h:600
rd_kafka_consumer_group_metadata_read
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_read(rd_kafka_consumer_group_metadata_t **cgmdp, const void *buffer, size_t size)
Reads serialized consumer group metadata and returns a consumer group metadata object....
rd_kafka_ConsumerGroupDescription_is_simple_consumer_group
RD_EXPORT int rd_kafka_ConsumerGroupDescription_is_simple_consumer_group(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Is the grpdesc group a simple consumer group.
RD_KAFKA_RESP_ERR_TOPIC_ALREADY_EXISTS
@ RD_KAFKA_RESP_ERR_TOPIC_ALREADY_EXISTS
Definition: rdkafka.h:501
rd_kafka_interceptor_f_on_send_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_send_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_send() is called from rd_kafka_produce*() (et.al) prior to the partitioner being called.
Definition: rdkafka.h:6276
rd_kafka_offsets_for_times
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_for_times(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets, int timeout_ms)
Look up the offsets for the given partitions by timestamp.
RD_KAFKA_THREAD_MAIN
@ RD_KAFKA_THREAD_MAIN
Definition: rdkafka.h:5351
RD_KAFKA_THREAD_BACKGROUND
@ RD_KAFKA_THREAD_BACKGROUND
Definition: rdkafka.h:5352
rd_kafka_DeleteAcls
RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t **del_acls, size_t del_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete acls matching the filteres provided in del_acls array of size del_acls_cnt.
rd_kafka_ConsumerGroupDescription_coordinator
const RD_EXPORT rd_kafka_Node_t * rd_kafka_ConsumerGroupDescription_coordinator(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the coordinator for the grpdesc group.
rd_kafka_consumer_group_metadata
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata(rd_kafka_t *rk)
RD_KAFKA_RESP_ERR_INVALID_PRINCIPAL_TYPE
@ RD_KAFKA_RESP_ERR_INVALID_PRINCIPAL_TYPE
Definition: rdkafka.h:569
rd_kafka_AdminOptions_destroy
RD_EXPORT void rd_kafka_AdminOptions_destroy(rd_kafka_AdminOptions_t *options)
Destroy a AdminOptions object.
rd_kafka_message_timestamp
RD_EXPORT int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage, rd_kafka_timestamp_type_t *tstype)
Returns the message timestamp for a consumed message.
rd_kafka_DeleteGroup_new
RD_EXPORT rd_kafka_DeleteGroup_t * rd_kafka_DeleteGroup_new(const char *group)
Create a new DeleteGroup object. This object is later passed to rd_kafka_DeleteGroups().
rd_kafka_TopicCollection_destroy
RD_EXPORT void rd_kafka_TopicCollection_destroy(rd_kafka_TopicCollection_t *topics)
Destroy and free a TopicCollection object created with rd_kafka_TopicCollection_new_* methods.
rd_kafka_interceptor_f_on_thread_start_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_thread_start_t(rd_kafka_t *rk, rd_kafka_thread_type_t thread_type, const char *thread_name, void *ic_opaque)
on_thread_start() is called from a newly created librdkafka-managed thread.
Definition: rdkafka.h:6446
rd_kafka_metadata_broker::port
int port
Definition: rdkafka.h:4933
rd_kafka_AclPermissionType_t
rd_kafka_AclPermissionType_t
Apache Kafka ACL permission types.
Definition: rdkafka.h:9511
RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
@ RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6914
rd_kafka_DeleteGroup_destroy_array
RD_EXPORT void rd_kafka_DeleteGroup_destroy_array(rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt)
Helper function to destroy all DeleteGroup objects in the del_groups array (of del_group_cnt elements...
RD_KAFKA_RESP_ERR_OFFSET_OUT_OF_RANGE
@ RD_KAFKA_RESP_ERR_OFFSET_OUT_OF_RANGE
Definition: rdkafka.h:420
rd_kafka_DescribeCluster_result_t
rd_kafka_event_t rd_kafka_DescribeCluster_result_t
Definition: rdkafka.h:5733
rd_kafka_AclBindingFilter_t
rd_kafka_AclBinding_t rd_kafka_AclBindingFilter_t
ACL Binding filter is used to filter access control lists.
Definition: rdkafka.h:9492
RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
Definition: rdkafka.h:7565
rd_kafka_conf_set_connect_cb
RD_EXPORT void rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf, int(*connect_cb)(int sockfd, const struct sockaddr *addr, int addrlen, const char *id, void *opaque))
Set connect callback.
rd_kafka_ListConsumerGroupOffsets_new
RD_EXPORT rd_kafka_ListConsumerGroupOffsets_t * rd_kafka_ListConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new ListConsumerGroupOffsets object. This object is later passed to rd_kafka_ListConsumerGro...
rd_kafka_group_list::group_cnt
int group_cnt
Definition: rdkafka.h:5123
RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
@ RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
Definition: rdkafka.h:363
rd_kafka_commit_message
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_message(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, int async)
Commit message's offset on broker for the message's partition. The committed offset is the message's ...
rd_kafka_AlterConsumerGroupOffsets_destroy_array
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy_array(rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffset_cnt)
Helper function to destroy all AlterConsumerGroupOffsets objects in the alter_grpoffsets array (of al...
RD_KAFKA_RESP_ERR__BEGIN
@ RD_KAFKA_RESP_ERR__BEGIN
Definition: rdkafka.h:284
RD_KAFKA_RESP_ERR__BAD_COMPRESSION
@ RD_KAFKA_RESP_ERR__BAD_COMPRESSION
Definition: rdkafka.h:288
rd_kafka_topic_conf_new
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_new(void)
Create topic configuration object.
RD_KAFKA_RESP_ERR_UNKNOWN_MEMBER_ID
@ RD_KAFKA_RESP_ERR_UNKNOWN_MEMBER_ID
Definition: rdkafka.h:479
rd_kafka_interceptor_f_on_thread_exit_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_thread_exit_t(rd_kafka_t *rk, rd_kafka_thread_type_t thread_type, const char *thread_name, void *ic_opaque)
on_thread_exit() is called just prior to a librdkafka-managed thread exiting from the exiting thread ...
Definition: rdkafka.h:6472
RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
@ RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
Definition: rdkafka.h:525
RD_KAFKA_VTYPE_HEADERS
@ RD_KAFKA_VTYPE_HEADERS
Definition: rdkafka.h:1133
rd_kafka_metadata_partition::replica_cnt
int replica_cnt
Definition: rdkafka.h:4943
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafka.h:563
RD_KAFKA_RESP_ERR_INVALID_RECORD
@ RD_KAFKA_RESP_ERR_INVALID_RECORD
Definition: rdkafka.h:611
rd_kafka_error_txn_requires_abort
RD_EXPORT int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error)
RD_KAFKA_RESP_ERR_TOPIC_DELETION_DISABLED
@ RD_KAFKA_RESP_ERR_TOPIC_DELETION_DISABLED
Definition: rdkafka.h:581
rd_kafka_interceptor_add_on_request_sent
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_request_sent(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_request_sent_t *on_request_sent, void *ic_opaque)
Append an on_request_sent() interceptor.
RD_KAFKA_RESP_ERR__NOT_CONFIGURED
@ RD_KAFKA_RESP_ERR__NOT_CONFIGURED
Definition: rdkafka.h:397
rd_kafka_metadata_broker
Broker information.
Definition: rdkafka.h:4930
rd_kafka_conf_get_default_topic_conf
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf)
Gets the default topic configuration as previously set with rd_kafka_conf_set_default_topic_conf() or...
rd_kafka_version
RD_EXPORT int rd_kafka_version(void)
Returns the librdkafka version as integer.
rd_kafka_topic_partition_list_s
A growable list of Topic+Partitions.
Definition: rdkafka.h:950
rd_kafka_NewTopic_t
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:7208
rd_kafka_conf_set_background_event_cb
RD_EXPORT void rd_kafka_conf_set_background_event_cb(rd_kafka_conf_t *conf, void(*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque))
Generic event callback to be used with the event API to trigger callbacks for rd_kafka_event_t object...
rd_kafka_wait_destroyed
RD_EXPORT int rd_kafka_wait_destroyed(int timeout_ms)
Wait for all rd_kafka_t objects to be destroyed.
RD_KAFKA_RESP_ERR_GROUP_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_GROUP_AUTHORIZATION_FAILED
Definition: rdkafka.h:489
rd_kafka_subscribe
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscribe(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *topics)
Subscribe to topic set using balanced consumer groups.
rd_kafka_AdminOptions_set_operation_timeout
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_operation_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the broker's operation timeout, such as the timeout for CreateTopics to complete the creation of...
RD_KAFKA_RESP_ERR_OFFSET_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_OFFSET_NOT_AVAILABLE
Definition: rdkafka.h:591
rd_kafka_ConfigResource_new
RD_EXPORT rd_kafka_ConfigResource_t * rd_kafka_ConfigResource_new(rd_kafka_ResourceType_t restype, const char *resname)
Create new ConfigResource object.
rd_kafka_conf_new
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_new(void)
Create configuration object.
RD_KAFKA_RESP_ERR_UNSUPPORTED_COMPRESSION_TYPE
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_COMPRESSION_TYPE
Definition: rdkafka.h:587
rd_kafka_destroy
RD_EXPORT void rd_kafka_destroy(rd_kafka_t *rk)
Destroy Kafka handle.
rd_kafka_metadata_topic::partitions
struct rd_kafka_metadata_partition * partitions
Definition: rdkafka.h:4955
rd_kafka_topic_partition_destroy
RD_EXPORT void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar)
Destroy a rd_kafka_topic_partition_t.
RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPS
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPS
Definition: rdkafka.h:6909
RD_KAFKA_TIMESTAMP_LOG_APPEND_TIME
@ RD_KAFKA_TIMESTAMP_LOG_APPEND_TIME
Definition: rdkafka.h:223
rd_kafka_commit
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, int async)
Commit offsets on broker for the provided list of partitions.
RD_KAFKA_RESP_ERR__FAIL
@ RD_KAFKA_RESP_ERR__FAIL
Definition: rdkafka.h:292
rd_kafka_event_DeleteGroups_result
const RD_EXPORT rd_kafka_DeleteGroups_result_t * rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev)
Get DeleteGroups result.
RD_KAFKA_RESP_ERR_CLUSTER_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_CLUSTER_AUTHORIZATION_FAILED
Definition: rdkafka.h:491
rd_kafka_conf_interceptor_add_on_conf_destroy
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_destroy(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_destroy_t *on_conf_destroy, void *ic_opaque)
Append an on_conf_destroy() interceptor.
rd_kafka_Uuid_copy
RD_EXPORT rd_kafka_Uuid_t * rd_kafka_Uuid_copy(const rd_kafka_Uuid_t *uuid)
Copies the given UUID.
RD_KAFKA_RESP_ERR_INVALID_REPLICATION_FACTOR
@ RD_KAFKA_RESP_ERR_INVALID_REPLICATION_FACTOR
Definition: rdkafka.h:505
rd_kafka_event_IncrementalAlterConfigs_result
const RD_EXPORT rd_kafka_IncrementalAlterConfigs_result_t * rd_kafka_event_IncrementalAlterConfigs_result(rd_kafka_event_t *rkev)
Get IncrementalAlterConfigs result.
rd_kafka_interceptor_add_on_send
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_send(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_send_t *on_send, void *ic_opaque)
Append an on_send() interceptor.
RD_KAFKA_RESP_ERR__UNKNOWN_BROKER
@ RD_KAFKA_RESP_ERR__UNKNOWN_BROKER
Definition: rdkafka.h:395
rd_kafka_DescribeUserScramCredentials_result_descriptions
const RD_EXPORT rd_kafka_UserScramCredentialsDescription_t ** rd_kafka_DescribeUserScramCredentials_result_descriptions(const rd_kafka_DescribeUserScramCredentials_result_t *result, size_t *cntp)
Get an array of descriptions from a DescribeUserScramCredentials result.
RD_KAFKA_RESP_ERR_LISTENER_NOT_FOUND
@ RD_KAFKA_RESP_ERR_LISTENER_NOT_FOUND
Definition: rdkafka.h:579
RD_KAFKA_ADMIN_OP_DESCRIBETOPICS
@ RD_KAFKA_ADMIN_OP_DESCRIBETOPICS
Definition: rdkafka.h:6921
RD_KAFKA_VTYPE_TOPIC
@ RD_KAFKA_VTYPE_TOPIC
Definition: rdkafka.h:1119
rd_kafka_metadata_partition::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:4941
rd_kafka_AdminOptions_set_include_authorized_operations
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_include_authorized_operations(rd_kafka_AdminOptions_t *options, int true_or_false)
Whether broker should return authorized operations for the given resource in the DescribeConsumerGrou...
rd_kafka_msg_partitioner_random
RD_EXPORT int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Random partitioner.
RD_KAFKA_RESOURCE_BROKER
@ RD_KAFKA_RESOURCE_BROKER
Definition: rdkafka.h:7667
rd_kafka_send_offsets_to_transaction
RD_EXPORT rd_kafka_error_t * rd_kafka_send_offsets_to_transaction(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, const rd_kafka_consumer_group_metadata_t *cgmetadata, int timeout_ms)
Sends a list of topic partition offsets to the consumer group coordinator for cgmetadata,...
rd_kafka_ConfigResource_destroy
RD_EXPORT void rd_kafka_ConfigResource_destroy(rd_kafka_ConfigResource_t *config)
Destroy and free a ConfigResource object previously created with rd_kafka_ConfigResource_new()
RD_KAFKA_RESP_ERR_RESOURCE_NOT_FOUND
@ RD_KAFKA_RESP_ERR_RESOURCE_NOT_FOUND
Definition: rdkafka.h:620
rd_kafka_OffsetSpec_t
rd_kafka_OffsetSpec_t
Allows to specify the desired offsets when using ListOffsets.
Definition: rdkafka.h:9161
rd_kafka_metadata::brokers
struct rd_kafka_metadata_broker * brokers
Definition: rdkafka.h:4965
rd_kafka_topic_partition_s::metadata
void * metadata
Definition: rdkafka.h:903
rd_kafka_message_s::payload
void * payload
Definition: rdkafka.h:1443
rd_kafka_ConfigResource_error_string
const RD_EXPORT char * rd_kafka_ConfigResource_error_string(const rd_kafka_ConfigResource_t *config)
RD_KAFKA_CONFIG_SOURCE__CNT
@ RD_KAFKA_CONFIG_SOURCE__CNT
Definition: rdkafka.h:7577
RD_KAFKA_RESOURCE_GROUP
@ RD_KAFKA_RESOURCE_GROUP
Definition: rdkafka.h:7666
rd_kafka_DescribeConfigs_result_resources
const RD_EXPORT rd_kafka_ConfigResource_t ** rd_kafka_DescribeConfigs_result_resources(const rd_kafka_DescribeConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeConfigs result.
rd_kafka_DeleteRecords_t
struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t
Definition: rdkafka.h:8019
rd_kafka_mem_free
RD_EXPORT void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr)
Free pointer returned by librdkafka.
rd_kafka_event_DescribeUserScramCredentials_result
const RD_EXPORT rd_kafka_DescribeUserScramCredentials_result_t * rd_kafka_event_DescribeUserScramCredentials_result(rd_kafka_event_t *rkev)
Get DescribeUserScramCredentials result.
rd_kafka_type
RD_EXPORT rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk)
Returns Kafka handle type.
rd_kafka_conf_set_ssl_cert
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf, rd_kafka_cert_type_t cert_type, rd_kafka_cert_enc_t cert_enc, const void *buffer, size_t size, char *errstr, size_t errstr_size)
Set certificate/key cert_type from the cert_enc encoded memory at buffer of size bytes.
rd_kafka_group_member_info::member_assignment_size
int member_assignment_size
Definition: rdkafka.h:5084
rd_kafka_TopicDescription_name
const RD_EXPORT char * rd_kafka_TopicDescription_name(const rd_kafka_TopicDescription_t *topicdesc)
Gets the topic name for the topicdesc topic.
rd_kafka_group_info::broker
struct rd_kafka_metadata_broker broker
Definition: rdkafka.h:5106
RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:555
rd_kafka_Uuid_least_significant_bits
RD_EXPORT int64_t rd_kafka_Uuid_least_significant_bits(const rd_kafka_Uuid_t *uuid)
Gets least significant 64 bits for the given UUID.
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTH_DISABLED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTH_DISABLED
Definition: rdkafka.h:557
rd_kafka_brokers_add
RD_EXPORT int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist)
Adds one or more brokers to the kafka handle's list of initial bootstrap brokers.
rd_kafka_topic_partition_available
RD_EXPORT int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt, int32_t partition)
Check if partition is available (has a leader broker).
rd_kafka_queue_length
RD_EXPORT size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu)
RD_KAFKA_RESP_ERR_FETCH_SESSION_ID_NOT_FOUND
@ RD_KAFKA_RESP_ERR_FETCH_SESSION_ID_NOT_FOUND
Definition: rdkafka.h:575
RD_KAFKA_ACL_OPERATION_WRITE
@ RD_KAFKA_ACL_OPERATION_WRITE
Definition: rdkafka.h:7181
rd_kafka_message_s::offset
int64_t offset
Definition: rdkafka.h:1454
rd_kafka_DescribeTopics_result_topics
const RD_EXPORT rd_kafka_TopicDescription_t ** rd_kafka_DescribeTopics_result_topics(const rd_kafka_DescribeTopics_result_t *result, size_t *cntp)
Get an array of topic results from a DescribeTopics result.
rd_kafka_consumer_group_state_t
rd_kafka_consumer_group_state_t
Consumer group state.
Definition: rdkafka.h:5092
rd_kafka_queue_destroy
RD_EXPORT void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu)
RD_KAFKA_RESP_ERR__OUTDATED
@ RD_KAFKA_RESP_ERR__OUTDATED
Definition: rdkafka.h:353
rd_kafka_event_AlterUserScramCredentials_result
const RD_EXPORT rd_kafka_AlterUserScramCredentials_result_t * rd_kafka_event_AlterUserScramCredentials_result(rd_kafka_event_t *rkev)
Get AlterUserScramCredentials result.
rd_kafka_queue_yield
RD_EXPORT void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu)
Cancels the current rd_kafka_queue_poll() on rkqu.
rd_kafka_errno
RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void)
Returns the thread-local system errno.
rd_kafka_ListOffsetsResultInfo_t
struct rd_kafka_ListOffsetsResultInfo_s rd_kafka_ListOffsetsResultInfo_t
Information returned from a ListOffsets call for a specific rd_kafka_topic_partition_t.
Definition: rdkafka.h:9179
rd_kafka_message_latency
RD_EXPORT int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage)
Returns the latency for a produced message measured from the produce() call.
RD_KAFKA_RESOURCE_TOPIC
@ RD_KAFKA_RESOURCE_TOPIC
Definition: rdkafka.h:7665
rd_kafka_ResourcePatternType_name
const RD_EXPORT char * rd_kafka_ResourcePatternType_name(rd_kafka_ResourcePatternType_t resource_pattern_type)
RD_KAFKA_RESP_ERR_INVALID_TIMESTAMP
@ RD_KAFKA_RESP_ERR_INVALID_TIMESTAMP
Definition: rdkafka.h:493
rd_kafka_event_error_string
const RD_EXPORT char * rd_kafka_event_error_string(rd_kafka_event_t *rkev)
rd_kafka_pause_partitions
RD_EXPORT rd_kafka_resp_err_t rd_kafka_pause_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Pause producing or consumption for the provided list of partitions.
RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
@ RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafka.h:534
rd_kafka_DeleteRecords_result_offsets
const RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_DeleteRecords_result_offsets(const rd_kafka_DeleteRecords_result_t *result)
Get a list of topic and partition results from a DeleteRecords result. The returned objects will cont...
RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_OR_PART
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_OR_PART
Definition: rdkafka.h:424
RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
@ RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafka.h:444
rd_kafka_group_list_destroy
RD_EXPORT void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist)
Release list memory.
rd_kafka_event_debug_contexts
RD_EXPORT int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev, char *dst, size_t dstsize)
Extract log debug context from event.
RD_KAFKA_RESP_ERR_BROKER_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_BROKER_NOT_AVAILABLE
Definition: rdkafka.h:436
rd_kafka_header_get
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get(const rd_kafka_headers_t *hdrs, size_t idx, const char *name, const void **valuep, size_t *sizep)
Iterator for headers matching name.
rd_kafka_NewTopic_destroy_array
RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt)
Helper function to destroy all NewTopic objects in the new_topics array (of new_topic_cnt elements)....
rd_kafka_conf_set_socket_cb
RD_EXPORT void rd_kafka_conf_set_socket_cb(rd_kafka_conf_t *conf, int(*socket_cb)(int domain, int type, int protocol, void *opaque))
Set socket callback.
rd_kafka_event_ListConsumerGroups_result
const RD_EXPORT rd_kafka_ListConsumerGroups_result_t * rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev)
Get ListConsumerGroups result.
rd_kafka_event_error_is_fatal
RD_EXPORT int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR__AUTHENTICATION
@ RD_KAFKA_RESP_ERR__AUTHENTICATION
Definition: rdkafka.h:349
rd_kafka_AclBinding_t
struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t
ACL Binding is used to create access control lists.
Definition: rdkafka.h:9486
rd_kafka_UserScramCredentialsDescription_scramcredentialinfo_count
RD_EXPORT size_t rd_kafka_UserScramCredentialsDescription_scramcredentialinfo_count(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the count of ScramCredentialInfos of a UserScramCredentialsDescription.
rd_kafka_DeleteConsumerGroupOffsets_result_t
rd_kafka_event_t rd_kafka_DeleteConsumerGroupOffsets_result_t
Definition: rdkafka.h:5725
rd_kafka_consume_stop
RD_EXPORT int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition)
Stop consuming messages for topic rkt and partition, purging all messages currently in the local queu...
rd_kafka_conf_get
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve configuration value for property name.
RD_KAFKA_RESP_ERR_LEADER_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:428
RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
@ RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
Definition: rdkafka.h:329
RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
@ RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafka.h:633
rd_kafka_topic_partition_list_s::elems
rd_kafka_topic_partition_t * elems
Definition: rdkafka.h:953
RD_KAFKA_TIMESTAMP_CREATE_TIME
@ RD_KAFKA_TIMESTAMP_CREATE_TIME
Definition: rdkafka.h:222
rd_kafka_metadata_partition::id
int32_t id
Definition: rdkafka.h:4940
RD_KAFKA_RESP_ERR__RESOLVE
@ RD_KAFKA_RESP_ERR__RESOLVE
Definition: rdkafka.h:298
rd_kafka_ConsumerGroupDescription_member
const RD_EXPORT rd_kafka_MemberDescription_t * rd_kafka_ConsumerGroupDescription_member(const rd_kafka_ConsumerGroupDescription_t *grpdesc, size_t idx)
Gets a member of grpdesc group.
rd_kafka_TopicDescription_partitions
const RD_EXPORT rd_kafka_TopicPartitionInfo_t ** rd_kafka_TopicDescription_partitions(const rd_kafka_TopicDescription_t *topicdesc, size_t *cntp)
Gets an array of partitions for the topicdesc topic.
RD_KAFKA_RESP_ERR_INVALID_COMMIT_OFFSET_SIZE
@ RD_KAFKA_RESP_ERR_INVALID_COMMIT_OFFSET_SIZE
Definition: rdkafka.h:485
RD_KAFKA_ADMIN_OP_DELETEACLS
@ RD_KAFKA_ADMIN_OP_DELETEACLS
Definition: rdkafka.h:6908
rd_kafka_assign
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Atomic assignment of partitions to consume.
rd_kafka_DeleteAcls_result_response_t
struct rd_kafka_DeleteAcls_result_response_s rd_kafka_DeleteAcls_result_response_t
Definition: rdkafka.h:9737
RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafka.h:602
rd_kafka_acl_result_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres)
rd_kafka_interceptor_add_on_thread_exit
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_exit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_exit_t *on_thread_exit, void *ic_opaque)
Append an on_thread_exit() interceptor.
RD_KAFKA_ACL_OPERATION_ALTER_CONFIGS
@ RD_KAFKA_ACL_OPERATION_ALTER_CONFIGS
Definition: rdkafka.h:7190
rd_kafka_NewPartitions_t
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:7426
rd_kafka_topic_partition_list_del_by_idx
RD_EXPORT int rd_kafka_topic_partition_list_del_by_idx(rd_kafka_topic_partition_list_t *rktparlist, int idx)
Delete partition from list by elems[] index.
rd_kafka_UserScramCredentialAlteration_t
struct rd_kafka_UserScramCredentialAlteration_s rd_kafka_UserScramCredentialAlteration_t
A request to alter a user's SASL/SCRAM credentials.
Definition: rdkafka.h:9353
rd_kafka_TopicDescription_authorized_operations
const RD_EXPORT rd_kafka_AclOperation_t * rd_kafka_TopicDescription_authorized_operations(const rd_kafka_TopicDescription_t *topicdesc, size_t *cntp)
Gets the topic authorized ACL operations for the topicdesc topic.
rd_kafka_metadata
RD_EXPORT rd_kafka_resp_err_t rd_kafka_metadata(rd_kafka_t *rk, int all_topics, rd_kafka_topic_t *only_rkt, const struct rd_kafka_metadata **metadatap, int timeout_ms)
Request Metadata from broker.
RD_KAFKA_RESP_ERR_OPERATION_NOT_ATTEMPTED
@ RD_KAFKA_RESP_ERR_OPERATION_NOT_ATTEMPTED
Definition: rdkafka.h:544
RD_KAFKA_VTYPE_TIMESTAMP
@ RD_KAFKA_VTYPE_TIMESTAMP
Definition: rdkafka.h:1130
rd_kafka_AdminOptions_t
struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t
AdminOptions provides a generic mechanism for setting optional parameters for the Admin API requests.
Definition: rdkafka.h:6939
rd_kafka_Uuid_base64str
const RD_EXPORT char * rd_kafka_Uuid_base64str(const rd_kafka_Uuid_t *uuid)
Computes base64 encoding for the given uuid string.
RD_KAFKA_RESP_ERR_DUPLICATE_SEQUENCE_NUMBER
@ RD_KAFKA_RESP_ERR_DUPLICATE_SEQUENCE_NUMBER
Definition: rdkafka.h:521
RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
Definition: rdkafka.h:483
rd_kafka_conf_dup
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup(const rd_kafka_conf_t *conf)
Creates a copy/duplicate of configuration object conf.
rd_kafka_AlterUserScramCredentials_result_response_user
const RD_EXPORT char * rd_kafka_AlterUserScramCredentials_result_response_user(const rd_kafka_AlterUserScramCredentials_result_response_t *response)
Returns the username for a rd_kafka_AlterUserScramCredentials_result_response.
rd_kafka_UserScramCredentialsDescription_user
const RD_EXPORT char * rd_kafka_UserScramCredentialsDescription_user(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the username of a UserScramCredentialsDescription.
rd_kafka_ConfigResource_destroy_array
RD_EXPORT void rd_kafka_ConfigResource_destroy_array(rd_kafka_ConfigResource_t **config, size_t config_cnt)
Helper function to destroy all ConfigResource objects in the configs array (of config_cnt elements)....
rd_kafka_get_debug_contexts
const RD_EXPORT char * rd_kafka_get_debug_contexts(void)
Retrieve supported debug contexts for use with the "debug" configuration property....
rd_kafka_group_member_info::member_metadata
void * member_metadata
Definition: rdkafka.h:5079
rd_kafka_consumer_group_metadata_new_with_genid
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new_with_genid(const char *group_id, int32_t generation_id, const char *member_id, const char *group_instance_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
RD_KAFKA_RESP_ERR_ILLEGAL_GENERATION
@ RD_KAFKA_RESP_ERR_ILLEGAL_GENERATION
Definition: rdkafka.h:473
rd_kafka_topic_partition_get_leader_epoch
RD_EXPORT int32_t rd_kafka_topic_partition_get_leader_epoch(const rd_kafka_topic_partition_t *rktpar)
rd_kafka_offset_store_message
RD_EXPORT rd_kafka_error_t * rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage)
Store offset +1 for the consumed message.
RD_KAFKA_RESP_ERR__CONFLICT
@ RD_KAFKA_RESP_ERR__CONFLICT
Definition: rdkafka.h:341
RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
@ RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafka.h:465
rd_kafka_TopicDescription_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_TopicDescription_error(const rd_kafka_TopicDescription_t *topicdesc)
Gets the error for the topicdesc topic.
rd_kafka_topic_conf_set_opaque
RD_EXPORT void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf, void *rkt_opaque)
Sets the application's opaque pointer that will be passed to all topic callbacks as the rkt_opaque ar...
rd_kafka_DescribeConfigs_result_t
rd_kafka_event_t rd_kafka_DescribeConfigs_result_t
Definition: rdkafka.h:5715
RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafka.h:523
rd_kafka_Uuid_new
RD_EXPORT rd_kafka_Uuid_t * rd_kafka_Uuid_new(int64_t most_significant_bits, int64_t least_significant_bits)
Creates a new UUID.
rd_kafka_clusterid
RD_EXPORT char * rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms)
Returns the ClusterId as reported in broker metadata.
RD_KAFKA_RESP_ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
Definition: rdkafka.h:540
RD_KAFKA_ADMIN_OP_DELETETOPICS
@ RD_KAFKA_ADMIN_OP_DELETETOPICS
Definition: rdkafka.h:6898
rd_kafka_AlterUserScramCredentials_result_response_t
struct rd_kafka_AlterUserScramCredentials_result_response_s rd_kafka_AlterUserScramCredentials_result_response_t
Result of a single user SCRAM alteration.
Definition: rdkafka.h:9419
rd_kafka_topic_conf_set_msg_order_cmp
RD_EXPORT void rd_kafka_topic_conf_set_msg_order_cmp(rd_kafka_topic_conf_t *topic_conf, int(*msg_order_cmp)(const rd_kafka_message_t *a, const rd_kafka_message_t *b))
Producer: Set message queueing order comparator callback.
rd_kafka_queue_poll_callback
RD_EXPORT int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for events served through callbacks for max timeout_ms.
rd_kafka_conf
const RD_EXPORT rd_kafka_conf_t * rd_kafka_conf(rd_kafka_t *rk)
rd_kafka_set_logger
RD_EXPORT RD_DEPRECATED void rd_kafka_set_logger(rd_kafka_t *rk, void(*func)(const rd_kafka_t *rk, int level, const char *fac, const char *buf))
Set logger function.
rd_kafka_AlterConsumerGroupOffsets_result_groups
const RD_EXPORT rd_kafka_group_result_t ** rd_kafka_AlterConsumerGroupOffsets_result_groups(const rd_kafka_AlterConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a AlterConsumerGroupOffsets result.
rd_kafka_DeleteConsumerGroupOffsets_new
RD_EXPORT rd_kafka_DeleteConsumerGroupOffsets_t * rd_kafka_DeleteConsumerGroupOffsets_new(const char *group, const rd_kafka_topic_partition_list_t *partitions)
Create a new DeleteConsumerGroupOffsets object. This object is later passed to rd_kafka_DeleteConsume...
RD_KAFKA_RESP_ERR__APPLICATION
@ RD_KAFKA_RESP_ERR__APPLICATION
Definition: rdkafka.h:401
rd_kafka_conf_set_rebalance_cb
RD_EXPORT void rd_kafka_conf_set_rebalance_cb(rd_kafka_conf_t *conf, void(*rebalance_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *partitions, void *opaque))
Consumer: Set rebalance callback for use with coordinated consumer group balancing.
rd_kafka_IncrementalAlterConfigs_result_resources
const RD_EXPORT rd_kafka_ConfigResource_t ** rd_kafka_IncrementalAlterConfigs_result_resources(const rd_kafka_IncrementalAlterConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a IncrementalAlterConfigs result.
rd_kafka_AclOperation_name
const RD_EXPORT char * rd_kafka_AclOperation_name(rd_kafka_AclOperation_t acl_operation)
rd_kafka_topic_partition_s::metadata_size
size_t metadata_size
Definition: rdkafka.h:904
rd_kafka_DescribeConsumerGroups_result_t
rd_kafka_event_t rd_kafka_DescribeConsumerGroups_result_t
Definition: rdkafka.h:5721
rd_kafka_AlterConsumerGroupOffsets
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Alter committed offsets for a set of partitions in a consumer group. This will succeed at the partiti...
RD_KAFKA_RESP_ERR__DESTROY
@ RD_KAFKA_RESP_ERR__DESTROY
Definition: rdkafka.h:290
rd_kafka_conf_interceptor_add_on_new
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_new_t *on_new, void *ic_opaque)
Append an on_new() interceptor.
rd_kafka_conf_set_closesocket_cb
RD_EXPORT void rd_kafka_conf_set_closesocket_cb(rd_kafka_conf_t *conf, int(*closesocket_cb)(int sockfd, void *opaque))
Set close socket callback.
rd_kafka_NewTopic_set_config
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_config(rd_kafka_NewTopic_t *new_topic, const char *name, const char *value)
Set (broker-side) topic configuration name/value pair.
rd_kafka_ListConsumerGroups_result_valid
const RD_EXPORT rd_kafka_ConsumerGroupListing_t ** rd_kafka_ListConsumerGroups_result_valid(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of valid list groups from a ListConsumerGroups result.
rd_kafka_incremental_unassign
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_unassign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally remove partitions from the current assignment.
rd_kafka_event_DeleteAcls_result
const RD_EXPORT rd_kafka_DeleteAcls_result_t * rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR_MEMBER_ID_REQUIRED
@ RD_KAFKA_RESP_ERR_MEMBER_ID_REQUIRED
Definition: rdkafka.h:593
rd_kafka_msg_partitioner_murmur2
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Murmur2 partitioner (Java compatible).
rd_kafka_Node_id
RD_EXPORT int rd_kafka_Node_id(const rd_kafka_Node_t *node)
Get the id of node.
RD_KAFKA_ACL_OPERATION_ALTER
@ RD_KAFKA_ACL_OPERATION_ALTER
Definition: rdkafka.h:7184
rd_kafka_consume_batch_queue
RD_EXPORT ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume batch of messages from queue.
rd_kafka_AclBinding_permission_type
RD_EXPORT rd_kafka_AclPermissionType_t rd_kafka_AclBinding_permission_type(const rd_kafka_AclBinding_t *acl)
rd_kafka_group_list
List of groups.
Definition: rdkafka.h:5121
rd_kafka_queue_poll
RD_EXPORT rd_kafka_event_t * rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for an event for max timeout_ms.
rd_kafka_TopicPartitionInfo_isr
const RD_EXPORT rd_kafka_Node_t ** rd_kafka_TopicPartitionInfo_isr(const rd_kafka_TopicPartitionInfo_t *partition, size_t *cntp)
Gets the partition in-sync replicas for partition.
rd_kafka_Node_host
const RD_EXPORT char * rd_kafka_Node_host(const rd_kafka_Node_t *node)
Get the host of node.
rd_kafka_TopicDescription_t
struct rd_kafka_TopicDescription_s rd_kafka_TopicDescription_t
DescribeTopics result type.
Definition: rdkafka.h:8124
rd_kafka_topic_partition_list_s::cnt
int cnt
Definition: rdkafka.h:951
RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
@ RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
Definition: rdkafka.h:393
rd_kafka_topic_partition_list_del
RD_EXPORT int rd_kafka_topic_partition_list_del(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Delete partition from list.
rd_kafka_AdminOptions_set_opaque
RD_EXPORT void rd_kafka_AdminOptions_set_opaque(rd_kafka_AdminOptions_t *options, void *ev_opaque)
Set application opaque value that can be extracted from the result event using rd_kafka_event_opaque(...
rd_kafka_consumer_close_queue
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_close_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Asynchronously close the consumer.
rd_kafka_thread_type_t
rd_kafka_thread_type_t
librdkafka internal thread type.
Definition: rdkafka.h:5350
rd_kafka_Node_t
struct rd_kafka_Node_s rd_kafka_Node_t
Node (broker) information.
Definition: rdkafka.h:5011
rd_kafka_get_watermark_offsets
RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high)
Get last known low (oldest/beginning) and high (newest/end) offsets for partition.
rd_kafka_ListConsumerGroups_result_t
rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t
Definition: rdkafka.h:5719
rd_kafka_assignment_lost
RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk)
Check whether the consumer considers the current assignment to have been lost involuntarily....
rd_kafka_AclBinding_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_AclBinding_error(const rd_kafka_AclBinding_t *acl)
rd_kafka_UserScramCredentialsDescription_scramcredentialinfo
const RD_EXPORT rd_kafka_ScramCredentialInfo_t * rd_kafka_UserScramCredentialsDescription_scramcredentialinfo(const rd_kafka_UserScramCredentialsDescription_t *description, size_t idx)
Returns the ScramCredentialInfo at index idx of UserScramCredentialsDescription.
RD_KAFKA_RESP_ERR_INVALID_GROUP_ID
@ RD_KAFKA_RESP_ERR_INVALID_GROUP_ID
Definition: rdkafka.h:477
rd_kafka_topic_conf_dup
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf)
Creates a copy/duplicate of topic configuration object conf.
RD_KAFKA_RESP_ERR__LOG_TRUNCATION
@ RD_KAFKA_RESP_ERR__LOG_TRUNCATION
Definition: rdkafka.h:409
rd_kafka_ConfigEntry_is_synonym
RD_EXPORT int rd_kafka_ConfigEntry_is_synonym(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_ConfigSource_name
const RD_EXPORT char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
@ RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
Definition: rdkafka.h:6899
rd_kafka_queue_get_main
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_main(rd_kafka_t *rk)
rd_kafka_event_type_t
int rd_kafka_event_type_t
Event types.
Definition: rdkafka.h:5413
RD_KAFKA_CONF_INVALID
@ RD_KAFKA_CONF_INVALID
Definition: rdkafka.h:1718
rd_kafka_metadata_topic::partition_cnt
int partition_cnt
Definition: rdkafka.h:4954
rd_kafka_vu_s
VTYPE + argument container for use with rd_kafka_produce_va()
Definition: rdkafka.h:1143
RD_KAFKA_ADMIN_OP_CREATETOPICS
@ RD_KAFKA_ADMIN_OP_CREATETOPICS
Definition: rdkafka.h:6897
rd_kafka_ListOffsetsResultInfo_topic_partition
const RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_ListOffsetsResultInfo_topic_partition(const rd_kafka_ListOffsetsResultInfo_t *result_info)
Returns the topic partition of the passed result_info.
rd_kafka_Node_rack
const RD_EXPORT char * rd_kafka_Node_rack(const rd_kafka_Node_t *node)
Get the rack of node.
RD_KAFKA_CONF_UNKNOWN
@ RD_KAFKA_CONF_UNKNOWN
Definition: rdkafka.h:1717
RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
@ RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
Definition: rdkafka.h:463
RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
@ RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafka.h:597
RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
@ RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafka.h:475
rd_kafka_queue_get_partition
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_partition(rd_kafka_t *rk, const char *topic, int32_t partition)
RD_KAFKA_RESP_ERR_POLICY_VIOLATION
@ RD_KAFKA_RESP_ERR_POLICY_VIOLATION
Definition: rdkafka.h:517
rd_kafka_list_groups
RD_EXPORT rd_kafka_resp_err_t rd_kafka_list_groups(rd_kafka_t *rk, const char *group, const struct rd_kafka_group_list **grplistp, int timeout_ms)
List and describe client groups in cluster.
rd_kafka_metadata::topics
struct rd_kafka_metadata_topic * topics
Definition: rdkafka.h:4968
rd_kafka_error_string
const RD_EXPORT char * rd_kafka_error_string(const rd_kafka_error_t *error)
RD_KAFKA_RESOURCE_ANY
@ RD_KAFKA_RESOURCE_ANY
Definition: rdkafka.h:7664
RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
@ RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafka.h:585
rd_kafka_queue_get_sasl
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_sasl(rd_kafka_t *rk)
RD_KAFKA_RESP_ERR__UNSUPPORTED_FEATURE
@ RD_KAFKA_RESP_ERR__UNSUPPORTED_FEATURE
Definition: rdkafka.h:357
rd_kafka_conf_set_offset_commit_cb
RD_EXPORT void rd_kafka_conf_set_offset_commit_cb(rd_kafka_conf_t *conf, void(*offset_commit_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *opaque))
Consumer: Set offset commit callback for use with consumer groups.
RD_KAFKA_RESP_ERR__PURGE_QUEUE
@ RD_KAFKA_RESP_ERR__PURGE_QUEUE
Definition: rdkafka.h:383
rd_kafka_DeleteConsumerGroupOffsets_result_groups
const RD_EXPORT rd_kafka_group_result_t ** rd_kafka_DeleteConsumerGroupOffsets_result_groups(const rd_kafka_DeleteConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a DeleteConsumerGroupOffsets result.
RD_KAFKA_CERT_PUBLIC_KEY
@ RD_KAFKA_CERT_PUBLIC_KEY
Definition: rdkafka.h:2454
rd_kafka_conf_set_events
RD_EXPORT void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events)
Enable event sourcing. events is a bitmask of RD_KAFKA_EVENT_* of events to enable for consumption by...
RD_KAFKA_ACL_OPERATION_IDEMPOTENT_WRITE
@ RD_KAFKA_ACL_OPERATION_IDEMPOTENT_WRITE
Definition: rdkafka.h:7192
rd_kafka_MemberDescription_client_id
const RD_EXPORT char * rd_kafka_MemberDescription_client_id(const rd_kafka_MemberDescription_t *member)
Gets client id of member.
rd_kafka_sasl_background_callbacks_enable
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk)
Enable SASL OAUTHBEARER refresh callbacks on the librdkafka background thread.
rd_kafka_flush
RD_EXPORT rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms)
Wait until all outstanding produce requests, et.al, are completed. This should typically be done prio...
rd_kafka_TopicDescription_is_internal
RD_EXPORT int rd_kafka_TopicDescription_is_internal(const rd_kafka_TopicDescription_t *topicdesc)
Gets if the topicdesc topic is internal.
rd_kafka_metadata_destroy
RD_EXPORT void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata)
Release metadata memory.
rd_kafka_topic_partition_s::partition
int32_t partition
Definition: rdkafka.h:901
RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
@ RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6918
rd_kafka_err2str
const RD_EXPORT char * rd_kafka_err2str(rd_kafka_resp_err_t err)
Returns a human readable representation of a kafka error.
RD_KAFKA_RESP_ERR__INVALID_TYPE
@ RD_KAFKA_RESP_ERR__INVALID_TYPE
Definition: rdkafka.h:379
rd_kafka_queue_io_event_enable
RD_EXPORT void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu, int fd, const void *payload, size_t size)
Enable IO event triggering for queue.
rd_kafka_topic_partition_s::_private
void * _private
Definition: rdkafka.h:907
rd_kafka_AclBinding_resource_pattern_type
RD_EXPORT rd_kafka_ResourcePatternType_t rd_kafka_AclBinding_resource_pattern_type(const rd_kafka_AclBinding_t *acl)
RD_KAFKA_ADMIN_OP_DESCRIBECLUSTER
@ RD_KAFKA_ADMIN_OP_DESCRIBECLUSTER
Definition: rdkafka.h:6922
RD_KAFKA_VTYPE_VALUE
@ RD_KAFKA_VTYPE_VALUE
Definition: rdkafka.h:1122
rd_kafka_ListOffsets
RD_EXPORT void rd_kafka_ListOffsets(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *topic_partitions, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List offsets for the specified topic_partitions. This operation enables to find the beginning offset,...
RD_KAFKA_RESP_ERR_NO_ERROR
@ RD_KAFKA_RESP_ERR_NO_ERROR
Definition: rdkafka.h:418
RD_KAFKA_RESOURCE_UNKNOWN
@ RD_KAFKA_RESOURCE_UNKNOWN
Definition: rdkafka.h:7663
rd_kafka_assignment
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assignment(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **partitions)
Returns the current partition assignment as set by rd_kafka_assign() or rd_kafka_incremental_assign()...
rd_kafka_TopicPartitionInfo_leader
const RD_EXPORT rd_kafka_Node_t * rd_kafka_TopicPartitionInfo_leader(const rd_kafka_TopicPartitionInfo_t *partition)
Gets the partition leader for partition.
rd_kafka_TopicPartitionInfo_partition
const RD_EXPORT int rd_kafka_TopicPartitionInfo_partition(const rd_kafka_TopicPartitionInfo_t *partition)
Gets the partition id for partition.
rd_kafka_event_message_count
RD_EXPORT size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev)
rd_kafka_DescribeConsumerGroups_result_groups
const RD_EXPORT rd_kafka_ConsumerGroupDescription_t ** rd_kafka_DescribeConsumerGroups_result_groups(const rd_kafka_DescribeConsumerGroups_result_t *result, size_t *cntp)
Get an array of group results from a DescribeConsumerGroups result.
rd_kafka_message_s::key
void * key
Definition: rdkafka.h:1450
rd_kafka_AlterUserScramCredentials
RD_EXPORT void rd_kafka_AlterUserScramCredentials(rd_kafka_t *rk, rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Alter SASL/SCRAM credentials. This operation is supported by brokers with version 2....
rd_kafka_cert_enc_t
rd_kafka_cert_enc_t
SSL certificate encoding.
Definition: rdkafka.h:2467
rd_kafka_ScramCredentialInfo_mechanism
RD_EXPORT rd_kafka_ScramMechanism_t rd_kafka_ScramCredentialInfo_mechanism(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the mechanism of a given ScramCredentialInfo.
RD_KAFKA_RESP_ERR_INCONSISTENT_VOTER_SET
@ RD_KAFKA_RESP_ERR_INCONSISTENT_VOTER_SET
Definition: rdkafka.h:627
RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
@ RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
Definition: rdkafka.h:7188
rd_kafka_event_destroy
RD_EXPORT void rd_kafka_event_destroy(rd_kafka_event_t *rkev)
Destroy an event.
rd_kafka_event_DescribeCluster_result
const RD_EXPORT rd_kafka_DescribeCluster_result_t * rd_kafka_event_DescribeCluster_result(rd_kafka_event_t *rkev)
Get DescribeCluster result.
rd_kafka_NewPartitions_set_replica_assignment
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewPartitions_set_replica_assignment(rd_kafka_NewPartitions_t *new_parts, int32_t new_partition_idx, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker id) assignment for new_partition_idx to the replica set in broker_ids (of bro...
rd_kafka_err2name
const RD_EXPORT char * rd_kafka_err2name(rd_kafka_resp_err_t err)
Returns the error code name (enum name).
RD_KAFKA_VTYPE_MSGFLAGS
@ RD_KAFKA_VTYPE_MSGFLAGS
Definition: rdkafka.h:1129
rd_kafka_error_destroy
RD_EXPORT void rd_kafka_error_destroy(rd_kafka_error_t *error)
Free and destroy an error object.
RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION
@ RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION
Definition: rdkafka.h:307
rd_kafka_DescribeAcls_result_t
rd_kafka_event_t rd_kafka_DescribeAcls_result_t
Definition: rdkafka.h:5705
rd_kafka_DeleteGroup_t
struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t
Definition: rdkafka.h:8797
rd_kafka_IncrementalAlterConfigs
RD_EXPORT void rd_kafka_IncrementalAlterConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Incrementally update the configuration for the specified resources. Updates are not transactional so ...
rd_kafka_message_s::_private
void * _private
Definition: rdkafka.h:1462
rd_kafka_event_DescribeConfigs_result
const RD_EXPORT rd_kafka_DescribeConfigs_result_t * rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev)
Get DescribeConfigs result.
rd_kafka_ConfigResource_type
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_ConfigResource_type(const rd_kafka_ConfigResource_t *config)
rd_kafka_TopicCollection_of_topic_names
RD_EXPORT rd_kafka_TopicCollection_t * rd_kafka_TopicCollection_of_topic_names(const char **topics, size_t topics_cnt)
Creates a new TopicCollection for passing to rd_kafka_DescribeTopics.
rd_kafka_AclBinding_new
RD_EXPORT rd_kafka_AclBinding_t * rd_kafka_AclBinding_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBinding object. This object is later passed to rd_kafka_CreateAcls().
rd_kafka_commit_queue
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_queue(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, rd_kafka_queue_t *rkqu, void(*cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *commit_opaque), void *commit_opaque)
Commit offsets on broker for the provided list of partitions.
rd_kafka_DeleteAcls_result_response_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_DeleteAcls_result_response_error(const rd_kafka_DeleteAcls_result_response_t *result_response)
rd_kafka_metadata_partition::replicas
int32_t * replicas
Definition: rdkafka.h:4944
RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
@ RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
Definition: rdkafka.h:573
rd_kafka_interceptor_add_on_thread_start
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_start(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_start_t *on_thread_start, void *ic_opaque)
Append an on_thread_start() interceptor.
rd_kafka_version_str
const RD_EXPORT char * rd_kafka_version_str(void)
Returns the librdkafka version as string.
rd_kafka_event_DeleteTopics_result
const RD_EXPORT rd_kafka_DeleteTopics_result_t * rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev)
Get DeleteTopics result.
RD_KAFKA_RESP_ERR__NOENT
@ RD_KAFKA_RESP_ERR__NOENT
Definition: rdkafka.h:375
rd_kafka_interceptor_add_on_destroy
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_destroy(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_destroy_t *on_destroy, void *ic_opaque)
Append an on_destroy() interceptor.
RD_KAFKA_RESP_ERR_FENCED_LEADER_EPOCH
@ RD_KAFKA_RESP_ERR_FENCED_LEADER_EPOCH
Definition: rdkafka.h:583
rd_kafka_timestamp_type_t
rd_kafka_timestamp_type_t
Definition: rdkafka.h:220
rd_kafka_headers_copy
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_copy(const rd_kafka_headers_t *src)
Make a copy of headers list src.
rd_kafka_committed
RD_EXPORT rd_kafka_resp_err_t rd_kafka_committed(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Retrieve committed offsets for topics+partitions.
rd_kafka_group_member_info::member_assignment
void * member_assignment
Definition: rdkafka.h:5082
RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
@ RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
Definition: rdkafka.h:339
rd_kafka_metadata
Metadata container.
Definition: rdkafka.h:4963
RD_KAFKA_RESP_ERR__FENCED
@ RD_KAFKA_RESP_ERR__FENCED
Definition: rdkafka.h:399
rd_kafka_interceptor_f_on_new_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_new_t(rd_kafka_t *rk, const rd_kafka_conf_t *conf, void *ic_opaque, char *errstr, size_t errstr_size)
on_new() is called from rd_kafka_new() prior toreturning the newly created client instance to the app...
Definition: rdkafka.h:6236
rd_kafka_group_info
Group information.
Definition: rdkafka.h:5105
RD_KAFKA_CERT_ENC_PEM
@ RD_KAFKA_CERT_ENC_PEM
Definition: rdkafka.h:2470
rd_kafka_DescribeAcls_result_acls
const RD_EXPORT rd_kafka_AclBinding_t ** rd_kafka_DescribeAcls_result_acls(const rd_kafka_DescribeAcls_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeAcls result.
rd_kafka_AlterConsumerGroupOffsets_new
RD_EXPORT rd_kafka_AlterConsumerGroupOffsets_t * rd_kafka_AlterConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new AlterConsumerGroupOffsets object. This object is later passed to rd_kafka_AlterConsumerG...
rd_kafka_DeleteRecords_destroy
RD_EXPORT void rd_kafka_DeleteRecords_destroy(rd_kafka_DeleteRecords_t *del_records)
Destroy and free a DeleteRecords object previously created with rd_kafka_DeleteRecords_new()
RD_KAFKA_VTYPE_HEADER
@ RD_KAFKA_VTYPE_HEADER
Definition: rdkafka.h:1131
rd_kafka_err_desc
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:644
RD_KAFKA_RESP_ERR_INVALID_REQUEST
@ RD_KAFKA_RESP_ERR_INVALID_REQUEST
Definition: rdkafka.h:513
rd_kafka_NewTopic_set_replica_assignment
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_replica_assignment(rd_kafka_NewTopic_t *new_topic, int32_t partition, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker) assignment for partition to the replica set in broker_ids (of broker_id_cnt ...
rd_kafka_error_is_fatal
RD_EXPORT int rd_kafka_error_is_fatal(const rd_kafka_error_t *error)
rd_kafka_message_s::partition
int32_t partition
Definition: rdkafka.h:1442
rd_kafka_errno2err
RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox)
Converts the system errno value errnox to a rd_kafka_resp_err_t error code upon failure from the foll...
rd_kafka_event_DescribeConsumerGroups_result
const RD_EXPORT rd_kafka_DescribeConsumerGroups_result_t * rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev)
Get DescribeConsumerGroups result.
RD_KAFKA_RESOURCE_PATTERN_PREFIXED
@ RD_KAFKA_RESOURCE_PATTERN_PREFIXED
Definition: rdkafka.h:7685
RD_KAFKA_RESP_ERR_NOT_COORDINATOR
@ RD_KAFKA_RESP_ERR_NOT_COORDINATOR
Definition: rdkafka.h:458
RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
Definition: rdkafka.h:442
rd_kafka_topic_partition_list_s::size
int size
Definition: rdkafka.h:952
rd_kafka_ConfigEntry_t
struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t
Definition: rdkafka.h:7589
rd_kafka_DeleteConsumerGroupOffsets_destroy_array
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy_array(rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffset_cnt)
Helper function to destroy all DeleteConsumerGroupOffsets objects in the del_grpoffsets array (of del...
rd_kafka_NewPartitions_destroy
RD_EXPORT void rd_kafka_NewPartitions_destroy(rd_kafka_NewPartitions_t *new_parts)
Destroy and free a NewPartitions object previously created with rd_kafka_NewPartitions_new()
rd_kafka_UserScramCredentialAlteration_destroy_array
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy_array(rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt)
Destroys an array of UserScramCredentialAlteration.
rd_kafka_event_message_array
RD_EXPORT size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev, const rd_kafka_message_t **rkmessages, size_t size)
Extacts size message(s) from the event into the pre-allocated array rkmessages.
rd_kafka_message_broker_id
RD_EXPORT int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage)
Returns the broker id of the broker the message was produced to or fetched from.
rd_kafka_topic_partition_list_find
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_find(const rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Find element by topic and partition.
RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
@ RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafka.h:221
RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
@ RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
Definition: rdkafka.h:7186
rd_kafka_UserScramCredentialsDescription_t
struct rd_kafka_UserScramCredentialsDescription_s rd_kafka_UserScramCredentialsDescription_t
Representation of all SASL/SCRAM credentials associated with a user that can be retrieved,...
Definition: rdkafka.h:9281
rd_kafka_ConfigEntry_name
const RD_EXPORT char * rd_kafka_ConfigEntry_name(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_headers_destroy
RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs)
Destroy the headers list. The object and any returned value pointers are not usable after this call.
rd_kafka_consumer_poll
RD_EXPORT rd_kafka_message_t * rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms)
Poll the consumer for messages or events.
rd_kafka_UserScramCredentialUpsertion_new
RD_EXPORT rd_kafka_UserScramCredentialAlteration_t * rd_kafka_UserScramCredentialUpsertion_new(const char *username, rd_kafka_ScramMechanism_t mechanism, int32_t iterations, const unsigned char *password, size_t password_size, const unsigned char *salt, size_t salt_size)
Allocates a new UserScramCredentialUpsertion given its fields. If salt isn't given a 64 B salt is gen...
rd_kafka_consumer_group_metadata_destroy
RD_EXPORT void rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *)
Frees the consumer group metadata object as returned by rd_kafka_consumer_group_metadata().
RD_KAFKA_RESP_ERR__END
@ RD_KAFKA_RESP_ERR__END
Definition: rdkafka.h:412
RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafka.h:487
rd_kafka_event_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev)
rd_kafka_begin_transaction
RD_EXPORT rd_kafka_error_t * rd_kafka_begin_transaction(rd_kafka_t *rk)
Begin a new transaction.
rd_kafka_conf_set_opaque
RD_EXPORT void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque)
Sets the application's opaque pointer that will be passed to callbacks.
RD_KAFKA_RESP_ERR_REPLICA_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_REPLICA_NOT_AVAILABLE
Definition: rdkafka.h:438
RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
@ RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
Definition: rdkafka.h:471
rd_kafka_AclBinding_destroy_array
RD_EXPORT void rd_kafka_AclBinding_destroy_array(rd_kafka_AclBinding_t **acl_bindings, size_t acl_bindings_cnt)
Helper function to destroy all AclBinding objects in the acl_bindings array (of acl_bindings_cnt elem...
rd_kafka_AclBinding_host
const RD_EXPORT char * rd_kafka_AclBinding_host(const rd_kafka_AclBinding_t *acl)
RD_KAFKA_ACL_OPERATION_ANY
@ RD_KAFKA_ACL_OPERATION_ANY
Definition: rdkafka.h:7177
rd_kafka_conf_set_dr_cb
RD_EXPORT void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf, void(*dr_cb)(rd_kafka_t *rk, void *payload, size_t len, rd_kafka_resp_err_t err, void *opaque, void *msg_opaque))
rd_kafka_AclBinding_destroy
RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding)
Destroy and free an AclBinding object previously created with rd_kafka_AclBinding_new()
rd_kafka_DeleteTopic_destroy
RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic)
Destroy and free a DeleteTopic object previously created with rd_kafka_DeleteTopic_new()
rd_kafka_ListConsumerGroupOffsets
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List committed offsets for a set of partitions in a consumer group.
rd_kafka_AdminOptions_set_broker
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_broker(rd_kafka_AdminOptions_t *options, int32_t broker_id, char *errstr, size_t errstr_size)
Override what broker the Admin request will be sent to.
RD_KAFKA_ACL_PERMISSION_TYPE_ANY
@ RD_KAFKA_ACL_PERMISSION_TYPE_ANY
Definition: rdkafka.h:9513
RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
@ RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
Definition: rdkafka.h:296
rd_kafka_ConfigResource_name
const RD_EXPORT char * rd_kafka_ConfigResource_name(const rd_kafka_ConfigResource_t *config)
rd_kafka_ConfigResource_t
struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t
Definition: rdkafka.h:7714
RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
@ RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafka.h:519
rd_kafka_ConsumerGroupDescription_member_count
RD_EXPORT size_t rd_kafka_ConsumerGroupDescription_member_count(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the members count of grpdesc group.
RD_KAFKA_RESOURCE_PATTERN_MATCH
@ RD_KAFKA_RESOURCE_PATTERN_MATCH
Definition: rdkafka.h:7681
rd_kafka_interceptor_f_on_commit_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_commit_t(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, rd_kafka_resp_err_t err, void *ic_opaque)
on_commit() is called on completed or failed offset commit. It is called from internal librdkafka thr...
Definition: rdkafka.h:6351
RD_KAFKA_ACL_OPERATION_CREATE
@ RD_KAFKA_ACL_OPERATION_CREATE
Definition: rdkafka.h:7182
rd_kafka_conf_set_stats_cb
RD_EXPORT void rd_kafka_conf_set_stats_cb(rd_kafka_conf_t *conf, int(*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque))
Set statistics callback in provided conf object.
RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
@ RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafka.h:624
rd_kafka_DeleteConsumerGroupOffsets
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete committed offsets for a set of partitions in a consumer group. This will succeed at the partit...
rd_kafka_DescribeCluster
RD_EXPORT void rd_kafka_DescribeCluster(rd_kafka_t *rk, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describes the cluster.
rd_kafka_message_set_headers
RD_EXPORT void rd_kafka_message_set_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t *hdrs)
Replace the message's current headers with a new list.
RD_KAFKA_PRODUCER
@ RD_KAFKA_PRODUCER
Definition: rdkafka.h:210
rd_kafka_poll
RD_EXPORT int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms)
Polls the provided kafka handle for events.
rd_kafka_topic_partition_list_add_range
RD_EXPORT void rd_kafka_topic_partition_list_add_range(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t start, int32_t stop)
Add range of partitions from start to stop inclusive.
RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
Definition: rdkafka.h:453
rd_kafka_conf_properties_show
RD_EXPORT void rd_kafka_conf_properties_show(FILE *fp)
Prints a table to fp of all supported configuration properties, their default values as well as a des...
RD_KAFKA_RESP_ERR__VALUE_SERIALIZATION
@ RD_KAFKA_RESP_ERR__VALUE_SERIALIZATION
Definition: rdkafka.h:365
rd_kafka_seek
RD_EXPORT rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, int timeout_ms)
Seek consumer for topic+partition to offset which is either an absolute or logical offset.
rd_kafka_DeleteGroups
RD_EXPORT void rd_kafka_DeleteGroups(rd_kafka_t *rk, rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete groups from cluster as specified by the del_groups array of size del_group_cnt elements.
RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
@ RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
Definition: rdkafka.h:369
rd_kafka_metadata::broker_cnt
int broker_cnt
Definition: rdkafka.h:4964
rd_kafka_produce_batch
RD_EXPORT int rd_kafka_produce_batch(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, rd_kafka_message_t *rkmessages, int message_cnt)
Produce multiple messages.
RD_KAFKA_CERT_ENC_PKCS12
@ RD_KAFKA_CERT_ENC_PKCS12
Definition: rdkafka.h:2468
RD_KAFKA_ACL_OPERATION_UNKNOWN
@ RD_KAFKA_ACL_OPERATION_UNKNOWN
Definition: rdkafka.h:7176
rd_kafka_sasl_set_credentials
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_set_credentials(rd_kafka_t *rk, const char *username, const char *password)
Sets SASL credentials used for SASL PLAIN and SCRAM mechanisms by this Kafka client.
rd_kafka_ConfigEntry_is_default
RD_EXPORT int rd_kafka_ConfigEntry_is_default(const rd_kafka_ConfigEntry_t *entry)
RD_KAFKA_RESP_ERR__NODE_UPDATE
@ RD_KAFKA_RESP_ERR__NODE_UPDATE
Definition: rdkafka.h:323
rd_kafka_DescribeTopics_result_t
rd_kafka_event_t rd_kafka_DescribeTopics_result_t
Definition: rdkafka.h:5731
RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
@ RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
Definition: rdkafka.h:407
rd_kafka_queue_cb_event_enable
RD_EXPORT void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu, void(*event_cb)(rd_kafka_t *rk, void *qev_opaque), void *qev_opaque)
Enable callback event triggering for queue.
rd_kafka_event_AlterConfigs_result
const RD_EXPORT rd_kafka_AlterConfigs_result_t * rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev)
Get AlterConfigs result.
rd_kafka_topic_opaque
RD_EXPORT void * rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt)
Get the rkt_opaque pointer that was set in the topic configuration with rd_kafka_topic_conf_set_opaqu...
rd_kafka_group_info::protocol
char * protocol
Definition: rdkafka.h:5111
rd_kafka_cert_type_t
rd_kafka_cert_type_t
SSL certificate type.
Definition: rdkafka.h:2453
RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
@ RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafka.h:577
rd_kafka_DeleteTopic_new
RD_EXPORT rd_kafka_DeleteTopic_t * rd_kafka_DeleteTopic_new(const char *topic)
Create a new DeleteTopic object. This object is later passed to rd_kafka_DeleteTopics().
rd_kafka_vu_s::_pad
char _pad[64]
Definition: rdkafka.h:1163
rd_kafka_DeleteGroup_destroy
RD_EXPORT void rd_kafka_DeleteGroup_destroy(rd_kafka_DeleteGroup_t *del_group)
Destroy and free a DeleteGroup object previously created with rd_kafka_DeleteGroup_new()
RD_KAFKA_RESP_ERR__PARTITION_EOF
@ RD_KAFKA_RESP_ERR__PARTITION_EOF
Definition: rdkafka.h:305
rd_kafka_metadata_topic::topic
char * topic
Definition: rdkafka.h:4953
rd_kafka_error_code
RD_EXPORT rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error)
rd_kafka_fatal_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size)
Returns the first fatal error set on this client instance, or RD_KAFKA_RESP_ERR_NO_ERROR if no fatal ...
rd_kafka_event_topic_partition_list
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev)
RD_KAFKA_RESP_ERR_UNKNOWN_PRODUCER_ID
@ RD_KAFKA_RESP_ERR_UNKNOWN_PRODUCER_ID
Definition: rdkafka.h:553
rd_kafka_consumer_group_state_name
const RD_EXPORT char * rd_kafka_consumer_group_state_name(rd_kafka_consumer_group_state_t state)
Returns a name for a state code.
rd_kafka_ConsumerGroupListing_group_id
const RD_EXPORT char * rd_kafka_ConsumerGroupListing_group_id(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets the group id for the grplist group.
rd_kafka_err_desc::code
rd_kafka_resp_err_t code
Definition: rdkafka.h:645
rd_kafka_ScramMechanism_t
rd_kafka_ScramMechanism_t
Apache Kafka ScramMechanism values.
Definition: rdkafka.h:9247
RD_KAFKA_VTYPE_RKT
@ RD_KAFKA_VTYPE_RKT
Definition: rdkafka.h:1120
rd_kafka_DescribeCluster_result_authorized_operations
const RD_EXPORT rd_kafka_AclOperation_t * rd_kafka_DescribeCluster_result_authorized_operations(const rd_kafka_DescribeCluster_result_t *result, size_t *cntp)
Gets the authorized ACL operations for the result cluster.
rd_kafka_AlterConfigs_result_resources
const RD_EXPORT rd_kafka_ConfigResource_t ** rd_kafka_AlterConfigs_result_resources(const rd_kafka_AlterConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a AlterConfigs result.
rd_kafka_DeleteTopic_t
struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t
Definition: rdkafka.h:7349
rd_kafka_conf_destroy
RD_EXPORT void rd_kafka_conf_destroy(rd_kafka_conf_t *conf)
Destroys a conf object.
rd_kafka_msg_partitioner_fnv1a_random
RD_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random FNV-1a partitioner.
rd_kafka_event_DescribeTopics_result
const RD_EXPORT rd_kafka_DescribeTopics_result_t * rd_kafka_event_DescribeTopics_result(rd_kafka_event_t *rkev)
Get DescribeTopics result.
rd_kafka_TopicCollection_t
struct rd_kafka_TopicCollection_s rd_kafka_TopicCollection_t
Represents a collection of topics, to be passed to DescribeTopics.
Definition: rdkafka.h:8112
rd_kafka_AdminOptions_set_isolation_level
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_isolation_level(rd_kafka_AdminOptions_t *options, rd_kafka_IsolationLevel_t value)
Set Isolation Level to an allowed rd_kafka_IsolationLevel_t value.
rd_kafka_ListOffsets_result_t
rd_kafka_event_t rd_kafka_ListOffsets_result_t
Definition: rdkafka.h:5739
rd_kafka_event_ListOffsets_result
const RD_EXPORT rd_kafka_ListOffsets_result_t * rd_kafka_event_ListOffsets_result(rd_kafka_event_t *rkev)
Get ListOffsets result.
rd_kafka_TopicPartitionInfo_replicas
const RD_EXPORT rd_kafka_Node_t ** rd_kafka_TopicPartitionInfo_replicas(const rd_kafka_TopicPartitionInfo_t *partition, size_t *cntp)
Gets the partition replicas for partition.
rd_kafka_topic_partition_s::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:906
rd_kafka_conf_interceptor_add_on_conf_set
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_set(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_set_t *on_conf_set, void *ic_opaque)
Append an on_conf_set() interceptor.
rd_kafka_query_watermark_offsets
RD_EXPORT rd_kafka_resp_err_t rd_kafka_query_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high, int timeout_ms)
Query broker for low (oldest/beginning) and high (newest/end) offsets for partition.
rd_kafka_conf_set_resolve_cb
RD_EXPORT void rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf, int(*resolve_cb)(const char *node, const char *service, const struct addrinfo *hints, struct addrinfo **res, void *opaque))
Set address resolution callback.
RD_KAFKA_RESP_ERR_DUPLICATE_RESOURCE
@ RD_KAFKA_RESP_ERR_DUPLICATE_RESOURCE
Definition: rdkafka.h:622
rd_kafka_consume
RD_EXPORT rd_kafka_message_t * rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms)
Consume a single message from topic rkt and partition.
rd_kafka_DeleteGroups_result_t
rd_kafka_event_t rd_kafka_DeleteGroups_result_t
Definition: rdkafka.h:5723
rd_kafka_DeleteAcls_result_response_matching_acls
const RD_EXPORT rd_kafka_AclBinding_t ** rd_kafka_DeleteAcls_result_response_matching_acls(const rd_kafka_DeleteAcls_result_response_t *result_response, size_t *matching_acls_cntp)
RD_KAFKA_ACL_OPERATION_DESCRIBE
@ RD_KAFKA_ACL_OPERATION_DESCRIBE
Definition: rdkafka.h:7185
RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
@ RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
Definition: rdkafka.h:333
RD_KAFKA_RESP_ERR__SSL
@ RD_KAFKA_RESP_ERR__SSL
Definition: rdkafka.h:325
rd_kafka_event_config_string
const RD_EXPORT char * rd_kafka_event_config_string(rd_kafka_event_t *rkev)
rd_kafka_unsubscribe
RD_EXPORT rd_kafka_resp_err_t rd_kafka_unsubscribe(rd_kafka_t *rk)
Unsubscribe from the current subscription set.
rd_kafka_DeleteRecords_result_t
rd_kafka_event_t rd_kafka_DeleteRecords_result_t
Definition: rdkafka.h:5717
rd_kafka_DescribeAcls
RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t *acl_filter, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe acls matching the filter provided in acl_filter.
rd_kafka_vu_s::u
union rd_kafka_vu_s::@0 u
rd_kafka_ListConsumerGroupOffsets_destroy
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy(rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets)
Destroy and free a ListConsumerGroupOffsets object previously created with rd_kafka_ListConsumerGroup...
rd_kafka_message_detach_headers
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_detach_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list and detach the list from the message making the application the owner of ...
rd_kafka_Node_port
RD_EXPORT uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node)
Get the port of node.
RD_KAFKA_CONSUMER
@ RD_KAFKA_CONSUMER
Definition: rdkafka.h:211
rd_kafka_memberid
RD_EXPORT char * rd_kafka_memberid(const rd_kafka_t *rk)
Returns this client's broker-assigned group member id.
RD_KAFKA_RESP_ERR_INVALID_MSG_SIZE
@ RD_KAFKA_RESP_ERR_INVALID_MSG_SIZE
Definition: rdkafka.h:426
rd_kafka_error_is_retriable
RD_EXPORT int rd_kafka_error_is_retriable(const rd_kafka_error_t *error)
RD_KAFKA_RESP_ERR_GROUP_SUBSCRIBED_TO_TOPIC
@ RD_KAFKA_RESP_ERR_GROUP_SUBSCRIBED_TO_TOPIC
Definition: rdkafka.h:609
rd_kafka_event_AlterConsumerGroupOffsets_result
const RD_EXPORT rd_kafka_AlterConsumerGroupOffsets_result_t * rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get AlterConsumerGroupOffsets result.
rd_kafka_ResourcePatternType_t
rd_kafka_ResourcePatternType_t
Apache Kafka pattern types.
Definition: rdkafka.h:7675
rd_kafka_conf_set_open_cb
RD_EXPORT void rd_kafka_conf_set_open_cb(rd_kafka_conf_t *conf, int(*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque))
Set open callback.
rd_kafka_metadata_partition
Partition information.
Definition: rdkafka.h:4939
RD_KAFKA_VTYPE_OPAQUE
@ RD_KAFKA_VTYPE_OPAQUE
Definition: rdkafka.h:1124
RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
@ RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
Definition: rdkafka.h:549
RD_KAFKA_CERT_ENC_DER
@ RD_KAFKA_CERT_ENC_DER
Definition: rdkafka.h:2469
rd_kafka_conf_interceptor_add_on_conf_dup
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_dup(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_dup_t *on_conf_dup, void *ic_opaque)
Append an on_conf_dup() interceptor.
RD_KAFKA_RESP_ERR_REQUEST_TIMED_OUT
@ RD_KAFKA_RESP_ERR_REQUEST_TIMED_OUT
Definition: rdkafka.h:434
rd_kafka_consume_callback_queue
RD_EXPORT int rd_kafka_consume_callback_queue(rd_kafka_queue_t *rkqu, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consume multiple messages from queue with callback.
RD_KAFKA_CERT_PRIVATE_KEY
@ RD_KAFKA_CERT_PRIVATE_KEY
Definition: rdkafka.h:2455
rd_kafka_topic_destroy
RD_EXPORT void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt)
Loose application's topic handle refcount as previously created with rd_kafka_topic_new().
RD_KAFKA_RESP_ERR__ALL_BROKERS_DOWN
@ RD_KAFKA_RESP_ERR__ALL_BROKERS_DOWN
Definition: rdkafka.h:313
rd_kafka_ListConsumerGroupOffsets_t
struct rd_kafka_ListConsumerGroupOffsets_s rd_kafka_ListConsumerGroupOffsets_t
Definition: rdkafka.h:8877
RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
@ RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafka.h:507
rd_kafka_topic_partition_list_destroy
RD_EXPORT void rd_kafka_topic_partition_list_destroy(rd_kafka_topic_partition_list_t *rkparlist)
Free all resources used by the list and the list itself.
rd_kafka_producev
RD_EXPORT rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk,...)
Produce and send a single message to broker.
RD_KAFKA_RESP_ERR__STATE
@ RD_KAFKA_RESP_ERR__STATE
Definition: rdkafka.h:343
RD_KAFKA_RESP_ERR__NO_OFFSET
@ RD_KAFKA_RESP_ERR__NO_OFFSET
Definition: rdkafka.h:351
rd_kafka_yield
RD_EXPORT void rd_kafka_yield(rd_kafka_t *rk)
Cancels the current callback dispatcher (rd_kafka_poll(), rd_kafka_consume_callback(),...
rd_kafka_NewPartitions_new
RD_EXPORT rd_kafka_NewPartitions_t * rd_kafka_NewPartitions_new(const char *topic, size_t new_total_cnt, char *errstr, size_t errstr_size)
Create a new NewPartitions. This object is later passed to rd_kafka_CreatePartitions() to increase th...
RD_KAFKA_RESP_ERR__TIMED_OUT_QUEUE
@ RD_KAFKA_RESP_ERR__TIMED_OUT_QUEUE
Definition: rdkafka.h:355
rd_kafka_topic_result_name
const RD_EXPORT char * rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres)
rd_kafka_interceptor_f_on_response_received_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_response_received_t(rd_kafka_t *rk, int sockfd, const char *brokername, int32_t brokerid, int16_t ApiKey, int16_t ApiVersion, int32_t CorrId, size_t size, int64_t rtt, rd_kafka_resp_err_t err, void *ic_opaque)
on_response_received() is called when a protocol response has been fully received from a broker TCP c...
Definition: rdkafka.h:6416
rd_kafka_MemberAssignment_partitions
const RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_MemberAssignment_partitions(const rd_kafka_MemberAssignment_t *assignment)
Gets assigned partitions of a member assignment.
RD_KAFKA_ADMIN_OP__CNT
@ RD_KAFKA_ADMIN_OP__CNT
Definition: rdkafka.h:6924
RD_KAFKA_THREAD_BROKER
@ RD_KAFKA_THREAD_BROKER
Definition: rdkafka.h:5353
rd_kafka_interceptor_add_on_response_received
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_response_received(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_response_received_t *on_response_received, void *ic_opaque)
Append an on_response_received() interceptor.
rd_kafka_header_remove
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs, const char *name)
Remove all headers for the given key (if any).
RD_KAFKA_RESP_ERR_NO_REASSIGNMENT_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_NO_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:606
RD_KAFKA_RESP_ERR__ISR_INSUFF
@ RD_KAFKA_RESP_ERR__ISR_INSUFF
Definition: rdkafka.h:321
rd_kafka_AclBinding_operation
RD_EXPORT rd_kafka_AclOperation_t rd_kafka_AclBinding_operation(const rd_kafka_AclBinding_t *acl)
rd_kafka_ScramCredentialInfo_t
struct rd_kafka_ScramCredentialInfo_s rd_kafka_ScramCredentialInfo_t
Scram credential info. Mechanism and iterations for a SASL/SCRAM credential associated with a user.
Definition: rdkafka.h:9259
rd_kafka_topic_conf_destroy
RD_EXPORT void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf)
Destroys a topic conf object.
rd_kafka_IncrementalAlterConfigs_result_t
rd_kafka_event_t rd_kafka_IncrementalAlterConfigs_result_t
Definition: rdkafka.h:5713
rd_kafka_MemberAssignment_t
struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t
Member assignment included in MemberDescription.
Definition: rdkafka.h:8535
rd_kafka_ListConsumerGroupOffsets_destroy_array
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy_array(rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffset_cnt)
Helper function to destroy all ListConsumerGroupOffsets objects in the list_grpoffsets array (of list...
RD_KAFKA_RESP_ERR_NOT_CONTROLLER
@ RD_KAFKA_RESP_ERR_NOT_CONTROLLER
Definition: rdkafka.h:511
rd_kafka_MemberDescription_consumer_id
const RD_EXPORT char * rd_kafka_MemberDescription_consumer_id(const rd_kafka_MemberDescription_t *member)
Gets consumer id of member.
RD_KAFKA_ADMIN_OP_DESCRIBECONSUMERGROUPS
@ RD_KAFKA_ADMIN_OP_DESCRIBECONSUMERGROUPS
Definition: rdkafka.h:6910
RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
@ RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
Definition: rdkafka.h:9512
rd_kafka_ConfigEntry_value
const RD_EXPORT char * rd_kafka_ConfigEntry_value(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_consumer_group_state_code
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_consumer_group_state_code(const char *name)
Returns a code for a state name.
rd_kafka_topic_partition_list_add
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_add(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Add topic+partition to list.
rd_kafka_msg_partitioner_murmur2_random
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random Murmur2 partitioner (Java compatible).
rd_kafka_IsolationLevel_t
rd_kafka_IsolationLevel_t
IsolationLevel enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:6948
rd_kafka_DeleteGroups_result_groups
const RD_EXPORT rd_kafka_group_result_t ** rd_kafka_DeleteGroups_result_groups(const rd_kafka_DeleteGroups_result_t *result, size_t *cntp)
Get an array of group results from a DeleteGroups result.
rd_kafka_UserScramCredentialDeletion_new
RD_EXPORT rd_kafka_UserScramCredentialAlteration_t * rd_kafka_UserScramCredentialDeletion_new(const char *username, rd_kafka_ScramMechanism_t mechanism)
Allocates a new UserScramCredentialDeletion given its fields.
rd_kafka_position
RD_EXPORT rd_kafka_resp_err_t rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Retrieve current positions (offsets) for topics+partitions.
rd_kafka_message_s::len
size_t len
Definition: rdkafka.h:1447
rd_kafka_resume_partitions
RD_EXPORT rd_kafka_resp_err_t rd_kafka_resume_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Resume producing consumption for the provided list of partitions.
rd_kafka_DeleteAcls_result_t
rd_kafka_event_t rd_kafka_DeleteAcls_result_t
Definition: rdkafka.h:5707
RD_KAFKA_ADMIN_OP_ALTERCONFIGS
@ RD_KAFKA_ADMIN_OP_ALTERCONFIGS
Definition: rdkafka.h:6900
rd_kafka_queue_new
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_new(rd_kafka_t *rk)
Create a new message queue.
rd_kafka_header_get_all
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs, size_t idx, const char **namep, const void **valuep, size_t *sizep)
Iterator for all headers.
RD_KAFKA_CONFIG_SOURCE_DYNAMIC_TOPIC_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_TOPIC_CONFIG
Definition: rdkafka.h:7563
RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
Definition: rdkafka.h:7574
rd_kafka_topic_partition_list_copy
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_copy(const rd_kafka_topic_partition_list_t *src)
Make a copy of an existing list.
rd_kafka_DescribeCluster_result_cluster_id
const RD_EXPORT char * rd_kafka_DescribeCluster_result_cluster_id(const rd_kafka_DescribeCluster_result_t *result)
Gets the cluster id for the result cluster.
rd_kafka_group_member_info::client_id
char * client_id
Definition: rdkafka.h:5077
rd_kafka_consume_queue
RD_EXPORT rd_kafka_message_t * rd_kafka_consume_queue(rd_kafka_queue_t *rkqu, int timeout_ms)
Consume from queue.
rd_kafka_event_opaque
RD_EXPORT void * rd_kafka_event_opaque(rd_kafka_event_t *rkev)
rd_kafka_CreateAcls
RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk, rd_kafka_AclBinding_t **new_acls, size_t new_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create acls as specified by the new_acls array of size new_topic_cnt elements.
rd_kafka_group_result_partitions
const RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres)
RD_KAFKA_RESP_ERR_UNKNOWN
@ RD_KAFKA_RESP_ERR_UNKNOWN
Definition: rdkafka.h:416
rd_kafka_ResourceType_name
const RD_EXPORT char * rd_kafka_ResourceType_name(rd_kafka_ResourceType_t restype)
rd_kafka_plugin_f_conf_init_t
rd_kafka_resp_err_t() rd_kafka_plugin_f_conf_init_t(rd_kafka_conf_t *conf, void **plug_opaquep, char *errstr, size_t errstr_size)
Plugin's configuration initializer method called each time the library is referenced from configurati...
Definition: rdkafka.h:6085
rd_kafka_interceptor_f_on_conf_set_t
rd_kafka_conf_res_t() rd_kafka_interceptor_f_on_conf_set_t(rd_kafka_conf_t *conf, const char *name, const char *val, char *errstr, size_t errstr_size, void *ic_opaque)
on_conf_set() is called from rd_kafka_*_conf_set() in the order the interceptors were added.
Definition: rdkafka.h:6171
rd_kafka_offsets_store
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_store(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets)
Store offsets for next auto-commit for one or more partitions.
RD_KAFKA_RESP_ERR__BAD_MSG
@ RD_KAFKA_RESP_ERR__BAD_MSG
Definition: rdkafka.h:286
rd_kafka_event_DeleteRecords_result
const RD_EXPORT rd_kafka_DeleteRecords_result_t * rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev)
rd_kafka_group_info::state
char * state
Definition: rdkafka.h:5109
RD_KAFKA_ADMIN_OP_ANY
@ RD_KAFKA_ADMIN_OP_ANY
Definition: rdkafka.h:6896
rd_kafka_consumer_group_metadata_write
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_write(const rd_kafka_consumer_group_metadata_t *cgmd, void **bufferp, size_t *sizep)
Serialize the consumer group metadata to a binary format. This is mainly for client binding use and n...
RD_KAFKA_RESP_ERR__MSG_TIMED_OUT
@ RD_KAFKA_RESP_ERR__MSG_TIMED_OUT
Definition: rdkafka.h:300
rd_kafka_consume_callback
RD_EXPORT int rd_kafka_consume_callback(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consumes messages from topic rkt and partition, calling the provided callback for each consumed messs...
RD_KAFKA_RESP_ERR_UNSUPPORTED_VERSION
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_VERSION
Definition: rdkafka.h:499
rd_kafka_AclOperation_t
rd_kafka_AclOperation_t
Apache Kafka ACL operation types. Common type for multiple Admin API functions.
Definition: rdkafka.h:7175
rd_kafka_init_transactions
RD_EXPORT rd_kafka_error_t * rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms)
Initialize transactions for the producer instance.
rd_kafka_message_s::rkt
rd_kafka_topic_t * rkt
Definition: rdkafka.h:1441
RD_KAFKA_RESOURCE__CNT
@ RD_KAFKA_RESOURCE__CNT
Definition: rdkafka.h:7668
rd_kafka_topic_new
RD_EXPORT rd_kafka_topic_t * rd_kafka_topic_new(rd_kafka_t *rk, const char *topic, rd_kafka_topic_conf_t *conf)
Creates a new topic handle for topic named topic.
rd_kafka_metadata_topic
Topic information.
Definition: rdkafka.h:4952
rd_kafka_group_result_name
const RD_EXPORT char * rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres)
rd_kafka_type_t
rd_kafka_type_t
rd_kafka_t handle type.
Definition: rdkafka.h:209
rd_kafka_metadata_partition::isrs
int32_t * isrs
Definition: rdkafka.h:4946
rd_kafka_NewPartitions_destroy_array
RD_EXPORT void rd_kafka_NewPartitions_destroy_array(rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt)
Helper function to destroy all NewPartitions objects in the new_parts array (of new_parts_cnt element...
rd_kafka_oauthbearer_set_token
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token(rd_kafka_t *rk, const char *token_value, int64_t md_lifetime_ms, const char *md_principal_name, const char **extensions, size_t extension_size, char *errstr, size_t errstr_size)
Set SASL/OAUTHBEARER token and metadata.
rd_kafka_MemberDescription_host
const RD_EXPORT char * rd_kafka_MemberDescription_host(const rd_kafka_MemberDescription_t *member)
Gets host of member.
rd_kafka_interceptor_f_on_conf_dup_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_dup_t(rd_kafka_conf_t *new_conf, const rd_kafka_conf_t *old_conf, size_t filter_cnt, const char **filter, void *ic_opaque)
on_conf_dup() is called from rd_kafka_conf_dup() in the order the interceptors were added and is used...
Definition: rdkafka.h:6201
rd_kafka_ConfigEntry_is_sensitive
RD_EXPORT int rd_kafka_ConfigEntry_is_sensitive(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_metadata_partition::isr_cnt
int isr_cnt
Definition: rdkafka.h:4945
rd_kafka_MemberDescription_group_instance_id
const RD_EXPORT char * rd_kafka_MemberDescription_group_instance_id(const rd_kafka_MemberDescription_t *member)
Gets group instance id of member.
rd_kafka_resp_err_t
rd_kafka_resp_err_t
Error codes.
Definition: rdkafka.h:281
RD_KAFKA_RESP_ERR__PARTIAL
@ RD_KAFKA_RESP_ERR__PARTIAL
Definition: rdkafka.h:371
rd_kafka_AclBinding_principal
const RD_EXPORT char * rd_kafka_AclBinding_principal(const rd_kafka_AclBinding_t *acl)
RD_KAFKA_RESP_ERR__WAIT_COORD
@ RD_KAFKA_RESP_ERR__WAIT_COORD
Definition: rdkafka.h:327
rd_kafka_conf_dup_filter
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf, size_t filter_cnt, const char **filter)
Same as rd_kafka_conf_dup() but with an array of property name prefixes to filter out (ignore) when c...
rd_kafka_ConsumerGroupDescription_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_ConsumerGroupDescription_error(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the error for the grpdesc group.
RD_KAFKA_ADMIN_OP_DESCRIBEACLS
@ RD_KAFKA_ADMIN_OP_DESCRIBEACLS
Definition: rdkafka.h:6907
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafka.h:565
RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafka.h:515
RD_KAFKA_ACL_OPERATION_ALL
@ RD_KAFKA_ACL_OPERATION_ALL
Definition: rdkafka.h:7179
rd_kafka_seek_partitions
RD_EXPORT rd_kafka_error_t * rd_kafka_seek_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Seek consumer for partitions in partitions to the per-partition offset in the .offset field of partit...
RD_KAFKA_RESP_ERR_INVALID_PRODUCER_ID_MAPPING
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_ID_MAPPING
Definition: rdkafka.h:528
rd_kafka_msg_status_t
rd_kafka_msg_status_t
Message persistence status can be used by the application to find out if a produced message was persi...
Definition: rdkafka.h:1593
rd_kafka_UserScramCredentialsDescription_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_UserScramCredentialsDescription_error(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the error associated with a UserScramCredentialsDescription.
RD_KAFKA_RESP_ERR__IN_PROGRESS
@ RD_KAFKA_RESP_ERR__IN_PROGRESS
Definition: rdkafka.h:331
rd_kafka_topic_partition_set_leader_epoch
RD_EXPORT void rd_kafka_topic_partition_set_leader_epoch(rd_kafka_topic_partition_t *rktpar, int32_t leader_epoch)
Sets the offset leader epoch (use -1 to clear).
RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC
@ RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC
Definition: rdkafka.h:311
rd_kafka_event_message_next
const RD_EXPORT rd_kafka_message_t * rd_kafka_event_message_next(rd_kafka_event_t *rkev)
rd_kafka_test_fatal_error
RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk, rd_kafka_resp_err_t err, const char *reason)
Trigger a fatal error for testing purposes.
rd_kafka_event_CreatePartitions_result
const RD_EXPORT rd_kafka_CreatePartitions_result_t * rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev)
Get CreatePartitions result.
RD_KAFKA_MSG_STATUS_POSSIBLY_PERSISTED
@ RD_KAFKA_MSG_STATUS_POSSIBLY_PERSISTED
Definition: rdkafka.h:1602
rd_kafka_interceptor_f_on_acknowledgement_t
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_acknowledgement_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_acknowledgement() is called to inform interceptors that a message was succesfully delivered or per...
Definition: rdkafka.h:6303
rd_kafka_header_get_last
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs, const char *name, const void **valuep, size_t *sizep)
Find last header in list hdrs matching name.