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 0x020600ff
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 typedef struct rd_kafka_topic_partition_result_s
267  rd_kafka_topic_partition_result_t;
268 /* @endcond */
269 
270 
283 typedef enum {
284  /* Internal errors to rdkafka: */
415 
418 
419  /* Kafka broker errors: */
435 #define RD_KAFKA_RESP_ERR_NOT_LEADER_OR_FOLLOWER \
436  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION
437  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION = 6,
455 #define RD_KAFKA_RESP_ERR_GROUP_LOAD_IN_PROGRESS \
456  RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
457 
460 #define RD_KAFKA_RESP_ERR_GROUP_COORDINATOR_NOT_AVAILABLE \
461  RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
462 
465 #define RD_KAFKA_RESP_ERR_NOT_COORDINATOR_FOR_GROUP \
466  RD_KAFKA_RESP_ERR_NOT_COORDINATOR
467 
657  RD_KAFKA_RESP_ERR_END_ALL,
659 
660 
668  const char *name;
669  const char *desc;
670 };
671 
672 
676 RD_EXPORT
677 void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs,
678  size_t *cntp);
679 
680 
681 
687 RD_EXPORT
688 const char *rd_kafka_err2str(rd_kafka_resp_err_t err);
689 
690 
691 
697 RD_EXPORT
698 const char *rd_kafka_err2name(rd_kafka_resp_err_t err);
699 
700 
726 RD_EXPORT
728 
729 
754 RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox);
755 
756 
769 RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void);
770 
771 
772 
799 RD_EXPORT
801 rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size);
802 
803 
821 RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk,
823  const char *reason);
824 
825 
830 RD_EXPORT
831 rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error);
832 
841 RD_EXPORT
842 const char *rd_kafka_error_name(const rd_kafka_error_t *error);
843 
850 RD_EXPORT
851 const char *rd_kafka_error_string(const rd_kafka_error_t *error);
852 
853 
858 RD_EXPORT
859 int rd_kafka_error_is_fatal(const rd_kafka_error_t *error);
860 
861 
866 RD_EXPORT
867 int rd_kafka_error_is_retriable(const rd_kafka_error_t *error);
868 
869 
880 RD_EXPORT
881 int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error);
882 
888 RD_EXPORT
889 void rd_kafka_error_destroy(rd_kafka_error_t *error);
890 
891 
900 RD_EXPORT
901 rd_kafka_error_t *rd_kafka_error_new(rd_kafka_resp_err_t code,
902  const char *fmt,
903  ...) RD_FORMAT(printf, 2, 3);
904 
905 
922  char *topic;
923  int32_t partition;
924  int64_t offset;
925  void *metadata;
926  size_t metadata_size;
927  void *opaque;
929  void *_private;
933 } rd_kafka_topic_partition_t;
934 
939 RD_EXPORT
940 void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar);
941 
942 
951 RD_EXPORT
953  rd_kafka_topic_partition_t *rktpar,
954  int32_t leader_epoch);
955 
964 RD_EXPORT
966  const rd_kafka_topic_partition_t *rktpar);
967 
973  int cnt;
974  int size;
975  rd_kafka_topic_partition_t *elems;
976 } rd_kafka_topic_partition_list_t;
977 
992 RD_EXPORT
993 rd_kafka_topic_partition_list_t *rd_kafka_topic_partition_list_new(int size);
994 
998 RD_EXPORT
1000  rd_kafka_topic_partition_list_t *rkparlist);
1001 
1011 RD_EXPORT
1012 rd_kafka_topic_partition_t *
1013 rd_kafka_topic_partition_list_add(rd_kafka_topic_partition_list_t *rktparlist,
1014  const char *topic,
1015  int32_t partition);
1016 
1017 
1026 RD_EXPORT
1028  rd_kafka_topic_partition_list_t *rktparlist,
1029  const char *topic,
1030  int32_t start,
1031  int32_t stop);
1032 
1033 
1034 
1046 RD_EXPORT
1048  rd_kafka_topic_partition_list_t *rktparlist,
1049  const char *topic,
1050  int32_t partition);
1051 
1052 
1060 RD_EXPORT
1062  rd_kafka_topic_partition_list_t *rktparlist,
1063  int idx);
1064 
1065 
1073 RD_EXPORT
1074 rd_kafka_topic_partition_list_t *
1075 rd_kafka_topic_partition_list_copy(const rd_kafka_topic_partition_list_t *src);
1076 
1077 
1078 
1086 RD_EXPORT
1088  rd_kafka_topic_partition_list_t *rktparlist,
1089  const char *topic,
1090  int32_t partition,
1091  int64_t offset);
1092 
1093 
1094 
1100 RD_EXPORT
1101 rd_kafka_topic_partition_t *rd_kafka_topic_partition_list_find(
1102  const rd_kafka_topic_partition_list_t *rktparlist,
1103  const char *topic,
1104  int32_t partition);
1105 
1106 
1116 RD_EXPORT void rd_kafka_topic_partition_list_sort(
1117  rd_kafka_topic_partition_list_t *rktparlist,
1118  int (*cmp)(const void *a, const void *b, void *cmp_opaque),
1119  void *cmp_opaque);
1120 
1121 
1139 typedef enum rd_kafka_vtype_t {
1157 
1158 
1165 typedef struct rd_kafka_vu_s {
1168  union {
1169  const char *cstr;
1170  rd_kafka_topic_t *rkt;
1171  int i;
1172  int32_t i32;
1173  int64_t i64;
1174  struct {
1175  void *ptr;
1176  size_t size;
1177  } mem;
1178  struct {
1179  const char *name;
1180  const void *val;
1181  ssize_t size;
1182  } header;
1183  rd_kafka_headers_t *headers;
1184  void *ptr;
1185  char _pad[64];
1186  } u;
1187 } rd_kafka_vu_t;
1188 
1197 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
1198 
1204 #define RD_KAFKA_V_TOPIC(topic) \
1205  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
1206  (const char *)topic
1207 
1212 #define RD_KAFKA_V_RKT(rkt) \
1213  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
1214  (rd_kafka_topic_t *)rkt
1215 
1220 #define RD_KAFKA_V_PARTITION(partition) \
1221  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
1222  (int32_t)partition
1223 
1228 #define RD_KAFKA_V_VALUE(VALUE, LEN) \
1229  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
1230  (void *)VALUE, (size_t)LEN
1231 
1236 #define RD_KAFKA_V_KEY(KEY, LEN) \
1237  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
1238  (void *)KEY, (size_t)LEN
1239 
1246 #define RD_KAFKA_V_OPAQUE(msg_opaque) \
1247  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, msg_opaque), \
1248  (void *)msg_opaque
1249 
1255 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
1256  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), (int)msgflags
1257 
1263 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
1264  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
1265  (int64_t)timestamp
1266 
1274 #define RD_KAFKA_V_HEADER(NAME, VALUE, LEN) \
1275  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
1276  const void *, VALUE, ssize_t, LEN), \
1277  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
1278 
1290 #define RD_KAFKA_V_HEADERS(HDRS) \
1291  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
1292  (rd_kafka_headers_t *)HDRS
1293 
1294 
1325 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new(size_t initial_count);
1326 
1331 RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs);
1332 
1336 RD_EXPORT rd_kafka_headers_t *
1337 rd_kafka_headers_copy(const rd_kafka_headers_t *src);
1338 
1356 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs,
1357  const char *name,
1358  ssize_t name_size,
1359  const void *value,
1360  ssize_t value_size);
1361 
1369 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs,
1370  const char *name);
1371 
1372 
1390 RD_EXPORT rd_kafka_resp_err_t
1391 rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs,
1392  const char *name,
1393  const void **valuep,
1394  size_t *sizep);
1395 
1409 RD_EXPORT rd_kafka_resp_err_t
1410 rd_kafka_header_get(const rd_kafka_headers_t *hdrs,
1411  size_t idx,
1412  const char *name,
1413  const void **valuep,
1414  size_t *sizep);
1415 
1416 
1424 RD_EXPORT rd_kafka_resp_err_t
1425 rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs,
1426  size_t idx,
1427  const char **namep,
1428  const void **valuep,
1429  size_t *sizep);
1430 
1431 
1432 
1445 // FIXME: This doesn't show up in docs for some reason
1446 // "Compound rd_kafka_message_t is not documented."
1447 
1461 typedef struct rd_kafka_message_s {
1463  rd_kafka_topic_t *rkt;
1464  int32_t partition;
1465  void *payload;
1469  size_t len;
1472  void *key;
1474  size_t key_len;
1476  int64_t offset;
1484  void *_private;
1491 } rd_kafka_message_t;
1492 
1493 
1497 RD_EXPORT
1498 void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage);
1499 
1500 
1501 
1508 RD_EXPORT
1509 const char *rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage);
1510 
1517 RD_EXPORT
1518 const char *
1519 rd_kafka_message_produce_errstr(const rd_kafka_message_t *rkmessage);
1520 
1521 
1533 RD_EXPORT
1534 int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage,
1535  rd_kafka_timestamp_type_t *tstype);
1536 
1537 
1538 
1545 RD_EXPORT
1546 int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage);
1547 
1548 
1555 RD_EXPORT
1556 int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage);
1557 
1558 
1575 RD_EXPORT rd_kafka_resp_err_t
1576 rd_kafka_message_headers(const rd_kafka_message_t *rkmessage,
1577  rd_kafka_headers_t **hdrsp);
1578 
1590 RD_EXPORT rd_kafka_resp_err_t
1591 rd_kafka_message_detach_headers(rd_kafka_message_t *rkmessage,
1592  rd_kafka_headers_t **hdrsp);
1593 
1594 
1607 RD_EXPORT
1608 void rd_kafka_message_set_headers(rd_kafka_message_t *rkmessage,
1609  rd_kafka_headers_t *hdrs);
1610 
1611 
1617 RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs);
1618 
1619 
1625 typedef enum {
1630 
1635 
1641 
1642 
1649 RD_EXPORT rd_kafka_msg_status_t
1650 rd_kafka_message_status(const rd_kafka_message_t *rkmessage);
1651 
1652 
1660 RD_EXPORT int32_t
1661 rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage);
1662 
1663 
1680 RD_EXPORT const char *rd_kafka_Uuid_base64str(const rd_kafka_Uuid_t *uuid);
1681 
1689 RD_EXPORT int64_t
1690 rd_kafka_Uuid_least_significant_bits(const rd_kafka_Uuid_t *uuid);
1691 
1692 
1700 RD_EXPORT int64_t
1701 rd_kafka_Uuid_most_significant_bits(const rd_kafka_Uuid_t *uuid);
1702 
1703 
1713 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_new(int64_t most_significant_bits,
1714  int64_t least_significant_bits);
1715 
1724 RD_EXPORT rd_kafka_Uuid_t *rd_kafka_Uuid_copy(const rd_kafka_Uuid_t *uuid);
1725 
1731 RD_EXPORT void rd_kafka_Uuid_destroy(rd_kafka_Uuid_t *uuid);
1732 
1748 typedef enum {
1755 
1756 
1790 RD_EXPORT
1791 rd_kafka_conf_t *rd_kafka_conf_new(void);
1792 
1793 
1797 RD_EXPORT
1798 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1799 
1800 
1807 RD_EXPORT
1808 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1809 
1810 
1815 RD_EXPORT
1816 rd_kafka_conf_t *rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf,
1817  size_t filter_cnt,
1818  const char **filter);
1819 
1820 
1821 
1830 RD_EXPORT
1831 const rd_kafka_conf_t *rd_kafka_conf(rd_kafka_t *rk);
1832 
1833 
1853 RD_EXPORT
1854 rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf,
1855  const char *name,
1856  const char *value,
1857  char *errstr,
1858  size_t errstr_size);
1859 
1860 
1866 RD_EXPORT
1867 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1868 
1869 
1906  rd_kafka_conf_t *conf,
1907  void (*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque));
1908 
1909 
1913 RD_EXPORT
1914 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1915  void (*dr_cb)(rd_kafka_t *rk,
1916  void *payload,
1917  size_t len,
1918  rd_kafka_resp_err_t err,
1919  void *opaque,
1920  void *msg_opaque));
1921 
1952 RD_EXPORT
1954  rd_kafka_conf_t *conf,
1955  void (*dr_msg_cb)(rd_kafka_t *rk,
1956  const rd_kafka_message_t *rkmessage,
1957  void *opaque));
1958 
1959 
1967 RD_EXPORT
1969  rd_kafka_conf_t *conf,
1970  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque));
1971 
2076 RD_EXPORT
2078  rd_kafka_conf_t *conf,
2079  void (*rebalance_cb)(rd_kafka_t *rk,
2080  rd_kafka_resp_err_t err,
2081  rd_kafka_topic_partition_list_t *partitions,
2082  void *opaque));
2083 
2084 
2085 
2103 RD_EXPORT
2105  rd_kafka_conf_t *conf,
2106  void (*offset_commit_cb)(rd_kafka_t *rk,
2107  rd_kafka_resp_err_t err,
2108  rd_kafka_topic_partition_list_t *offsets,
2109  void *opaque));
2110 
2111 
2134 RD_EXPORT
2135 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
2136  void (*error_cb)(rd_kafka_t *rk,
2137  int err,
2138  const char *reason,
2139  void *opaque));
2140 
2158 RD_EXPORT
2159 void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf,
2160  void (*throttle_cb)(rd_kafka_t *rk,
2161  const char *broker_name,
2162  int32_t broker_id,
2163  int throttle_time_ms,
2164  void *opaque));
2165 
2166 
2183 RD_EXPORT
2184 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
2185  void (*log_cb)(const rd_kafka_t *rk,
2186  int level,
2187  const char *fac,
2188  const char *buf));
2189 
2190 
2213 RD_EXPORT
2215  rd_kafka_conf_t *conf,
2216  int (*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque));
2217 
2267 RD_EXPORT
2269  rd_kafka_conf_t *conf,
2270  void (*oauthbearer_token_refresh_cb)(rd_kafka_t *rk,
2271  const char *oauthbearer_config,
2272  void *opaque));
2273 
2301 RD_EXPORT
2302 void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable);
2303 
2304 
2322 RD_EXPORT
2324  rd_kafka_conf_t *conf,
2325  int (*socket_cb)(int domain, int type, int protocol, void *opaque));
2326 
2327 
2328 
2344 RD_EXPORT void
2345 rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf,
2346  int (*connect_cb)(int sockfd,
2347  const struct sockaddr *addr,
2348  int addrlen,
2349  const char *id,
2350  void *opaque));
2351 
2362 RD_EXPORT void rd_kafka_conf_set_closesocket_cb(
2363  rd_kafka_conf_t *conf,
2364  int (*closesocket_cb)(int sockfd, void *opaque));
2365 
2366 
2367 
2368 #ifndef _WIN32
2369 
2386 RD_EXPORT
2388  rd_kafka_conf_t *conf,
2389  int (*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque));
2390 #endif
2391 
2393 struct addrinfo;
2394 
2414 RD_EXPORT void
2415 rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf,
2416  int (*resolve_cb)(const char *node,
2417  const char *service,
2418  const struct addrinfo *hints,
2419  struct addrinfo **res,
2420  void *opaque));
2421 
2463 RD_EXPORT
2465  rd_kafka_conf_t *conf,
2466  int (*ssl_cert_verify_cb)(rd_kafka_t *rk,
2467  const char *broker_name,
2468  int32_t broker_id,
2469  int *x509_error,
2470  int depth,
2471  const char *buf,
2472  size_t size,
2473  char *errstr,
2474  size_t errstr_size,
2475  void *opaque));
2476 
2477 
2485 typedef enum rd_kafka_cert_type_t {
2489  RD_KAFKA_CERT__CNT,
2491 
2499 typedef enum rd_kafka_cert_enc_t {
2503  RD_KAFKA_CERT_ENC__CNT,
2505 
2506 
2549 RD_EXPORT rd_kafka_conf_res_t
2550 rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf,
2551  rd_kafka_cert_type_t cert_type,
2552  rd_kafka_cert_enc_t cert_enc,
2553  const void *buffer,
2554  size_t size,
2555  char *errstr,
2556  size_t errstr_size);
2557 
2558 
2573 RD_EXPORT
2574 void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf,
2575  void *callback_data);
2576 
2577 
2583 RD_EXPORT
2584 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
2585 
2590 RD_EXPORT
2591 void *rd_kafka_opaque(const rd_kafka_t *rk);
2592 
2593 
2594 
2608 RD_EXPORT
2609 void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf,
2610  rd_kafka_topic_conf_t *tconf);
2611 
2624 RD_EXPORT rd_kafka_topic_conf_t *
2625 rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf);
2626 
2627 
2647 RD_EXPORT
2648 rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf,
2649  const char *name,
2650  char *dest,
2651  size_t *dest_size);
2652 
2653 
2659 RD_EXPORT
2660 rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf,
2661  const char *name,
2662  char *dest,
2663  size_t *dest_size);
2664 
2665 
2674 RD_EXPORT
2675 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
2676 
2677 
2686 RD_EXPORT
2687 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
2688  size_t *cntp);
2689 
2694 RD_EXPORT
2695 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
2696 
2704 RD_EXPORT
2705 void rd_kafka_conf_properties_show(FILE *fp);
2706 
2723 RD_EXPORT
2724 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
2725 
2726 
2730 RD_EXPORT
2731 rd_kafka_topic_conf_t *
2732 rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf);
2733 
2738 RD_EXPORT
2739 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup(rd_kafka_t *rk);
2740 
2741 
2745 RD_EXPORT
2746 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
2747 
2748 
2757 RD_EXPORT
2758 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
2759  const char *name,
2760  const char *value,
2761  char *errstr,
2762  size_t errstr_size);
2763 
2770 RD_EXPORT
2771 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf,
2772  void *rkt_opaque);
2773 
2774 
2794 RD_EXPORT
2796  rd_kafka_topic_conf_t *topic_conf,
2797  int32_t (*partitioner)(const rd_kafka_topic_t *rkt,
2798  const void *keydata,
2799  size_t keylen,
2800  int32_t partition_cnt,
2801  void *rkt_opaque,
2802  void *msg_opaque));
2803 
2804 
2831  rd_kafka_topic_conf_t *topic_conf,
2832  int (*msg_order_cmp)(const rd_kafka_message_t *a,
2833  const rd_kafka_message_t *b));
2834 
2835 
2843 RD_EXPORT
2844 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
2845  int32_t partition);
2846 
2847 
2848 /*******************************************************************
2849  * *
2850  * Partitioners provided by rdkafka *
2851  * *
2852  *******************************************************************/
2853 
2867 RD_EXPORT
2868 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
2869  const void *key,
2870  size_t keylen,
2871  int32_t partition_cnt,
2872  void *rkt_opaque,
2873  void *msg_opaque);
2874 
2888 RD_EXPORT
2889 int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt,
2890  const void *key,
2891  size_t keylen,
2892  int32_t partition_cnt,
2893  void *rkt_opaque,
2894  void *msg_opaque);
2895 
2911 RD_EXPORT
2912 int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt,
2913  const void *key,
2914  size_t keylen,
2915  int32_t partition_cnt,
2916  void *rkt_opaque,
2917  void *msg_opaque);
2918 
2919 
2933 RD_EXPORT
2934 int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt,
2935  const void *key,
2936  size_t keylen,
2937  int32_t partition_cnt,
2938  void *rkt_opaque,
2939  void *msg_opaque);
2940 
2955 RD_EXPORT
2956 int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt,
2957  const void *key,
2958  size_t keylen,
2959  int32_t partition_cnt,
2960  void *rkt_opaque,
2961  void *msg_opaque);
2962 
2963 
2977 RD_EXPORT
2978 int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt,
2979  const void *key,
2980  size_t keylen,
2981  int32_t partition_cnt,
2982  void *rkt_opaque,
2983  void *msg_opaque);
2984 
2985 
3000 RD_EXPORT
3001 int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt,
3002  const void *key,
3003  size_t keylen,
3004  int32_t partition_cnt,
3005  void *rkt_opaque,
3006  void *msg_opaque);
3007 
3008 
3048 RD_EXPORT
3049 rd_kafka_t *rd_kafka_new(rd_kafka_type_t type,
3050  rd_kafka_conf_t *conf,
3051  char *errstr,
3052  size_t errstr_size);
3053 
3054 
3069 RD_EXPORT
3070 void rd_kafka_destroy(rd_kafka_t *rk);
3071 
3072 
3077 RD_EXPORT
3078 void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags);
3079 
3094 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
3095 
3096 
3097 
3101 RD_EXPORT
3102 const char *rd_kafka_name(const rd_kafka_t *rk);
3103 
3104 
3108 RD_EXPORT
3109 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
3110 
3111 
3122 RD_EXPORT
3123 char *rd_kafka_memberid(const rd_kafka_t *rk);
3124 
3125 
3126 
3145 RD_EXPORT
3146 char *rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms);
3147 
3148 
3164 RD_EXPORT
3165 int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms);
3166 
3167 
3189 RD_EXPORT
3190 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk,
3191  const char *topic,
3192  rd_kafka_topic_conf_t *conf);
3193 
3194 
3195 
3204 RD_EXPORT
3205 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
3206 
3207 
3211 RD_EXPORT
3212 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
3213 
3214 
3219 RD_EXPORT
3220 void *rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt);
3221 
3222 
3229 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
3230 
3231 
3259 RD_EXPORT
3260 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
3261 
3262 
3273 RD_EXPORT
3274 void rd_kafka_yield(rd_kafka_t *rk);
3275 
3276 
3277 
3285 RD_EXPORT rd_kafka_resp_err_t
3286 rd_kafka_pause_partitions(rd_kafka_t *rk,
3287  rd_kafka_topic_partition_list_t *partitions);
3288 
3289 
3290 
3298 RD_EXPORT rd_kafka_resp_err_t
3299 rd_kafka_resume_partitions(rd_kafka_t *rk,
3300  rd_kafka_topic_partition_list_t *partitions);
3301 
3302 
3303 
3312 RD_EXPORT rd_kafka_resp_err_t
3313 rd_kafka_query_watermark_offsets(rd_kafka_t *rk,
3314  const char *topic,
3315  int32_t partition,
3316  int64_t *low,
3317  int64_t *high,
3318  int timeout_ms);
3319 
3320 
3337 RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk,
3338  const char *topic,
3339  int32_t partition,
3340  int64_t *low,
3341  int64_t *high);
3342 
3343 
3344 
3370 RD_EXPORT rd_kafka_resp_err_t
3371 rd_kafka_offsets_for_times(rd_kafka_t *rk,
3372  rd_kafka_topic_partition_list_t *offsets,
3373  int timeout_ms);
3374 
3375 
3376 
3390 RD_EXPORT
3391 void *rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size);
3392 
3393 
3394 
3408 RD_EXPORT
3409 void *rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size);
3410 
3411 
3412 
3429 RD_EXPORT
3430 void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr);
3431 
3432 
3454 RD_EXPORT
3455 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
3456 
3460 RD_EXPORT
3461 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
3462 
3463 
3470 RD_EXPORT
3471 rd_kafka_queue_t *rd_kafka_queue_get_main(rd_kafka_t *rk);
3472 
3473 
3474 
3484 RD_EXPORT
3485 rd_kafka_queue_t *rd_kafka_queue_get_sasl(rd_kafka_t *rk);
3486 
3487 
3503 RD_EXPORT
3504 rd_kafka_error_t *rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk);
3505 
3506 
3521 RD_EXPORT
3522 rd_kafka_error_t *rd_kafka_sasl_set_credentials(rd_kafka_t *rk,
3523  const char *username,
3524  const char *password);
3525 
3541 RD_EXPORT
3542 rd_kafka_queue_t *rd_kafka_queue_get_consumer(rd_kafka_t *rk);
3543 
3554 RD_EXPORT
3555 rd_kafka_queue_t *rd_kafka_queue_get_partition(rd_kafka_t *rk,
3556  const char *topic,
3557  int32_t partition);
3558 
3581 RD_EXPORT
3582 rd_kafka_queue_t *rd_kafka_queue_get_background(rd_kafka_t *rk);
3583 
3584 
3595 RD_EXPORT
3596 void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
3597 
3616 RD_EXPORT
3618  rd_kafka_queue_t *rkqu);
3619 
3620 
3624 RD_EXPORT
3625 size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu);
3626 
3627 
3645 RD_EXPORT
3646 void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu,
3647  int fd,
3648  const void *payload,
3649  size_t size);
3650 
3667 RD_EXPORT
3668 void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu,
3669  void (*event_cb)(rd_kafka_t *rk,
3670  void *qev_opaque),
3671  void *qev_opaque);
3672 
3673 
3681 RD_EXPORT
3682 void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu);
3683 
3684 
3695 #define RD_KAFKA_OFFSET_BEGINNING \
3696  -2
3698 #define RD_KAFKA_OFFSET_END \
3699  -1
3701 #define RD_KAFKA_OFFSET_STORED \
3702  -1000
3704 #define RD_KAFKA_OFFSET_INVALID -1001
3708 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
3709 
3716 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
3717 
3751 RD_EXPORT
3752 int rd_kafka_consume_start(rd_kafka_topic_t *rkt,
3753  int32_t partition,
3754  int64_t offset);
3755 
3770 RD_EXPORT
3771 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt,
3772  int32_t partition,
3773  int64_t offset,
3774  rd_kafka_queue_t *rkqu);
3775 
3789 RD_EXPORT
3790 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
3791 
3792 
3793 
3822 RD_EXPORT
3823 rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt,
3824  int32_t partition,
3825  int64_t offset,
3826  int timeout_ms);
3827 
3828 
3829 
3861 RD_EXPORT rd_kafka_error_t *
3862 rd_kafka_seek_partitions(rd_kafka_t *rk,
3863  rd_kafka_topic_partition_list_t *partitions,
3864  int timeout_ms);
3865 
3866 
3891 RD_EXPORT
3892 rd_kafka_message_t *
3893 rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms);
3894 
3895 
3896 
3922 RD_EXPORT
3923 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt,
3924  int32_t partition,
3925  int timeout_ms,
3926  rd_kafka_message_t **rkmessages,
3927  size_t rkmessages_size);
3928 
3929 
3930 
3964 RD_EXPORT
3965 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt,
3966  int32_t partition,
3967  int timeout_ms,
3968  void (*consume_cb)(rd_kafka_message_t *rkmessage,
3969  void *commit_opaque),
3970  void *commit_opaque);
3971 
3972 
3991 RD_EXPORT
3992 rd_kafka_message_t *rd_kafka_consume_queue(rd_kafka_queue_t *rkqu,
3993  int timeout_ms);
3994 
4000 RD_EXPORT
4001 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
4002  int timeout_ms,
4003  rd_kafka_message_t **rkmessages,
4004  size_t rkmessages_size);
4005 
4015 RD_EXPORT
4017  rd_kafka_queue_t *rkqu,
4018  int timeout_ms,
4019  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque),
4020  void *commit_opaque);
4021 
4022 
4062 RD_EXPORT
4064 rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset);
4065 
4066 
4099 RD_EXPORT rd_kafka_resp_err_t
4100 rd_kafka_offsets_store(rd_kafka_t *rk,
4101  rd_kafka_topic_partition_list_t *offsets);
4102 
4103 
4124 RD_EXPORT
4125 rd_kafka_error_t *rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage);
4126 
4180 RD_EXPORT rd_kafka_resp_err_t
4181 rd_kafka_subscribe(rd_kafka_t *rk,
4182  const rd_kafka_topic_partition_list_t *topics);
4183 
4184 
4188 RD_EXPORT
4190 
4191 
4201 RD_EXPORT rd_kafka_resp_err_t
4202 rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics);
4203 
4204 
4205 
4235 RD_EXPORT
4236 rd_kafka_message_t *rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms);
4237 
4255 RD_EXPORT
4257 
4258 
4276 RD_EXPORT
4277 rd_kafka_error_t *rd_kafka_consumer_close_queue(rd_kafka_t *rk,
4278  rd_kafka_queue_t *rkqu);
4279 
4280 
4289 RD_EXPORT
4290 int rd_kafka_consumer_closed(rd_kafka_t *rk);
4291 
4292 
4310 RD_EXPORT rd_kafka_error_t *
4311 rd_kafka_incremental_assign(rd_kafka_t *rk,
4312  const rd_kafka_topic_partition_list_t *partitions);
4313 
4314 
4332 RD_EXPORT rd_kafka_error_t *rd_kafka_incremental_unassign(
4333  rd_kafka_t *rk,
4334  const rd_kafka_topic_partition_list_t *partitions);
4335 
4336 
4350 RD_EXPORT
4351 const char *rd_kafka_rebalance_protocol(rd_kafka_t *rk);
4352 
4353 
4374 RD_EXPORT rd_kafka_resp_err_t
4375 rd_kafka_assign(rd_kafka_t *rk,
4376  const rd_kafka_topic_partition_list_t *partitions);
4377 
4394 RD_EXPORT rd_kafka_resp_err_t
4395 rd_kafka_assignment(rd_kafka_t *rk,
4396  rd_kafka_topic_partition_list_t **partitions);
4397 
4398 
4415 RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk);
4416 
4417 
4453 RD_EXPORT rd_kafka_resp_err_t
4454 rd_kafka_commit(rd_kafka_t *rk,
4455  const rd_kafka_topic_partition_list_t *offsets,
4456  int async);
4457 
4458 
4465 RD_EXPORT rd_kafka_resp_err_t
4466 rd_kafka_commit_message(rd_kafka_t *rk,
4467  const rd_kafka_message_t *rkmessage,
4468  int async);
4469 
4470 
4493 RD_EXPORT rd_kafka_resp_err_t
4494 rd_kafka_commit_queue(rd_kafka_t *rk,
4495  const rd_kafka_topic_partition_list_t *offsets,
4496  rd_kafka_queue_t *rkqu,
4497  void (*cb)(rd_kafka_t *rk,
4498  rd_kafka_resp_err_t err,
4499  rd_kafka_topic_partition_list_t *offsets,
4500  void *commit_opaque),
4501  void *commit_opaque);
4502 
4503 
4521 RD_EXPORT rd_kafka_resp_err_t
4522 rd_kafka_committed(rd_kafka_t *rk,
4523  rd_kafka_topic_partition_list_t *partitions,
4524  int timeout_ms);
4525 
4526 
4527 
4544 RD_EXPORT rd_kafka_resp_err_t
4545 rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions);
4546 
4547 
4548 
4560 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4561 rd_kafka_consumer_group_metadata(rd_kafka_t *rk);
4562 
4563 
4573 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4574 rd_kafka_consumer_group_metadata_new(const char *group_id);
4575 
4576 
4589 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4591  int32_t generation_id,
4592  const char *member_id,
4593  const char *group_instance_id);
4594 
4595 
4605 RD_EXPORT
4607  const rd_kafka_consumer_group_metadata_t *group_metadata);
4608 
4609 
4614 RD_EXPORT void
4615 rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *);
4616 
4617 
4638 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_write(
4639  const rd_kafka_consumer_group_metadata_t *cgmd,
4640  void **bufferp,
4641  size_t *sizep);
4642 
4663 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_read(
4664  rd_kafka_consumer_group_metadata_t **cgmdp,
4665  const void *buffer,
4666  size_t size);
4667 
4683 #define RD_KAFKA_MSG_F_FREE \
4684  0x1
4686 #define RD_KAFKA_MSG_F_COPY \
4687  0x2
4689 #define RD_KAFKA_MSG_F_BLOCK \
4690  0x4
4701 #define RD_KAFKA_MSG_F_PARTITION \
4702  0x8
4807 RD_EXPORT
4808 int rd_kafka_produce(rd_kafka_topic_t *rkt,
4809  int32_t partition,
4810  int msgflags,
4811  void *payload,
4812  size_t len,
4813  const void *key,
4814  size_t keylen,
4815  void *msg_opaque);
4816 
4817 
4831 RD_EXPORT
4832 rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk, ...);
4833 
4834 
4846 RD_EXPORT
4847 rd_kafka_error_t *
4848 rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt);
4849 
4850 
4880 RD_EXPORT
4881 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt,
4882  int32_t partition,
4883  int msgflags,
4884  rd_kafka_message_t *rkmessages,
4885  int message_cnt);
4886 
4887 
4888 
4913 RD_EXPORT
4914 rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms);
4915 
4916 
4917 
4948 RD_EXPORT
4949 rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags);
4950 
4951 
4959 #define RD_KAFKA_PURGE_F_QUEUE 0x1
4960 
4968 #define RD_KAFKA_PURGE_F_INFLIGHT 0x2
4969 
4970 
4974 #define RD_KAFKA_PURGE_F_NON_BLOCKING 0x4
4975 
4976 
4992  int32_t id;
4993  char *host;
4994  int port;
4995 } rd_kafka_metadata_broker_t;
4996 
5001  int32_t id;
5003  int32_t leader;
5005  int32_t *replicas;
5006  int isr_cnt;
5007  int32_t *isrs;
5008 } rd_kafka_metadata_partition_t;
5009 
5013 typedef struct rd_kafka_metadata_topic {
5014  char *topic;
5018 } rd_kafka_metadata_topic_t;
5019 
5020 
5024 typedef struct rd_kafka_metadata {
5031  int32_t orig_broker_id;
5033 } rd_kafka_metadata_t;
5034 
5055 RD_EXPORT
5057 rd_kafka_metadata(rd_kafka_t *rk,
5058  int all_topics,
5059  rd_kafka_topic_t *only_rkt,
5060  const struct rd_kafka_metadata **metadatap,
5061  int timeout_ms);
5062 
5066 RD_EXPORT
5067 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
5068 
5072 typedef struct rd_kafka_Node_s rd_kafka_Node_t;
5073 
5081 RD_EXPORT
5082 int rd_kafka_Node_id(const rd_kafka_Node_t *node);
5083 
5094 RD_EXPORT
5095 const char *rd_kafka_Node_host(const rd_kafka_Node_t *node);
5096 
5104 RD_EXPORT
5105 uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node);
5106 
5114 RD_EXPORT
5115 const char *rd_kafka_Node_rack(const rd_kafka_Node_t *node);
5116 
5137  char *member_id;
5138  char *client_id;
5139  char *client_host;
5146 };
5147 
5153 typedef enum {
5154  RD_KAFKA_CONSUMER_GROUP_STATE_UNKNOWN = 0,
5155  RD_KAFKA_CONSUMER_GROUP_STATE_PREPARING_REBALANCE = 1,
5156  RD_KAFKA_CONSUMER_GROUP_STATE_COMPLETING_REBALANCE = 2,
5157  RD_KAFKA_CONSUMER_GROUP_STATE_STABLE = 3,
5158  RD_KAFKA_CONSUMER_GROUP_STATE_DEAD = 4,
5159  RD_KAFKA_CONSUMER_GROUP_STATE_EMPTY = 5,
5160  RD_KAFKA_CONSUMER_GROUP_STATE__CNT
5162 
5168 typedef enum {
5169  RD_KAFKA_CONSUMER_GROUP_TYPE_UNKNOWN = 0,
5170  RD_KAFKA_CONSUMER_GROUP_TYPE_CONSUMER = 1,
5171  RD_KAFKA_CONSUMER_GROUP_TYPE_CLASSIC = 2,
5172  RD_KAFKA_CONSUMER_GROUP_TYPE__CNT
5174 
5180  char *group;
5182  char *state;
5184  char *protocol;
5187 };
5188 
5197 };
5198 
5199 
5230 RD_EXPORT
5232 rd_kafka_list_groups(rd_kafka_t *rk,
5233  const char *group,
5234  const struct rd_kafka_group_list **grplistp,
5235  int timeout_ms);
5236 
5244 RD_EXPORT
5245 const char *
5247 
5255 RD_EXPORT
5257 rd_kafka_consumer_group_state_code(const char *name);
5258 
5266 RD_EXPORT
5267 const char *
5269 
5279 RD_EXPORT
5281 rd_kafka_consumer_group_type_code(const char *name);
5282 
5286 RD_EXPORT
5287 void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist);
5288 
5289 
5333 RD_EXPORT
5334 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
5335 
5336 
5337 
5350 RD_EXPORT RD_DEPRECATED void
5351 rd_kafka_set_logger(rd_kafka_t *rk,
5352  void (*func)(const rd_kafka_t *rk,
5353  int level,
5354  const char *fac,
5355  const char *buf));
5356 
5357 
5367 RD_EXPORT
5368 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
5369 
5370 
5374 RD_EXPORT
5375 void rd_kafka_log_print(const rd_kafka_t *rk,
5376  int level,
5377  const char *fac,
5378  const char *buf);
5379 
5380 
5386 RD_EXPORT
5387 void rd_kafka_log_syslog(const rd_kafka_t *rk,
5388  int level,
5389  const char *fac,
5390  const char *buf);
5391 
5392 
5415 RD_EXPORT
5416 int rd_kafka_outq_len(rd_kafka_t *rk);
5417 
5418 
5419 
5426 RD_EXPORT
5427 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
5428 
5429 
5430 
5436 RD_EXPORT
5437 int rd_kafka_thread_cnt(void);
5438 
5439 
5452 
5453 
5462 RD_EXPORT
5463 int rd_kafka_wait_destroyed(int timeout_ms);
5464 
5465 
5471 RD_EXPORT
5472 int rd_kafka_unittest(void);
5473 
5474 
5491 RD_EXPORT
5493 
5494 
5511 #define RD_KAFKA_EVENT_NONE 0x0
5512 #define RD_KAFKA_EVENT_DR 0x1
5513 #define RD_KAFKA_EVENT_FETCH 0x2
5514 #define RD_KAFKA_EVENT_LOG 0x4
5515 #define RD_KAFKA_EVENT_ERROR 0x8
5516 #define RD_KAFKA_EVENT_REBALANCE 0x10
5517 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
5518 #define RD_KAFKA_EVENT_STATS 0x40
5519 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
5520 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
5521 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT \
5522  102
5523 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
5524 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT \
5525  104
5526 #define RD_KAFKA_EVENT_DELETERECORDS_RESULT 105
5527 #define RD_KAFKA_EVENT_DELETEGROUPS_RESULT 106
5529 #define RD_KAFKA_EVENT_DELETECONSUMERGROUPOFFSETS_RESULT 107
5530 
5531 #define RD_KAFKA_EVENT_OAUTHBEARER_TOKEN_REFRESH 0x100
5532 #define RD_KAFKA_EVENT_BACKGROUND 0x200
5533 #define RD_KAFKA_EVENT_CREATEACLS_RESULT 0x400
5534 #define RD_KAFKA_EVENT_DESCRIBEACLS_RESULT 0x800
5535 #define RD_KAFKA_EVENT_DELETEACLS_RESULT 0x1000
5537 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPS_RESULT 0x2000
5538 
5539 #define RD_KAFKA_EVENT_DESCRIBECONSUMERGROUPS_RESULT 0x4000
5540 
5541 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPOFFSETS_RESULT 0x8000
5542 
5543 #define RD_KAFKA_EVENT_ALTERCONSUMERGROUPOFFSETS_RESULT 0x10000
5544 
5545 #define RD_KAFKA_EVENT_INCREMENTALALTERCONFIGS_RESULT 0x20000
5546 
5547 #define RD_KAFKA_EVENT_DESCRIBEUSERSCRAMCREDENTIALS_RESULT 0x40000
5548 
5549 #define RD_KAFKA_EVENT_ALTERUSERSCRAMCREDENTIALS_RESULT 0x80000
5550 
5551 #define RD_KAFKA_EVENT_DESCRIBETOPICS_RESULT 0x100000
5552 
5553 #define RD_KAFKA_EVENT_DESCRIBECLUSTER_RESULT 0x200000
5554 
5555 #define RD_KAFKA_EVENT_LISTOFFSETS_RESULT 0x400000
5556 
5557 #define RD_KAFKA_EVENT_ELECTLEADERS_RESULT 0x800000
5558 
5565 RD_EXPORT
5566 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev);
5567 
5574 RD_EXPORT
5575 const char *rd_kafka_event_name(const rd_kafka_event_t *rkev);
5576 
5577 
5587 RD_EXPORT
5588 void rd_kafka_event_destroy(rd_kafka_event_t *rkev);
5589 
5590 
5606 RD_EXPORT
5607 const rd_kafka_message_t *rd_kafka_event_message_next(rd_kafka_event_t *rkev);
5608 
5609 
5623 RD_EXPORT
5624 size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev,
5625  const rd_kafka_message_t **rkmessages,
5626  size_t size);
5627 
5628 
5636 RD_EXPORT
5637 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev);
5638 
5639 
5651 RD_EXPORT
5652 const char *rd_kafka_event_config_string(rd_kafka_event_t *rkev);
5653 
5654 
5663 RD_EXPORT
5664 rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev);
5665 
5666 
5675 RD_EXPORT
5676 const char *rd_kafka_event_error_string(rd_kafka_event_t *rkev);
5677 
5678 
5687 RD_EXPORT
5688 int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev);
5689 
5690 
5718 RD_EXPORT
5719 void *rd_kafka_event_opaque(rd_kafka_event_t *rkev);
5720 
5721 
5730 RD_EXPORT
5731 int rd_kafka_event_log(rd_kafka_event_t *rkev,
5732  const char **fac,
5733  const char **str,
5734  int *level);
5735 
5736 
5748 RD_EXPORT
5749 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev,
5750  char *dst,
5751  size_t dstsize);
5752 
5753 
5766 RD_EXPORT
5767 const char *rd_kafka_event_stats(rd_kafka_event_t *rkev);
5768 
5769 
5780 RD_EXPORT rd_kafka_topic_partition_list_t *
5781 rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev);
5782 
5783 
5794 RD_EXPORT rd_kafka_topic_partition_t *
5795 rd_kafka_event_topic_partition(rd_kafka_event_t *rkev);
5796 
5797 
5799 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
5801 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
5803 typedef rd_kafka_event_t rd_kafka_CreateAcls_result_t;
5805 typedef rd_kafka_event_t rd_kafka_DescribeAcls_result_t;
5807 typedef rd_kafka_event_t rd_kafka_DeleteAcls_result_t;
5809 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
5811 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
5815 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
5817 typedef rd_kafka_event_t rd_kafka_DeleteRecords_result_t;
5819 typedef rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t;
5823 typedef rd_kafka_event_t rd_kafka_DeleteGroups_result_t;
5831 typedef rd_kafka_event_t rd_kafka_DescribeTopics_result_t;
5833 typedef rd_kafka_event_t rd_kafka_DescribeCluster_result_t;
5839 typedef rd_kafka_event_t rd_kafka_ListOffsets_result_t;
5841 typedef rd_kafka_event_t rd_kafka_ElectLeaders_result_t;
5842 
5852 RD_EXPORT const rd_kafka_CreateTopics_result_t *
5853 rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev);
5854 
5864 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
5865 rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev);
5866 
5876 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
5877 rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev);
5878 
5888 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
5889 rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev);
5890 
5901 rd_kafka_event_IncrementalAlterConfigs_result(rd_kafka_event_t *rkev);
5902 
5912 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
5913 rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev);
5914 
5922 RD_EXPORT const rd_kafka_DeleteRecords_result_t *
5923 rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev);
5924 
5937 RD_EXPORT const rd_kafka_ListConsumerGroups_result_t *
5938 rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev);
5939 
5953 rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev);
5954 
5967 RD_EXPORT const rd_kafka_DescribeTopics_result_t *
5968 rd_kafka_event_DescribeTopics_result(rd_kafka_event_t *rkev);
5969 
5982 RD_EXPORT const rd_kafka_DescribeCluster_result_t *
5983 rd_kafka_event_DescribeCluster_result(rd_kafka_event_t *rkev);
5993 RD_EXPORT const rd_kafka_DeleteGroups_result_t *
5994 rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev);
5995 
6006 rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
6007 
6015 RD_EXPORT const rd_kafka_CreateAcls_result_t *
6016 rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev);
6017 
6025 RD_EXPORT const rd_kafka_DescribeAcls_result_t *
6026 rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev);
6027 
6035 RD_EXPORT const rd_kafka_DeleteAcls_result_t *
6036 rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev);
6037 
6051 rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
6052 
6066 rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
6067 
6080 RD_EXPORT const rd_kafka_ListOffsets_result_t *
6081 rd_kafka_event_ListOffsets_result(rd_kafka_event_t *rkev);
6082 
6083 
6098 
6112 rd_kafka_event_AlterUserScramCredentials_result(rd_kafka_event_t *rkev);
6113 
6126 RD_EXPORT const rd_kafka_ElectLeaders_result_t *
6127 rd_kafka_event_ElectLeaders_result(rd_kafka_event_t *rkev);
6128 
6138 RD_EXPORT
6139 rd_kafka_event_t *rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms);
6140 
6155 RD_EXPORT
6156 int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms);
6157 
6158 
6203  rd_kafka_conf_t *conf,
6204  void **plug_opaquep,
6205  char *errstr,
6206  size_t errstr_size);
6207 
6289  rd_kafka_conf_t *conf,
6290  const char *name,
6291  const char *val,
6292  char *errstr,
6293  size_t errstr_size,
6294  void *ic_opaque);
6295 
6296 
6319  rd_kafka_conf_t *new_conf,
6320  const rd_kafka_conf_t *old_conf,
6321  size_t filter_cnt,
6322  const char **filter,
6323  void *ic_opaque);
6324 
6325 
6333  void *ic_opaque);
6334 
6335 
6354  rd_kafka_t *rk,
6355  const rd_kafka_conf_t *conf,
6356  void *ic_opaque,
6357  char *errstr,
6358  size_t errstr_size);
6359 
6360 
6368 typedef rd_kafka_resp_err_t(
6369  rd_kafka_interceptor_f_on_destroy_t)(rd_kafka_t *rk, void *ic_opaque);
6370 
6371 
6372 
6394  rd_kafka_t *rk,
6395  rd_kafka_message_t *rkmessage,
6396  void *ic_opaque);
6397 
6421  rd_kafka_t *rk,
6422  rd_kafka_message_t *rkmessage,
6423  void *ic_opaque);
6424 
6425 
6443  rd_kafka_t *rk,
6444  rd_kafka_message_t *rkmessage,
6445  void *ic_opaque);
6446 
6469  rd_kafka_t *rk,
6470  const rd_kafka_topic_partition_list_t *offsets,
6472  void *ic_opaque);
6473 
6474 
6497  rd_kafka_t *rk,
6498  int sockfd,
6499  const char *brokername,
6500  int32_t brokerid,
6501  int16_t ApiKey,
6502  int16_t ApiVersion,
6503  int32_t CorrId,
6504  size_t size,
6505  void *ic_opaque);
6506 
6507 
6534  rd_kafka_t *rk,
6535  int sockfd,
6536  const char *brokername,
6537  int32_t brokerid,
6538  int16_t ApiKey,
6539  int16_t ApiVersion,
6540  int32_t CorrId,
6541  size_t size,
6542  int64_t rtt,
6544  void *ic_opaque);
6545 
6546 
6564  rd_kafka_t *rk,
6565  rd_kafka_thread_type_t thread_type,
6566  const char *thread_name,
6567  void *ic_opaque);
6568 
6569 
6590  rd_kafka_t *rk,
6591  rd_kafka_thread_type_t thread_type,
6592  const char *thread_name,
6593  void *ic_opaque);
6594 
6595 
6611  rd_kafka_t *rk,
6612  int32_t broker_id,
6613  const char *secproto,
6614  const char *name,
6615  int port,
6616  const char *state,
6617  void *ic_opaque);
6618 
6619 
6633  rd_kafka_conf_t *conf,
6634  const char *ic_name,
6636  void *ic_opaque);
6637 
6638 
6652  rd_kafka_conf_t *conf,
6653  const char *ic_name,
6655  void *ic_opaque);
6656 
6671  rd_kafka_conf_t *conf,
6672  const char *ic_name,
6674  void *ic_opaque);
6675 
6676 
6698 RD_EXPORT rd_kafka_resp_err_t
6699 rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf,
6700  const char *ic_name,
6702  void *ic_opaque);
6703 
6704 
6705 
6719  rd_kafka_t *rk,
6720  const char *ic_name,
6722  void *ic_opaque);
6723 
6724 
6737 RD_EXPORT rd_kafka_resp_err_t
6738 rd_kafka_interceptor_add_on_send(rd_kafka_t *rk,
6739  const char *ic_name,
6741  void *ic_opaque);
6742 
6756  rd_kafka_t *rk,
6757  const char *ic_name,
6758  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
6759  void *ic_opaque);
6760 
6761 
6775  rd_kafka_t *rk,
6776  const char *ic_name,
6778  void *ic_opaque);
6779 
6780 
6794  rd_kafka_t *rk,
6795  const char *ic_name,
6797  void *ic_opaque);
6798 
6799 
6813  rd_kafka_t *rk,
6814  const char *ic_name,
6816  void *ic_opaque);
6817 
6818 
6832  rd_kafka_t *rk,
6833  const char *ic_name,
6834  rd_kafka_interceptor_f_on_response_received_t *on_response_received,
6835  void *ic_opaque);
6836 
6837 
6851  rd_kafka_t *rk,
6852  const char *ic_name,
6854  void *ic_opaque);
6855 
6856 
6870  rd_kafka_t *rk,
6871  const char *ic_name,
6873  void *ic_opaque);
6874 
6875 
6888 RD_EXPORT
6890  rd_kafka_t *rk,
6891  const char *ic_name,
6892  rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change,
6893  void *ic_opaque);
6894 
6895 
6896 
6917 RD_EXPORT rd_kafka_resp_err_t
6918 rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres);
6919 
6926 RD_EXPORT const char *
6927 rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres);
6928 
6934 RD_EXPORT const char *
6935 rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres);
6936 
6946 RD_EXPORT const rd_kafka_error_t *
6947 rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres);
6948 
6954 RD_EXPORT const char *
6955 rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres);
6956 
6962 RD_EXPORT const rd_kafka_topic_partition_list_t *
6963 rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres);
6964 
6978 RD_EXPORT const rd_kafka_topic_partition_t *
6980  const rd_kafka_topic_partition_result_t *partition_result);
6981 
6987 RD_EXPORT const rd_kafka_error_t *rd_kafka_topic_partition_result_error(
6988  const rd_kafka_topic_partition_result_t *partition_result);
6989 
7036 typedef enum rd_kafka_admin_op_t {
7068 
7081 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
7082 
7091  RD_KAFKA_ISOLATION_LEVEL_READ_UNCOMMITTED = 0,
7092  RD_KAFKA_ISOLATION_LEVEL_READ_COMMITTED = 1
7094 
7114 RD_EXPORT rd_kafka_AdminOptions_t *
7115 rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api);
7116 
7117 
7122 
7123 
7141 RD_EXPORT rd_kafka_resp_err_t
7143  int timeout_ms,
7144  char *errstr,
7145  size_t errstr_size);
7146 
7147 
7174 RD_EXPORT rd_kafka_resp_err_t
7176  int timeout_ms,
7177  char *errstr,
7178  size_t errstr_size);
7179 
7180 
7199 RD_EXPORT rd_kafka_resp_err_t
7201  int true_or_false,
7202  char *errstr,
7203  size_t errstr_size);
7204 
7205 
7233 RD_EXPORT rd_kafka_resp_err_t
7235  int32_t broker_id,
7236  char *errstr,
7237  size_t errstr_size);
7238 
7239 
7252 RD_EXPORT
7254  rd_kafka_AdminOptions_t *options,
7255  int true_or_false);
7256 
7271 RD_EXPORT
7273  rd_kafka_AdminOptions_t *options,
7274  int true_or_false);
7275 
7288 RD_EXPORT
7290  rd_kafka_AdminOptions_t *options,
7291  const rd_kafka_consumer_group_state_t *consumer_group_states,
7292  size_t consumer_group_states_cnt);
7293 
7306 RD_EXPORT
7308  rd_kafka_AdminOptions_t *options,
7309  const rd_kafka_consumer_group_type_t *consumer_group_types,
7310  size_t consumer_group_types_cnt);
7311 
7315 RD_EXPORT
7316 rd_kafka_error_t *
7319 
7324 RD_EXPORT void
7326  void *ev_opaque);
7327 
7328 
7329 
7338  1,
7347  9,
7349  10,
7351  11,
7353  12,
7354  RD_KAFKA_ACL_OPERATION__CNT
7356 
7368 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
7369 
7391 RD_EXPORT rd_kafka_NewTopic_t *rd_kafka_NewTopic_new(const char *topic,
7392  int num_partitions,
7393  int replication_factor,
7394  char *errstr,
7395  size_t errstr_size);
7396 
7401 RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic);
7402 
7403 
7409 RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics,
7410  size_t new_topic_cnt);
7411 
7412 
7434 RD_EXPORT rd_kafka_resp_err_t
7436  int32_t partition,
7437  int32_t *broker_ids,
7438  size_t broker_id_cnt,
7439  char *errstr,
7440  size_t errstr_size);
7441 
7454 RD_EXPORT rd_kafka_resp_err_t
7456  const char *name,
7457  const char *value);
7458 
7459 
7478 RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk,
7479  rd_kafka_NewTopic_t **new_topics,
7480  size_t new_topic_cnt,
7481  const rd_kafka_AdminOptions_t *options,
7482  rd_kafka_queue_t *rkqu);
7483 
7484 
7485 /*
7486  * CreateTopics result type and methods
7487  */
7488 
7497 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_CreateTopics_result_topics(
7498  const rd_kafka_CreateTopics_result_t *result,
7499  size_t *cntp);
7500 
7501 
7502 
7503 /*
7504  * DeleteTopics - delete topics from cluster
7505  *
7506  */
7507 
7509 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
7510 
7520 RD_EXPORT rd_kafka_DeleteTopic_t *rd_kafka_DeleteTopic_new(const char *topic);
7521 
7526 RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic);
7527 
7533 RD_EXPORT void
7535  size_t del_topic_cnt);
7536 
7550 RD_EXPORT
7551 void rd_kafka_DeleteTopics(rd_kafka_t *rk,
7552  rd_kafka_DeleteTopic_t **del_topics,
7553  size_t del_topic_cnt,
7554  const rd_kafka_AdminOptions_t *options,
7555  rd_kafka_queue_t *rkqu);
7556 
7557 
7558 
7559 /*
7560  * DeleteTopics result type and methods
7561  */
7562 
7571 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_DeleteTopics_result_topics(
7572  const rd_kafka_DeleteTopics_result_t *result,
7573  size_t *cntp);
7574 
7575 
7586 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
7587 
7604 RD_EXPORT rd_kafka_NewPartitions_t *
7605 rd_kafka_NewPartitions_new(const char *topic,
7606  size_t new_total_cnt,
7607  char *errstr,
7608  size_t errstr_size);
7609 
7614 RD_EXPORT void
7616 
7622 RD_EXPORT void
7624  size_t new_parts_cnt);
7625 
7649  rd_kafka_NewPartitions_t *new_parts,
7650  int32_t new_partition_idx,
7651  int32_t *broker_ids,
7652  size_t broker_id_cnt,
7653  char *errstr,
7654  size_t errstr_size);
7655 
7656 
7675 RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk,
7676  rd_kafka_NewPartitions_t **new_parts,
7677  size_t new_parts_cnt,
7678  const rd_kafka_AdminOptions_t *options,
7679  rd_kafka_queue_t *rkqu);
7680 
7681 
7682 
7683 /*
7684  * CreatePartitions result type and methods
7685  */
7686 
7695 RD_EXPORT const rd_kafka_topic_result_t **
7697  const rd_kafka_CreatePartitions_result_t *result,
7698  size_t *cntp);
7699 
7735 
7739 
7740 
7744 RD_EXPORT const char *
7746 
7747 
7749 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
7750 
7754 RD_EXPORT const char *
7756 
7761 RD_EXPORT const char *
7763 
7767 RD_EXPORT rd_kafka_ConfigSource_t
7769 
7774 RD_EXPORT int
7776 
7782 RD_EXPORT int
7784 
7792 RD_EXPORT int
7794 
7798 RD_EXPORT int
7800 
7801 
7812 RD_EXPORT const rd_kafka_ConfigEntry_t **
7814  size_t *cntp);
7815 
7816 
7817 
7831 
7847  RD_KAFKA_RESOURCE_PATTERN_TYPE__CNT,
7849 
7855  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SET = 0,
7856  RD_KAFKA_ALTER_CONFIG_OP_TYPE_DELETE = 1,
7857  RD_KAFKA_ALTER_CONFIG_OP_TYPE_APPEND = 2,
7858  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SUBTRACT = 3,
7859  RD_KAFKA_ALTER_CONFIG_OP_TYPE__CNT,
7861 
7865 RD_EXPORT const char *rd_kafka_ResourcePatternType_name(
7866  rd_kafka_ResourcePatternType_t resource_pattern_type);
7867 
7871 RD_EXPORT const char *
7873 
7875 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
7876 
7877 
7886 RD_EXPORT rd_kafka_ConfigResource_t *
7888  const char *resname);
7889 
7894 RD_EXPORT void
7896 
7897 
7903 RD_EXPORT void
7905  size_t config_cnt);
7906 
7907 
7921 RD_EXPORT rd_kafka_resp_err_t
7923  const char *name,
7924  const char *value);
7925 
7926 
7945 RD_EXPORT rd_kafka_error_t *rd_kafka_ConfigResource_add_incremental_config(
7946  rd_kafka_ConfigResource_t *config,
7947  const char *name,
7949  const char *value);
7950 
7951 
7960 RD_EXPORT const rd_kafka_ConfigEntry_t **
7962  size_t *cntp);
7963 
7964 
7965 
7969 RD_EXPORT rd_kafka_ResourceType_t
7971 
7975 RD_EXPORT const char *
7977 
7981 RD_EXPORT rd_kafka_resp_err_t
7983 
7988 RD_EXPORT const char *
7990 
7991 
7992 /*
7993  * AlterConfigs - alter cluster configuration.
7994  *
7995  */
7996 
7997 
8020 RD_EXPORT
8021 void rd_kafka_AlterConfigs(rd_kafka_t *rk,
8022  rd_kafka_ConfigResource_t **configs,
8023  size_t config_cnt,
8024  const rd_kafka_AdminOptions_t *options,
8025  rd_kafka_queue_t *rkqu);
8026 
8027 
8028 /*
8029  * AlterConfigs result type and methods
8030  */
8031 
8046 RD_EXPORT const rd_kafka_ConfigResource_t **
8048  const rd_kafka_AlterConfigs_result_t *result,
8049  size_t *cntp);
8050 
8051 
8052 
8053 /*
8054  * IncrementalAlterConfigs - alter cluster configuration incrementally.
8055  *
8056  */
8057 
8058 
8079 RD_EXPORT
8080 void rd_kafka_IncrementalAlterConfigs(rd_kafka_t *rk,
8081  rd_kafka_ConfigResource_t **configs,
8082  size_t config_cnt,
8083  const rd_kafka_AdminOptions_t *options,
8084  rd_kafka_queue_t *rkqu);
8085 
8086 
8087 /*
8088  * IncrementalAlterConfigs result type and methods
8089  */
8090 
8106 RD_EXPORT const rd_kafka_ConfigResource_t **
8109  size_t *cntp);
8110 
8111 
8112 
8113 /*
8114  * DescribeConfigs - retrieve cluster configuration.
8115  *
8116  */
8117 
8118 
8143 RD_EXPORT
8144 void rd_kafka_DescribeConfigs(rd_kafka_t *rk,
8145  rd_kafka_ConfigResource_t **configs,
8146  size_t config_cnt,
8147  const rd_kafka_AdminOptions_t *options,
8148  rd_kafka_queue_t *rkqu);
8149 
8150 
8151 
8152 /*
8153  * DescribeConfigs result type and methods
8154  */
8155 
8164 RD_EXPORT const rd_kafka_ConfigResource_t **
8166  const rd_kafka_DescribeConfigs_result_t *result,
8167  size_t *cntp);
8168 
8169 
8180 typedef struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t;
8181 
8199  const rd_kafka_topic_partition_list_t *before_offsets);
8200 
8205 RD_EXPORT void
8207 
8213 RD_EXPORT void
8215  size_t del_record_cnt);
8216 
8238 RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk,
8239  rd_kafka_DeleteRecords_t **del_records,
8240  size_t del_record_cnt,
8241  const rd_kafka_AdminOptions_t *options,
8242  rd_kafka_queue_t *rkqu);
8243 
8244 
8245 /*
8246  * DeleteRecords result type and methods
8247  */
8248 
8258 RD_EXPORT const rd_kafka_topic_partition_list_t *
8260  const rd_kafka_DeleteRecords_result_t *result);
8261 
8273 typedef struct rd_kafka_TopicCollection_s rd_kafka_TopicCollection_t;
8274 
8279 typedef struct rd_kafka_TopicPartitionInfo_s rd_kafka_TopicPartitionInfo_t;
8280 
8285 typedef struct rd_kafka_TopicDescription_s rd_kafka_TopicDescription_t;
8286 
8296 RD_EXPORT
8298 rd_kafka_TopicCollection_of_topic_names(const char **topics, size_t topics_cnt);
8299 
8304 RD_EXPORT void
8306 
8321 RD_EXPORT
8322 void rd_kafka_DescribeTopics(rd_kafka_t *rk,
8323  const rd_kafka_TopicCollection_t *topics,
8324  const rd_kafka_AdminOptions_t *options,
8325  rd_kafka_queue_t *rkqu);
8326 
8336 RD_EXPORT
8338  const rd_kafka_DescribeTopics_result_t *result,
8339  size_t *cntp);
8340 
8341 
8352 RD_EXPORT
8354  const rd_kafka_TopicDescription_t *topicdesc,
8355  size_t *cntp);
8356 
8357 
8365 RD_EXPORT
8367  const rd_kafka_TopicPartitionInfo_t *partition);
8368 
8369 
8380 RD_EXPORT
8382  const rd_kafka_TopicPartitionInfo_t *partition);
8383 
8395 RD_EXPORT
8396 const rd_kafka_Node_t **
8398  size_t *cntp);
8399 
8411 RD_EXPORT
8413  const rd_kafka_TopicPartitionInfo_t *partition,
8414  size_t *cntp);
8415 
8428 RD_EXPORT
8430  const rd_kafka_TopicDescription_t *topicdesc,
8431  size_t *cntp);
8432 
8443 RD_EXPORT
8444 const char *
8446 
8456 RD_EXPORT const rd_kafka_Uuid_t *rd_kafka_TopicDescription_topic_id(
8457  const rd_kafka_TopicDescription_t *topicdesc);
8458 
8466 RD_EXPORT
8468  const rd_kafka_TopicDescription_t *topicdesc);
8469 
8480 RD_EXPORT
8481 const rd_kafka_error_t *
8483 
8484 
8504 RD_EXPORT
8505 void rd_kafka_DescribeCluster(rd_kafka_t *rk,
8506  const rd_kafka_AdminOptions_t *options,
8507  rd_kafka_queue_t *rkqu);
8508 
8519 RD_EXPORT
8521  const rd_kafka_DescribeCluster_result_t *result,
8522  size_t *cntp);
8523 
8535 RD_EXPORT
8538  const rd_kafka_DescribeCluster_result_t *result,
8539  size_t *cntp);
8540 
8548 RD_EXPORT
8550  const rd_kafka_DescribeCluster_result_t *result);
8551 
8561 RD_EXPORT
8563  const rd_kafka_DescribeCluster_result_t *result);
8564 
8579 typedef struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t;
8580 
8582 typedef struct rd_kafka_ListConsumerGroupsResult_s
8584 
8595 RD_EXPORT
8596 void rd_kafka_ListConsumerGroups(rd_kafka_t *rk,
8597  const rd_kafka_AdminOptions_t *options,
8598  rd_kafka_queue_t *rkqu);
8599 
8610 RD_EXPORT
8612  const rd_kafka_ConsumerGroupListing_t *grplist);
8613 
8622 RD_EXPORT
8624  const rd_kafka_ConsumerGroupListing_t *grplist);
8625 
8633 RD_EXPORT
8635  const rd_kafka_ConsumerGroupListing_t *grplist);
8636 
8644 RD_EXPORT
8646  const rd_kafka_ConsumerGroupListing_t *grplist);
8647 
8659 RD_EXPORT
8663  size_t *cntp);
8664 
8678 RD_EXPORT
8679 const rd_kafka_error_t **rd_kafka_ListConsumerGroups_result_errors(
8681  size_t *cntp);
8682 
8694 typedef struct rd_kafka_ConsumerGroupDescription_s
8696 
8701 typedef struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t;
8702 
8707 typedef struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t;
8708 
8724 RD_EXPORT
8725 void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk,
8726  const char **groups,
8727  size_t groups_cnt,
8728  const rd_kafka_AdminOptions_t *options,
8729  rd_kafka_queue_t *rkqu);
8730 
8742 RD_EXPORT
8746  size_t *cntp);
8747 
8748 
8759 RD_EXPORT
8761  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8762 
8773 RD_EXPORT
8774 const rd_kafka_error_t *rd_kafka_ConsumerGroupDescription_error(
8775  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8776 
8784 RD_EXPORT
8786  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8787 
8788 
8799 RD_EXPORT
8801  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8802 
8815 RD_EXPORT
8818  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8819  size_t *cntp);
8820 
8828 RD_EXPORT
8830  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8831 
8842 RD_EXPORT
8844  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8845 
8853 RD_EXPORT
8855  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8856 
8869 RD_EXPORT
8871  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8872  size_t idx);
8873 
8884 RD_EXPORT
8886  const rd_kafka_MemberDescription_t *member);
8887 
8898 RD_EXPORT
8900  const rd_kafka_MemberDescription_t *member);
8901 
8912 RD_EXPORT
8914  const rd_kafka_MemberDescription_t *member);
8915 
8926 RD_EXPORT
8927 const char *
8929 
8940 RD_EXPORT
8942  const rd_kafka_MemberDescription_t *member);
8943 
8954 RD_EXPORT
8955 const rd_kafka_topic_partition_list_t *rd_kafka_MemberAssignment_partitions(
8956  const rd_kafka_MemberAssignment_t *assignment);
8957 
8969 typedef struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t;
8970 
8980 RD_EXPORT
8982 
8987 RD_EXPORT
8989 
8995 RD_EXPORT void
8997  size_t del_group_cnt);
8998 
9014 RD_EXPORT
9015 void rd_kafka_DeleteGroups(rd_kafka_t *rk,
9016  rd_kafka_DeleteGroup_t **del_groups,
9017  size_t del_group_cnt,
9018  const rd_kafka_AdminOptions_t *options,
9019  rd_kafka_queue_t *rkqu);
9020 
9021 
9022 
9023 /*
9024  * DeleteGroups result type and methods
9025  */
9026 
9035 RD_EXPORT const rd_kafka_group_result_t **rd_kafka_DeleteGroups_result_groups(
9036  const rd_kafka_DeleteGroups_result_t *result,
9037  size_t *cntp);
9038 
9049 typedef struct rd_kafka_ListConsumerGroupOffsets_s
9051 
9066  const char *group_id,
9067  const rd_kafka_topic_partition_list_t *partitions);
9068 
9074  rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets);
9075 
9082  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
9083  size_t list_grpoffset_cnt);
9084 
9102 RD_EXPORT
9104  rd_kafka_t *rk,
9105  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
9106  size_t list_grpoffsets_cnt,
9107  const rd_kafka_AdminOptions_t *options,
9108  rd_kafka_queue_t *rkqu);
9109 
9110 
9111 
9112 /*
9113  * ListConsumerGroupOffsets result type and methods
9114  */
9115 
9127 RD_EXPORT const rd_kafka_group_result_t **
9130  size_t *cntp);
9131 
9132 
9133 
9144 typedef struct rd_kafka_AlterConsumerGroupOffsets_s
9146 
9161  const char *group_id,
9162  const rd_kafka_topic_partition_list_t *partitions);
9163 
9169  rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets);
9170 
9177  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9178  size_t alter_grpoffset_cnt);
9179 
9198 RD_EXPORT
9200  rd_kafka_t *rk,
9201  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
9202  size_t alter_grpoffsets_cnt,
9203  const rd_kafka_AdminOptions_t *options,
9204  rd_kafka_queue_t *rkqu);
9205 
9206 
9207 
9208 /*
9209  * AlterConsumerGroupOffsets result type and methods
9210  */
9211 
9223 RD_EXPORT const rd_kafka_group_result_t **
9226  size_t *cntp);
9227 
9228 
9229 
9240 typedef struct rd_kafka_DeleteConsumerGroupOffsets_s
9242 
9257  const char *group,
9258  const rd_kafka_topic_partition_list_t *partitions);
9259 
9265  rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets);
9266 
9273  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9274  size_t del_grpoffset_cnt);
9275 
9294 RD_EXPORT
9296  rd_kafka_t *rk,
9297  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
9298  size_t del_grpoffsets_cnt,
9299  const rd_kafka_AdminOptions_t *options,
9300  rd_kafka_queue_t *rkqu);
9301 
9302 
9303 
9304 /*
9305  * DeleteConsumerGroupOffsets result type and methods
9306  */
9307 
9316 RD_EXPORT const rd_kafka_group_result_t **
9319  size_t *cntp);
9320 
9334  /* Used to retrieve the offset with the largest timestamp of a partition
9335  * as message timestamps can be specified client side this may not match
9336  * the log end offset returned by SPEC_LATEST.
9337  */
9338  RD_KAFKA_OFFSET_SPEC_MAX_TIMESTAMP = -3,
9339  /* Used to retrieve the offset with the earliest timestamp of a
9340  partition. */
9341  RD_KAFKA_OFFSET_SPEC_EARLIEST = -2,
9342  /* Used to retrieve the offset with the latest timestamp of a partition.
9343  */
9344  RD_KAFKA_OFFSET_SPEC_LATEST = -1,
9346 
9351 typedef struct rd_kafka_ListOffsetsResultInfo_s
9353 
9357 RD_EXPORT
9358 const rd_kafka_topic_partition_t *
9360  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9361 
9365 RD_EXPORT
9367  const rd_kafka_ListOffsetsResultInfo_t *result_info);
9368 
9373 RD_EXPORT
9376  size_t *cntp);
9377 
9402 RD_EXPORT
9403 void rd_kafka_ListOffsets(rd_kafka_t *rk,
9404  rd_kafka_topic_partition_list_t *topic_partitions,
9405  const rd_kafka_AdminOptions_t *options,
9406  rd_kafka_queue_t *rkqu);
9407 
9420  RD_KAFKA_SCRAM_MECHANISM_UNKNOWN = 0,
9421  RD_KAFKA_SCRAM_MECHANISM_SHA_256 = 1,
9422  RD_KAFKA_SCRAM_MECHANISM_SHA_512 = 2,
9423  RD_KAFKA_SCRAM_MECHANISM__CNT
9425 
9431 typedef struct rd_kafka_ScramCredentialInfo_s rd_kafka_ScramCredentialInfo_t;
9432 
9436 RD_EXPORT
9438  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9439 
9443 RD_EXPORT
9445  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
9446 
9453 typedef struct rd_kafka_UserScramCredentialsDescription_s
9455 
9459 RD_EXPORT
9461  const rd_kafka_UserScramCredentialsDescription_t *description);
9462 
9466 RD_EXPORT
9467 const rd_kafka_error_t *rd_kafka_UserScramCredentialsDescription_error(
9468  const rd_kafka_UserScramCredentialsDescription_t *description);
9469 
9474 RD_EXPORT
9476  const rd_kafka_UserScramCredentialsDescription_t *description);
9477 
9482 RD_EXPORT
9485  const rd_kafka_UserScramCredentialsDescription_t *description,
9486  size_t idx);
9487 
9497 RD_EXPORT
9501  size_t *cntp);
9502 
9514 RD_EXPORT
9516  rd_kafka_t *rk,
9517  const char **users,
9518  size_t user_cnt,
9519  const rd_kafka_AdminOptions_t *options,
9520  rd_kafka_queue_t *rkqu);
9521 
9525 typedef struct rd_kafka_UserScramCredentialAlteration_s
9527 
9548 RD_EXPORT
9550 rd_kafka_UserScramCredentialUpsertion_new(const char *username,
9551  rd_kafka_ScramMechanism_t mechanism,
9552  int32_t iterations,
9553  const unsigned char *password,
9554  size_t password_size,
9555  const unsigned char *salt,
9556  size_t salt_size);
9557 
9567 RD_EXPORT
9569 rd_kafka_UserScramCredentialDeletion_new(const char *username,
9570  rd_kafka_ScramMechanism_t mechanism);
9571 
9572 
9576 RD_EXPORT
9579 
9583 RD_EXPORT
9586  size_t alteration_cnt);
9587 
9591 typedef struct rd_kafka_AlterUserScramCredentials_result_response_s
9593 
9598 RD_EXPORT
9601 
9606 RD_EXPORT
9607 const rd_kafka_error_t *
9610 
9619 RD_EXPORT
9623  size_t *cntp);
9624 
9638 RD_EXPORT
9640  rd_kafka_t *rk,
9642  size_t alteration_cnt,
9643  const rd_kafka_AdminOptions_t *options,
9644  rd_kafka_queue_t *rkqu);
9645 
9658 typedef struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t;
9659 
9665 
9669 RD_EXPORT const rd_kafka_error_t *
9670 rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres);
9671 
9672 
9676 RD_EXPORT const char *
9678 
9686  1,
9689  RD_KAFKA_ACL_PERMISSION_TYPE__CNT
9691 
9695 RD_EXPORT const char *rd_kafka_AclPermissionType_name(
9696  rd_kafka_AclPermissionType_t acl_permission_type);
9697 
9716 RD_EXPORT rd_kafka_AclBinding_t *
9718  const char *name,
9719  rd_kafka_ResourcePatternType_t resource_pattern_type,
9720  const char *principal,
9721  const char *host,
9722  rd_kafka_AclOperation_t operation,
9723  rd_kafka_AclPermissionType_t permission_type,
9724  char *errstr,
9725  size_t errstr_size);
9726 
9753  rd_kafka_ResourceType_t restype,
9754  const char *name,
9755  rd_kafka_ResourcePatternType_t resource_pattern_type,
9756  const char *principal,
9757  const char *host,
9758  rd_kafka_AclOperation_t operation,
9759  rd_kafka_AclPermissionType_t permission_type,
9760  char *errstr,
9761  size_t errstr_size);
9762 
9766 RD_EXPORT rd_kafka_ResourceType_t
9768 
9774 RD_EXPORT const char *
9776 
9782 RD_EXPORT const char *
9784 
9790 RD_EXPORT const char *
9792 
9796 RD_EXPORT rd_kafka_AclOperation_t
9798 
9804 
9810 
9814 RD_EXPORT const rd_kafka_error_t *
9816 
9817 
9822 RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding);
9823 
9824 
9830 RD_EXPORT void
9832  size_t acl_bindings_cnt);
9833 
9841 RD_EXPORT const rd_kafka_acl_result_t **
9843  size_t *cntp);
9844 
9861 RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk,
9862  rd_kafka_AclBinding_t **new_acls,
9863  size_t new_acls_cnt,
9864  const rd_kafka_AdminOptions_t *options,
9865  rd_kafka_queue_t *rkqu);
9866 
9880 RD_EXPORT const rd_kafka_AclBinding_t **
9882  size_t *cntp);
9883 
9898 RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk,
9899  rd_kafka_AclBindingFilter_t *acl_filter,
9900  const rd_kafka_AdminOptions_t *options,
9901  rd_kafka_queue_t *rkqu);
9902 
9909 typedef struct rd_kafka_DeleteAcls_result_response_s
9911 
9919 RD_EXPORT const rd_kafka_DeleteAcls_result_response_t **
9921  size_t *cntp);
9922 
9927 RD_EXPORT const rd_kafka_error_t *rd_kafka_DeleteAcls_result_response_error(
9928  const rd_kafka_DeleteAcls_result_response_t *result_response);
9929 
9930 
9937 RD_EXPORT const rd_kafka_AclBinding_t **
9939  const rd_kafka_DeleteAcls_result_response_t *result_response,
9940  size_t *matching_acls_cntp);
9941 
9958 RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk,
9959  rd_kafka_AclBindingFilter_t **del_acls,
9960  size_t del_acls_cnt,
9961  const rd_kafka_AdminOptions_t *options,
9962  rd_kafka_queue_t *rkqu);
9963 
9977 typedef struct rd_kafka_ElectLeaders_s rd_kafka_ElectLeaders_t;
9978 
9987 
10001 RD_EXPORT rd_kafka_ElectLeaders_t *
10003  rd_kafka_topic_partition_list_t *partitions);
10004 
10011 RD_EXPORT void
10013 
10037 RD_EXPORT void rd_kafka_ElectLeaders(rd_kafka_t *rk,
10038  rd_kafka_ElectLeaders_t *elect_leaders,
10039  const rd_kafka_AdminOptions_t *options,
10040  rd_kafka_queue_t *rkqu);
10041 
10053 RD_EXPORT const rd_kafka_topic_partition_result_t **
10055  const rd_kafka_ElectLeaders_result_t *result,
10056  size_t *cntp);
10057 
10110 RD_EXPORT
10112 rd_kafka_oauthbearer_set_token(rd_kafka_t *rk,
10113  const char *token_value,
10114  int64_t md_lifetime_ms,
10115  const char *md_principal_name,
10116  const char **extensions,
10117  size_t extension_size,
10118  char *errstr,
10119  size_t errstr_size);
10120 
10141 RD_EXPORT
10143  const char *errstr);
10144 
10331 RD_EXPORT
10332 rd_kafka_error_t *rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms);
10333 
10334 
10335 
10380 RD_EXPORT
10381 rd_kafka_error_t *rd_kafka_begin_transaction(rd_kafka_t *rk);
10382 
10383 
10455 RD_EXPORT
10456 rd_kafka_error_t *rd_kafka_send_offsets_to_transaction(
10457  rd_kafka_t *rk,
10458  const rd_kafka_topic_partition_list_t *offsets,
10459  const rd_kafka_consumer_group_metadata_t *cgmetadata,
10460  int timeout_ms);
10461 
10462 
10530 RD_EXPORT
10531 rd_kafka_error_t *rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms);
10532 
10533 
10592 RD_EXPORT
10593 rd_kafka_error_t *rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms);
10594 
10595 
10598 /* @cond NO_DOC */
10599 #ifdef __cplusplus
10600 }
10601 #endif
10602 #endif /* _RDKAFKA_H_ */
10603 /* @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:7036
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:407
rd_kafka_ConsumerGroupDescription_t
struct rd_kafka_ConsumerGroupDescription_s rd_kafka_ConsumerGroupDescription_t
DescribeConsumerGroups result type.
Definition: rdkafka.h:8694
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:6610
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:5031
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:379
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:393
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:7728
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:361
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:6369
RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafka.h:500
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:924
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:572
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:5801
RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
@ RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
Definition: rdkafka.h:634
RD_KAFKA_RESP_ERR__TRANSPORT
@ RD_KAFKA_RESP_ERR__TRANSPORT
Definition: rdkafka.h:296
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:5185
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:620
RD_KAFKA_RESOURCE_PATTERN_LITERAL
@ RD_KAFKA_RESOURCE_PATTERN_LITERAL
Definition: rdkafka.h:7844
RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
@ RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafka.h:618
rd_kafka_group_member_info::client_host
char * client_host
Definition: rdkafka.h:5139
RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
@ RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
Definition: rdkafka.h:636
RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
Definition: rdkafka.h:7731
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:5829
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:508
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:927
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:1461
RD_KAFKA_RESP_ERR__INVALID_ARG
@ RD_KAFKA_RESP_ERR__INVALID_ARG
Definition: rdkafka.h:317
rd_kafka_ElectLeaders_destroy
RD_EXPORT void rd_kafka_ElectLeaders_destroy(rd_kafka_ElectLeaders_t *elect_leaders)
Destroy and free a rd_kafka_ElectLeaders_t object previously created with rd_kafka_ElectLeaders_new()
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_topic_partition_result_error
const RD_EXPORT rd_kafka_error_t * rd_kafka_topic_partition_result_error(const rd_kafka_topic_partition_result_t *partition_result)
rd_kafka_group_info::protocol_type
char * protocol_type
Definition: rdkafka.h:5183
rd_kafka_AlterConfigOpType_t
rd_kafka_AlterConfigOpType_t
Incremental alter configs operations.
Definition: rdkafka.h:7854
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:7343
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:7057
rd_kafka_metadata_broker::host
char * host
Definition: rdkafka.h:4993
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:347
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:6332
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:2488
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:5835
rd_kafka_group_info::member_cnt
int member_cnt
Definition: rdkafka.h:5186
RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
@ RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
Definition: rdkafka.h:405
RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
@ RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
Definition: rdkafka.h:609
RD_KAFKA_ADMIN_OP_DELETERECORDS
@ RD_KAFKA_ADMIN_OP_DELETERECORDS
Definition: rdkafka.h:7043
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:8279
rd_kafka_ConsumerGroupListing_t
struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t
ListConsumerGroups result for a single group.
Definition: rdkafka.h:8579
rd_kafka_group_info::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:5181
rd_kafka_topic_partition_s::topic
char * topic
Definition: rdkafka.h:922
RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
@ RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
Definition: rdkafka.h:7061
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:5028
rd_kafka_group_list::groups
struct rd_kafka_group_info * groups
Definition: rdkafka.h:5195
RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
@ RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
Definition: rdkafka.h:337
rd_kafka_metadata_partition::leader
int32_t leader
Definition: rdkafka.h:5003
RD_KAFKA_RESP_ERR__FS
@ RD_KAFKA_RESP_ERR__FS
Definition: rdkafka.h:311
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:9240
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:5811
RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafka.h:474
rd_kafka_group_info::group
char * group
Definition: rdkafka.h:5180
rd_kafka_message_s::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:1462
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:5136
rd_kafka_consumer_group_metadata_member_id
const RD_EXPORT char * rd_kafka_consumer_group_metadata_member_id(const rd_kafka_consumer_group_metadata_t *group_metadata)
Get member id of a group metadata.
rd_kafka_MemberDescription_assignment
const RD_EXPORT rd_kafka_MemberAssignment_t * rd_kafka_MemberDescription_assignment(const rd_kafka_MemberDescription_t *member)
Gets assignment of member.
rd_kafka_interceptor_add_on_broker_state_change
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_broker_state_change(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change, void *ic_opaque)
Append an on_broker_state_change() interceptor.
rd_kafka_DescribeConsumerGroups
RD_EXPORT void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk, const char **groups, size_t groups_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe groups from cluster as specified by the groups array of size groups_cnt elements.
RD_KAFKA_RESP_ERR__TIMED_OUT
@ RD_KAFKA_RESP_ERR__TIMED_OUT
Definition: rdkafka.h:319
RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
@ RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafka.h:536
RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
@ RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafka.h:543
RD_KAFKA_RESP_ERR__RETRY
@ RD_KAFKA_RESP_ERR__RETRY
Definition: rdkafka.h:383
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:7340
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:7042
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:514
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:7721
RD_KAFKA_VTYPE_PARTITION
@ RD_KAFKA_VTYPE_PARTITION
Definition: rdkafka.h:1143
RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
@ RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafka.h:556
RD_KAFKA_RESP_ERR__INCONSISTENT
@ RD_KAFKA_RESP_ERR__INCONSISTENT
Definition: rdkafka.h:391
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:623
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:1629
RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
@ RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
Definition: rdkafka.h:576
RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
@ RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
Definition: rdkafka.h:502
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:6496
RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
@ RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
Definition: rdkafka.h:551
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:921
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:566
rd_kafka_err_desc::desc
const char * desc
Definition: rdkafka.h:669
rd_kafka_conf_res_t
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1748
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:339
rd_kafka_ElectLeaders
RD_EXPORT void rd_kafka_ElectLeaders(rd_kafka_t *rk, rd_kafka_ElectLeaders_t *elect_leaders, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Elect Leaders for the provided Topic Partitions according to the specified election type.
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:445
rd_kafka_ConfigSource_t
rd_kafka_ConfigSource_t
Apache Kafka config sources.
Definition: rdkafka.h:7718
rd_kafka_metadata_broker::id
int32_t id
Definition: rdkafka.h:4992
RD_KAFKA_RESP_ERR__INTR
@ RD_KAFKA_RESP_ERR__INTR
Definition: rdkafka.h:363
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:9144
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:8701
rd_kafka_message_s::key_len
size_t key_len
Definition: rdkafka.h:1474
rd_kafka_CreateAcls_result_t
rd_kafka_event_t rd_kafka_CreateAcls_result_t
Definition: rdkafka.h:5803
RD_KAFKA_RESP_ERR_INVALID_MSG
@ RD_KAFKA_RESP_ERR_INVALID_MSG
Definition: rdkafka.h:427
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:5032
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:6442
RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
@ RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
Definition: rdkafka.h:9688
rd_kafka_AlterConsumerGroupOffsets_result_t
rd_kafka_event_t rd_kafka_AlterConsumerGroupOffsets_result_t
Definition: rdkafka.h:5827
rd_kafka_ListConsumerGroupsResult_t
struct rd_kafka_ListConsumerGroupsResult_s rd_kafka_ListConsumerGroupsResult_t
Definition: rdkafka.h:8582
RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
@ RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
Definition: rdkafka.h:451
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:7840
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:1639
rd_kafka_group_member_info::member_metadata_size
int member_metadata_size
Definition: rdkafka.h:5142
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:375
RD_KAFKA_ADMIN_OP_DELETEGROUPS
@ RD_KAFKA_ADMIN_OP_DELETEGROUPS
Definition: rdkafka.h:7044
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:486
rd_kafka_AlterUserScramCredentials_result_t
rd_kafka_event_t rd_kafka_AlterUserScramCredentials_result_t
Definition: rdkafka.h:5837
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:5017
rd_kafka_group_member_info::member_id
char * member_id
Definition: rdkafka.h:5137
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:668
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:1140
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:1166
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:7047
RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:600
rd_kafka_CreateTopics_result_t
rd_kafka_event_t rd_kafka_CreateTopics_result_t
Definition: rdkafka.h:5799
RD_KAFKA_RESP_ERR__FATAL
@ RD_KAFKA_RESP_ERR__FATAL
Definition: rdkafka.h:389
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:7064
RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
@ RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
Definition: rdkafka.h:369
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:7053
RD_KAFKA_RESP_ERR_SECURITY_DISABLED
@ RD_KAFKA_RESP_ERR_SECURITY_DISABLED
Definition: rdkafka.h:547
rd_kafka_vtype_t
rd_kafka_vtype_t
Var-arg tag types.
Definition: rdkafka.h:1139
rd_kafka_CreatePartitions_result_t
rd_kafka_event_t rd_kafka_CreatePartitions_result_t
Definition: rdkafka.h:5809
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:472
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:7046
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:349
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:564
RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
Definition: rdkafka.h:594
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:7838
RD_KAFKA_RESP_ERR__QUEUE_FULL
@ RD_KAFKA_RESP_ERR__QUEUE_FULL
Definition: rdkafka.h:321
RD_KAFKA_ELECTION_TYPE_UNCLEAN
@ RD_KAFKA_ELECTION_TYPE_UNCLEAN
Definition: rdkafka.h:9985
RD_KAFKA_VTYPE_KEY
@ RD_KAFKA_VTYPE_KEY
Definition: rdkafka.h:1145
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:453
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:7822
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:1753
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:9687
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:387
RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
@ RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
Definition: rdkafka.h:605
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:506
rd_kafka_ElectLeaders_result_partitions
const RD_EXPORT rd_kafka_topic_partition_result_t ** rd_kafka_ElectLeaders_result_partitions(const rd_kafka_ElectLeaders_result_t *result, size_t *cntp)
Get the array of topic partition result objects from the elect leaders result event and populates the...
rd_kafka_event_ElectLeaders_result
const RD_EXPORT rd_kafka_ElectLeaders_result_t * rd_kafka_event_ElectLeaders_result(rd_kafka_event_t *rkev)
Get ElectLeaders result.
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:6393
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:5448
RD_KAFKA_THREAD_BACKGROUND
@ RD_KAFKA_THREAD_BACKGROUND
Definition: rdkafka.h:5449
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:574
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:6563
rd_kafka_metadata_broker::port
int port
Definition: rdkafka.h:4994
RD_KAFKA_RESOURCE_TRANSACTIONAL_ID
@ RD_KAFKA_RESOURCE_TRANSACTIONAL_ID
Definition: rdkafka.h:7828
rd_kafka_AclPermissionType_t
rd_kafka_AclPermissionType_t
Apache Kafka ACL permission types.
Definition: rdkafka.h:9683
RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
@ RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
Definition: rdkafka.h:7055
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:425
rd_kafka_DescribeCluster_result_t
rd_kafka_event_t rd_kafka_DescribeCluster_result_t
Definition: rdkafka.h:5833
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:9664
RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
Definition: rdkafka.h:7725
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:5196
RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
@ RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
Definition: rdkafka.h:365
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:286
RD_KAFKA_RESP_ERR__BAD_COMPRESSION
@ RD_KAFKA_RESP_ERR__BAD_COMPRESSION
Definition: rdkafka.h:290
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:484
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:6589
RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
@ RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
Definition: rdkafka.h:530
RD_KAFKA_VTYPE_HEADERS
@ RD_KAFKA_VTYPE_HEADERS
Definition: rdkafka.h:1155
rd_kafka_metadata_partition::replica_cnt
int replica_cnt
Definition: rdkafka.h:5004
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafka.h:568
RD_KAFKA_RESP_ERR_INVALID_RECORD
@ RD_KAFKA_RESP_ERR_INVALID_RECORD
Definition: rdkafka.h:616
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:586
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:399
rd_kafka_metadata_broker
Broker information.
Definition: rdkafka.h:4991
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:972
rd_kafka_NewTopic_t
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:7368
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:494
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:596
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:592
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:5016
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:7050
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:294
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:496
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:510
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_ConsumerGroupListing_type
RD_EXPORT rd_kafka_consumer_group_type_t rd_kafka_ConsumerGroupListing_type(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets type for the grplist group.
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:397
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:584
RD_KAFKA_ADMIN_OP_DESCRIBETOPICS
@ RD_KAFKA_ADMIN_OP_DESCRIBETOPICS
Definition: rdkafka.h:7062
RD_KAFKA_VTYPE_TOPIC
@ RD_KAFKA_VTYPE_TOPIC
Definition: rdkafka.h:1141
rd_kafka_metadata_partition::err
rd_kafka_resp_err_t err
Definition: rdkafka.h:5002
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:7827
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:625
rd_kafka_OffsetSpec_t
rd_kafka_OffsetSpec_t
Allows to specify the desired offsets when using ListOffsets.
Definition: rdkafka.h:9333
rd_kafka_metadata::brokers
struct rd_kafka_metadata_broker * brokers
Definition: rdkafka.h:5026
rd_kafka_topic_partition_s::metadata
void * metadata
Definition: rdkafka.h:925
rd_kafka_message_s::payload
void * payload
Definition: rdkafka.h:1465
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:7737
RD_KAFKA_RESOURCE_GROUP
@ RD_KAFKA_RESOURCE_GROUP
Definition: rdkafka.h:7826
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:8180
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:5145
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:5179
RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:560
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:562
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_consumer_group_type_name
const RD_EXPORT char * rd_kafka_consumer_group_type_name(rd_kafka_consumer_group_type_t type)
Returns a name for a group type code.
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:580
RD_KAFKA_ACL_OPERATION_WRITE
@ RD_KAFKA_ACL_OPERATION_WRITE
Definition: rdkafka.h:7341
rd_kafka_message_s::offset
int64_t offset
Definition: rdkafka.h:1476
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:5153
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:355
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:9351
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:7825
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:498
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_ElectLeaders_new
RD_EXPORT rd_kafka_ElectLeaders_t * rd_kafka_ElectLeaders_new(rd_kafka_ElectionType_t election_type, rd_kafka_topic_partition_list_t *partitions)
Create a new rd_kafka_ElectLeaders_t object. This object is later passed to rd_kafka_ElectLeaders().
RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
@ RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafka.h:539
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:429
RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
@ RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafka.h:449
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:441
RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_ID
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_ID
Definition: rdkafka.h:640
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:351
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:9658
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:5825
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:433
RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
@ RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
Definition: rdkafka.h:331
RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
@ RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafka.h:638
rd_kafka_topic_partition_list_s::elems
rd_kafka_topic_partition_t * elems
Definition: rdkafka.h:975
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:5001
RD_KAFKA_RESP_ERR__RESOLVE
@ RD_KAFKA_RESP_ERR__RESOLVE
Definition: rdkafka.h:300
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:490
RD_KAFKA_ADMIN_OP_DELETEACLS
@ RD_KAFKA_ADMIN_OP_DELETEACLS
Definition: rdkafka.h:7049
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:9909
RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
@ RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafka.h:607
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_consumer_group_type_t
rd_kafka_consumer_group_type_t
Consumer group type.
Definition: rdkafka.h:5168
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:7350
rd_kafka_NewPartitions_t
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:7586
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:9525
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:549
RD_KAFKA_VTYPE_TIMESTAMP
@ RD_KAFKA_VTYPE_TIMESTAMP
Definition: rdkafka.h:1152
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:7081
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:526
RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
@ RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
Definition: rdkafka.h:488
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:5140
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:478
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:343
RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
@ RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafka.h:470
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:5815
RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafka.h:528
rd_kafka_consumer_group_type_code
RD_EXPORT rd_kafka_consumer_group_type_t rd_kafka_consumer_group_type_code(const char *name)
Returns a code for a group type name.
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:545
RD_KAFKA_ADMIN_OP_DELETETOPICS
@ RD_KAFKA_ADMIN_OP_DELETETOPICS
Definition: rdkafka.h:7039
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:9591
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:403
RD_KAFKA_RESP_ERR__INVALID_DIFFERENT_RECORD
@ RD_KAFKA_RESP_ERR__INVALID_DIFFERENT_RECORD
Definition: rdkafka.h:414
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:926
rd_kafka_DescribeConsumerGroups_result_t
rd_kafka_event_t rd_kafka_DescribeConsumerGroups_result_t
Definition: rdkafka.h:5821
RD_KAFKA_RESP_ERR_TELEMETRY_TOO_LARGE
@ RD_KAFKA_RESP_ERR_TELEMETRY_TOO_LARGE
Definition: rdkafka.h:656
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:292
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:598
rd_kafka_msg_partitioner_murmur2
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Murmur2 partitioner (Java compatible).
rd_kafka_Node_id
RD_EXPORT int rd_kafka_Node_id(const rd_kafka_Node_t *node)
Get the id of node.
RD_KAFKA_RESP_ERR_STALE_MEMBER_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_MEMBER_EPOCH
Definition: rdkafka.h:650
RD_KAFKA_ACL_OPERATION_ALTER
@ RD_KAFKA_ACL_OPERATION_ALTER
Definition: rdkafka.h:7344
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:5194
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:8285
rd_kafka_topic_partition_list_s::cnt
int cnt
Definition: rdkafka.h:973
RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
@ RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
Definition: rdkafka.h:395
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:5447
rd_kafka_Node_t
struct rd_kafka_Node_s rd_kafka_Node_t
Node (broker) information.
Definition: rdkafka.h:5072
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:5819
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:482
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:411
RD_KAFKA_ELECTION_TYPE_PREFERRED
@ RD_KAFKA_ELECTION_TYPE_PREFERRED
Definition: rdkafka.h:9984
rd_kafka_ConfigEntry_is_synonym
RD_EXPORT int rd_kafka_ConfigEntry_is_synonym(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_message_produce_errstr
const RD_EXPORT char * rd_kafka_message_produce_errstr(const rd_kafka_message_t *rkmessage)
Returns the error string for an errored produced rd_kafka_message_t or NULL if there was no error.
rd_kafka_ConfigSource_name
const RD_EXPORT char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
@ RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
Definition: rdkafka.h:7040
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:5510
RD_KAFKA_CONF_INVALID
@ RD_KAFKA_CONF_INVALID
Definition: rdkafka.h:1750
rd_kafka_metadata_topic::partition_cnt
int partition_cnt
Definition: rdkafka.h:5015
rd_kafka_vu_s
VTYPE + argument container for use with rd_kafka_produce_va()
Definition: rdkafka.h:1165
RD_KAFKA_ADMIN_OP_CREATETOPICS
@ RD_KAFKA_ADMIN_OP_CREATETOPICS
Definition: rdkafka.h:7038
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:1749
RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
@ RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
Definition: rdkafka.h:468
RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
@ RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafka.h:602
RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
@ RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafka.h:480
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:522
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:5029
rd_kafka_error_string
const RD_EXPORT char * rd_kafka_error_string(const rd_kafka_error_t *error)
RD_KAFKA_ADMIN_OP_ELECTLEADERS
@ RD_KAFKA_ADMIN_OP_ELECTLEADERS
Definition: rdkafka.h:7065
RD_KAFKA_RESOURCE_ANY
@ RD_KAFKA_RESOURCE_ANY
Definition: rdkafka.h:7824
RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
@ RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafka.h:590
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:359
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:385
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:2486
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:7352
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:923
RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
@ RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
Definition: rdkafka.h:7059
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:381
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:929
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:7063
RD_KAFKA_VTYPE_VALUE
@ RD_KAFKA_VTYPE_VALUE
Definition: rdkafka.h:1144
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:423
RD_KAFKA_RESOURCE_UNKNOWN
@ RD_KAFKA_RESOURCE_UNKNOWN
Definition: rdkafka.h:7823
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:1472
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:2499
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:632
RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
@ RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
Definition: rdkafka.h:7348
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:1151
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:309
rd_kafka_DescribeAcls_result_t
rd_kafka_event_t rd_kafka_DescribeAcls_result_t
Definition: rdkafka.h:5805
rd_kafka_DeleteGroup_t
struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t
Definition: rdkafka.h:8969
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:1484
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:5005
RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
@ RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
Definition: rdkafka.h:578
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:377
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:588
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:5143
RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
@ RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
Definition: rdkafka.h:341
rd_kafka_metadata
Metadata container.
Definition: rdkafka.h:5024
RD_KAFKA_RESP_ERR__FENCED
@ RD_KAFKA_RESP_ERR__FENCED
Definition: rdkafka.h:401
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:6353
rd_kafka_group_info
Group information.
Definition: rdkafka.h:5178
RD_KAFKA_CERT_ENC_PEM
@ RD_KAFKA_CERT_ENC_PEM
Definition: rdkafka.h:2502
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:1153
rd_kafka_err_desc
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:666
RD_KAFKA_RESP_ERR_INVALID_REQUEST
@ RD_KAFKA_RESP_ERR_INVALID_REQUEST
Definition: rdkafka.h:518
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:1464
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:7846
RD_KAFKA_RESP_ERR_NOT_COORDINATOR
@ RD_KAFKA_RESP_ERR_NOT_COORDINATOR
Definition: rdkafka.h:463
RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
@ RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
Definition: rdkafka.h:447
rd_kafka_topic_partition_list_s::size
int size
Definition: rdkafka.h:974
rd_kafka_ConfigEntry_t
struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t
Definition: rdkafka.h:7749
rd_kafka_DeleteConsumerGroupOffsets_destroy_array
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy_array(rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffset_cnt)
Helper function to destroy all DeleteConsumerGroupOffsets objects in the del_grpoffsets array (of del...
rd_kafka_NewPartitions_destroy
RD_EXPORT void rd_kafka_NewPartitions_destroy(rd_kafka_NewPartitions_t *new_parts)
Destroy and free a NewPartitions object previously created with rd_kafka_NewPartitions_new()
rd_kafka_UserScramCredentialAlteration_destroy_array
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy_array(rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt)
Destroys an array of UserScramCredentialAlteration.
rd_kafka_event_message_array
RD_EXPORT size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev, const rd_kafka_message_t **rkmessages, size_t size)
Extacts size message(s) from the event into the pre-allocated array rkmessages.
rd_kafka_message_broker_id
RD_EXPORT int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage)
Returns the broker id of the broker the message was produced to or fetched from.
rd_kafka_topic_partition_list_find
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_find(const rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Find element by topic and partition.
RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
@ RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafka.h:221
RD_KAFKA_RESP_ERR_UNSUPPORTED_ASSIGNOR
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_ASSIGNOR
Definition: rdkafka.h:648
RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
@ RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
Definition: rdkafka.h:7346
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:9453
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:417
RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafka.h:492
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:443
RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
@ RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
Definition: rdkafka.h:476
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:7337
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:9685
RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
@ RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
Definition: rdkafka.h:298
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:7875
RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
@ RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafka.h:524
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:7842
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:6468
RD_KAFKA_ACL_OPERATION_CREATE
@ RD_KAFKA_ACL_OPERATION_CREATE
Definition: rdkafka.h:7342
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_topic_partition_result_partition
const RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_result_partition(const rd_kafka_topic_partition_result_t *partition_result)
Topic Partition Result provides per-topic+partition operation result Consists of TopicPartition objec...
RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
@ RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafka.h:629
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:458
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:367
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_UNKNOWN_SUBSCRIPTION_ID
@ RD_KAFKA_RESP_ERR_UNKNOWN_SUBSCRIPTION_ID
Definition: rdkafka.h:653
RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
@ RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
Definition: rdkafka.h:371
rd_kafka_metadata::broker_cnt
int broker_cnt
Definition: rdkafka.h:5025
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:2500
RD_KAFKA_ACL_OPERATION_UNKNOWN
@ RD_KAFKA_ACL_OPERATION_UNKNOWN
Definition: rdkafka.h:7336
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:325
rd_kafka_DescribeTopics_result_t
rd_kafka_event_t rd_kafka_DescribeTopics_result_t
Definition: rdkafka.h:5831
RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
@ RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
Definition: rdkafka.h:409
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:5184
rd_kafka_cert_type_t
rd_kafka_cert_type_t
SSL certificate type.
Definition: rdkafka.h:2485
RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
@ RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafka.h:582
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:1185
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:307
rd_kafka_metadata_topic::topic
char * topic
Definition: rdkafka.h:5014
rd_kafka_AdminOptions_set_match_consumer_group_types
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_match_consumer_group_types(rd_kafka_AdminOptions_t *options, const rd_kafka_consumer_group_type_t *consumer_group_types, size_t consumer_group_types_cnt)
Set consumer groups types to query for.
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:558
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:667
rd_kafka_ScramMechanism_t
rd_kafka_ScramMechanism_t
Apache Kafka ScramMechanism values.
Definition: rdkafka.h:9419
RD_KAFKA_VTYPE_RKT
@ RD_KAFKA_VTYPE_RKT
Definition: rdkafka.h:1142
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:7509
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:8273
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:5839
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:928
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:627
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:5823
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:7345
RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
@ RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
Definition: rdkafka.h:335
RD_KAFKA_RESP_ERR__SSL
@ RD_KAFKA_RESP_ERR__SSL
Definition: rdkafka.h:327
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:5817
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_ElectLeaders_t
struct rd_kafka_ElectLeaders_s rd_kafka_ElectLeaders_t
Represents elect leaders request.
Definition: rdkafka.h:9977
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_ElectLeaders_result_t
rd_kafka_event_t rd_kafka_ElectLeaders_result_t
Definition: rdkafka.h:5841
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:431
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:614
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:7836
rd_kafka_ElectionType_t
rd_kafka_ElectionType_t
Apache Kafka Election Types.
Definition: rdkafka.h:9983
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:5000
RD_KAFKA_VTYPE_OPAQUE
@ RD_KAFKA_VTYPE_OPAQUE
Definition: rdkafka.h:1146
RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
@ RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
Definition: rdkafka.h:554
RD_KAFKA_CERT_ENC_DER
@ RD_KAFKA_CERT_ENC_DER
Definition: rdkafka.h:2501
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:439
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:2487
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:315
rd_kafka_ListConsumerGroupOffsets_t
struct rd_kafka_ListConsumerGroupOffsets_s rd_kafka_ListConsumerGroupOffsets_t
Definition: rdkafka.h:9049
RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
@ RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafka.h:512
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:345
RD_KAFKA_RESP_ERR__NO_OFFSET
@ RD_KAFKA_RESP_ERR__NO_OFFSET
Definition: rdkafka.h:353
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:357
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:6533
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:7066
RD_KAFKA_THREAD_BROKER
@ RD_KAFKA_THREAD_BROKER
Definition: rdkafka.h:5450
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:611
RD_KAFKA_RESP_ERR__ISR_INSUFF
@ RD_KAFKA_RESP_ERR__ISR_INSUFF
Definition: rdkafka.h:323
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:9431
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:5813
rd_kafka_MemberAssignment_t
struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t
Member assignment included in MemberDescription.
Definition: rdkafka.h:8707
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:516
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:7051
RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
@ RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
Definition: rdkafka.h:9684
RD_KAFKA_RESP_ERR_UNRELEASED_INSTANCE_ID
@ RD_KAFKA_RESP_ERR_UNRELEASED_INSTANCE_ID
Definition: rdkafka.h:645
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:7090
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:1469
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:5807
RD_KAFKA_ADMIN_OP_ALTERCONFIGS
@ RD_KAFKA_ADMIN_OP_ALTERCONFIGS
Definition: rdkafka.h:7041
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:7723
RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
@ RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
Definition: rdkafka.h:7734
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:5138
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:421
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:6202
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:6288
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:288
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:5182
RD_KAFKA_ADMIN_OP_ANY
@ RD_KAFKA_ADMIN_OP_ANY
Definition: rdkafka.h:7037
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:302
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:504
rd_kafka_AclOperation_t
rd_kafka_AclOperation_t
Apache Kafka ACL operation types. Common type for multiple Admin API functions.
Definition: rdkafka.h:7335
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:1463
RD_KAFKA_RESOURCE__CNT
@ RD_KAFKA_RESOURCE__CNT
Definition: rdkafka.h:7829
rd_kafka_topic_new
RD_EXPORT rd_kafka_topic_t * rd_kafka_topic_new(rd_kafka_t *rk, const char *topic, rd_kafka_topic_conf_t *conf)
Creates a new topic handle for topic named topic.
RD_KAFKA_RESP_ERR_FENCED_MEMBER_EPOCH
@ RD_KAFKA_RESP_ERR_FENCED_MEMBER_EPOCH
Definition: rdkafka.h:642
rd_kafka_metadata_topic
Topic information.
Definition: rdkafka.h:5013
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:5007
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:6318
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:5006
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:283
RD_KAFKA_RESP_ERR__PARTIAL
@ RD_KAFKA_RESP_ERR__PARTIAL
Definition: rdkafka.h:373
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:329
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:7048
RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafka.h:570
RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafka.h:520
RD_KAFKA_ACL_OPERATION_ALL
@ RD_KAFKA_ACL_OPERATION_ALL
Definition: rdkafka.h:7339
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:533
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:1625
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:333
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:313
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:1634
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:6420
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.