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 0x020200ff
171 
180 RD_EXPORT
182 
188 RD_EXPORT
189 const char *rd_kafka_version_str(void);
190 
209 typedef enum rd_kafka_type_t {
213 
214 
225 
226 
227 
234 RD_EXPORT
235 const char *rd_kafka_get_debug_contexts(void);
236 
244 #define RD_KAFKA_DEBUG_CONTEXTS \
245  "all,generic,broker,topic,metadata,feature,queue,msg,protocol,cgrp," \
246  "security,fetch,interceptor,plugin,consumer,admin,eos,mock,assignor," \
247  "conf"
248 
249 
250 /* @cond NO_DOC */
251 /* Private types to provide ABI compatibility */
252 typedef struct rd_kafka_s rd_kafka_t;
253 typedef struct rd_kafka_topic_s rd_kafka_topic_t;
254 typedef struct rd_kafka_conf_s rd_kafka_conf_t;
255 typedef struct rd_kafka_topic_conf_s rd_kafka_topic_conf_t;
256 typedef struct rd_kafka_queue_s rd_kafka_queue_t;
257 typedef struct rd_kafka_op_s rd_kafka_event_t;
258 typedef struct rd_kafka_topic_result_s rd_kafka_topic_result_t;
259 typedef struct rd_kafka_consumer_group_metadata_s
260  rd_kafka_consumer_group_metadata_t;
261 typedef struct rd_kafka_error_s rd_kafka_error_t;
262 typedef struct rd_kafka_headers_s rd_kafka_headers_t;
263 typedef struct rd_kafka_group_result_s rd_kafka_group_result_t;
264 typedef struct rd_kafka_acl_result_s rd_kafka_acl_result_t;
265 /* @endcond */
266 
267 
280 typedef enum {
281  /* Internal errors to rdkafka: */
409 
412 
413  /* Kafka broker errors: */
429 #define RD_KAFKA_RESP_ERR_NOT_LEADER_OR_FOLLOWER \
430  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION
431  RD_KAFKA_RESP_ERR_NOT_LEADER_FOR_PARTITION = 6,
449 #define RD_KAFKA_RESP_ERR_GROUP_LOAD_IN_PROGRESS \
450  RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
454 #define RD_KAFKA_RESP_ERR_GROUP_COORDINATOR_NOT_AVAILABLE \
455  RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
459 #define RD_KAFKA_RESP_ERR_NOT_COORDINATOR_FOR_GROUP \
460  RD_KAFKA_RESP_ERR_NOT_COORDINATOR
633 
634  RD_KAFKA_RESP_ERR_END_ALL,
636 
637 
645  const char *name;
646  const char *desc;
647 };
648 
649 
653 RD_EXPORT
654 void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs,
655  size_t *cntp);
656 
657 
658 
664 RD_EXPORT
666 
667 
668 
674 RD_EXPORT
676 
677 
703 RD_EXPORT
705 
706 
731 RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox);
732 
733 
746 RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void);
747 
748 
749 
776 RD_EXPORT
778 rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size);
779 
780 
800  const char *reason);
801 
802 
807 RD_EXPORT
808 rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error);
809 
818 RD_EXPORT
819 const char *rd_kafka_error_name(const rd_kafka_error_t *error);
820 
827 RD_EXPORT
828 const char *rd_kafka_error_string(const rd_kafka_error_t *error);
829 
830 
835 RD_EXPORT
836 int rd_kafka_error_is_fatal(const rd_kafka_error_t *error);
837 
838 
843 RD_EXPORT
844 int rd_kafka_error_is_retriable(const rd_kafka_error_t *error);
845 
846 
857 RD_EXPORT
858 int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error);
859 
865 RD_EXPORT
866 void rd_kafka_error_destroy(rd_kafka_error_t *error);
867 
868 
877 RD_EXPORT
879  const char *fmt,
880  ...) RD_FORMAT(printf, 2, 3);
881 
882 
898 typedef struct rd_kafka_topic_partition_s {
899  char *topic;
900  int32_t partition;
901  int64_t offset;
902  void *metadata;
903  size_t metadata_size;
904  void *opaque;
906  void *_private;
911 
916 RD_EXPORT
918 
919 
928 RD_EXPORT
931  int32_t leader_epoch);
932 
941 RD_EXPORT
943  const rd_kafka_topic_partition_t *rktpar);
944 
949 typedef struct rd_kafka_topic_partition_list_s {
950  int cnt;
951  int size;
954 
969 RD_EXPORT
971 
975 RD_EXPORT
978 
988 RD_EXPORT
991  const char *topic,
992  int32_t partition);
993 
994 
1003 RD_EXPORT
1005  rd_kafka_topic_partition_list_t *rktparlist,
1006  const char *topic,
1007  int32_t start,
1008  int32_t stop);
1009 
1010 
1011 
1023 RD_EXPORT
1025  rd_kafka_topic_partition_list_t *rktparlist,
1026  const char *topic,
1027  int32_t partition);
1028 
1029 
1037 RD_EXPORT
1039  rd_kafka_topic_partition_list_t *rktparlist,
1040  int idx);
1041 
1042 
1050 RD_EXPORT
1053 
1054 
1055 
1063 RD_EXPORT
1065  rd_kafka_topic_partition_list_t *rktparlist,
1066  const char *topic,
1067  int32_t partition,
1068  int64_t offset);
1069 
1070 
1071 
1077 RD_EXPORT
1079  const rd_kafka_topic_partition_list_t *rktparlist,
1080  const char *topic,
1081  int32_t partition);
1082 
1083 
1094  rd_kafka_topic_partition_list_t *rktparlist,
1095  int (*cmp)(const void *a, const void *b, void *cmp_opaque),
1096  void *cmp_opaque);
1097 
1098 
1116 typedef enum rd_kafka_vtype_t {
1134 
1135 
1142 typedef struct rd_kafka_vu_s {
1145  union {
1146  const char *cstr;
1147  rd_kafka_topic_t *rkt;
1148  int i;
1149  int32_t i32;
1150  int64_t i64;
1151  struct {
1152  void *ptr;
1153  size_t size;
1154  } mem;
1155  struct {
1156  const char *name;
1157  const void *val;
1158  ssize_t size;
1159  } header;
1160  rd_kafka_headers_t *headers;
1161  void *ptr;
1162  char _pad[64];
1163  } u;
1164 } rd_kafka_vu_t;
1165 
1174 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
1175 
1181 #define RD_KAFKA_V_TOPIC(topic) \
1182  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
1183  (const char *)topic
1189 #define RD_KAFKA_V_RKT(rkt) \
1190  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
1191  (rd_kafka_topic_t *)rkt
1197 #define RD_KAFKA_V_PARTITION(partition) \
1198  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
1199  (int32_t)partition
1205 #define RD_KAFKA_V_VALUE(VALUE, LEN) \
1206  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
1207  (void *)VALUE, (size_t)LEN
1213 #define RD_KAFKA_V_KEY(KEY, LEN) \
1214  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
1215  (void *)KEY, (size_t)LEN
1223 #define RD_KAFKA_V_OPAQUE(msg_opaque) \
1224  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, msg_opaque), \
1225  (void *)msg_opaque
1232 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
1233  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), (int)msgflags
1240 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
1241  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
1242  (int64_t)timestamp
1251 #define RD_KAFKA_V_HEADER(NAME, VALUE, LEN) \
1252  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
1253  const void *, VALUE, ssize_t, LEN), \
1254  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
1255 
1267 #define RD_KAFKA_V_HEADERS(HDRS) \
1268  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
1269  (rd_kafka_headers_t *)HDRS
1270 
1271 
1302 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new(size_t initial_count);
1303 
1308 RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs);
1309 
1313 RD_EXPORT rd_kafka_headers_t *
1314 rd_kafka_headers_copy(const rd_kafka_headers_t *src);
1315 
1333 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs,
1334  const char *name,
1335  ssize_t name_size,
1336  const void *value,
1337  ssize_t value_size);
1338 
1346 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs,
1347  const char *name);
1348 
1349 
1367 RD_EXPORT rd_kafka_resp_err_t
1368 rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs,
1369  const char *name,
1370  const void **valuep,
1371  size_t *sizep);
1372 
1386 RD_EXPORT rd_kafka_resp_err_t
1387 rd_kafka_header_get(const rd_kafka_headers_t *hdrs,
1388  size_t idx,
1389  const char *name,
1390  const void **valuep,
1391  size_t *sizep);
1392 
1393 
1401 RD_EXPORT rd_kafka_resp_err_t
1402 rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs,
1403  size_t idx,
1404  const char **namep,
1405  const void **valuep,
1406  size_t *sizep);
1407 
1408 
1409 
1422 // FIXME: This doesn't show up in docs for some reason
1423 // "Compound rd_kafka_message_t is not documented."
1424 
1438 typedef struct rd_kafka_message_s {
1440  rd_kafka_topic_t *rkt;
1441  int32_t partition;
1442  void *payload;
1446  size_t len;
1449  void *key;
1451  size_t key_len;
1453  int64_t offset;
1461  void *_private;
1469 
1470 
1474 RD_EXPORT
1476 
1477 
1478 
1485 RD_EXPORT
1486 const char *rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage);
1487 
1488 
1500 RD_EXPORT
1502  rd_kafka_timestamp_type_t *tstype);
1503 
1504 
1505 
1512 RD_EXPORT
1514 
1515 
1522 RD_EXPORT
1524 
1525 
1542 RD_EXPORT rd_kafka_resp_err_t
1544  rd_kafka_headers_t **hdrsp);
1545 
1557 RD_EXPORT rd_kafka_resp_err_t
1559  rd_kafka_headers_t **hdrsp);
1560 
1561 
1574 RD_EXPORT
1576  rd_kafka_headers_t *hdrs);
1577 
1578 
1584 RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs);
1585 
1586 
1592 typedef enum {
1597 
1602 
1608 
1609 
1616 RD_EXPORT rd_kafka_msg_status_t
1618 
1619 
1627 RD_EXPORT int32_t
1629 
1630 
1646 typedef enum {
1648  RD_KAFKA_CONF_INVALID = -1,
1651  RD_KAFKA_CONF_OK = 0
1653 
1654 
1688 RD_EXPORT
1689 rd_kafka_conf_t *rd_kafka_conf_new(void);
1690 
1691 
1695 RD_EXPORT
1696 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1697 
1698 
1705 RD_EXPORT
1706 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1707 
1708 
1713 RD_EXPORT
1714 rd_kafka_conf_t *rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf,
1715  size_t filter_cnt,
1716  const char **filter);
1717 
1718 
1719 
1728 RD_EXPORT
1729 const rd_kafka_conf_t *rd_kafka_conf(rd_kafka_t *rk);
1730 
1731 
1751 RD_EXPORT
1753  const char *name,
1754  const char *value,
1755  char *errstr,
1756  size_t errstr_size);
1757 
1758 
1764 RD_EXPORT
1765 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1766 
1767 
1804  rd_kafka_conf_t *conf,
1805  void (*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque));
1806 
1807 
1811 RD_EXPORT
1812 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1813  void (*dr_cb)(rd_kafka_t *rk,
1814  void *payload,
1815  size_t len,
1816  rd_kafka_resp_err_t err,
1817  void *opaque,
1818  void *msg_opaque));
1819 
1850 RD_EXPORT
1852  rd_kafka_conf_t *conf,
1853  void (*dr_msg_cb)(rd_kafka_t *rk,
1854  const rd_kafka_message_t *rkmessage,
1855  void *opaque));
1856 
1857 
1865 RD_EXPORT
1867  rd_kafka_conf_t *conf,
1868  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque));
1869 
1974 RD_EXPORT
1976  rd_kafka_conf_t *conf,
1977  void (*rebalance_cb)(rd_kafka_t *rk,
1978  rd_kafka_resp_err_t err,
1979  rd_kafka_topic_partition_list_t *partitions,
1980  void *opaque));
1981 
1982 
1983 
2001 RD_EXPORT
2003  rd_kafka_conf_t *conf,
2004  void (*offset_commit_cb)(rd_kafka_t *rk,
2005  rd_kafka_resp_err_t err,
2007  void *opaque));
2008 
2009 
2032 RD_EXPORT
2033 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
2034  void (*error_cb)(rd_kafka_t *rk,
2035  int err,
2036  const char *reason,
2037  void *opaque));
2038 
2056 RD_EXPORT
2057 void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf,
2058  void (*throttle_cb)(rd_kafka_t *rk,
2059  const char *broker_name,
2060  int32_t broker_id,
2061  int throttle_time_ms,
2062  void *opaque));
2063 
2064 
2081 RD_EXPORT
2082 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
2083  void (*log_cb)(const rd_kafka_t *rk,
2084  int level,
2085  const char *fac,
2086  const char *buf));
2087 
2088 
2111 RD_EXPORT
2113  rd_kafka_conf_t *conf,
2114  int (*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque));
2115 
2165 RD_EXPORT
2167  rd_kafka_conf_t *conf,
2168  void (*oauthbearer_token_refresh_cb)(rd_kafka_t *rk,
2169  const char *oauthbearer_config,
2170  void *opaque));
2171 
2199 RD_EXPORT
2200 void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable);
2201 
2202 
2220 RD_EXPORT
2222  rd_kafka_conf_t *conf,
2223  int (*socket_cb)(int domain, int type, int protocol, void *opaque));
2224 
2225 
2226 
2242 RD_EXPORT void
2243 rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf,
2244  int (*connect_cb)(int sockfd,
2245  const struct sockaddr *addr,
2246  int addrlen,
2247  const char *id,
2248  void *opaque));
2249 
2261  rd_kafka_conf_t *conf,
2262  int (*closesocket_cb)(int sockfd, void *opaque));
2263 
2264 
2265 
2266 #ifndef _WIN32
2284 RD_EXPORT
2286  rd_kafka_conf_t *conf,
2287  int (*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque));
2288 #endif
2289 
2291 struct addrinfo;
2292 
2312 RD_EXPORT void
2313 rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf,
2314  int (*resolve_cb)(const char *node,
2315  const char *service,
2316  const struct addrinfo *hints,
2317  struct addrinfo **res,
2318  void *opaque));
2319 
2361 RD_EXPORT
2363  rd_kafka_conf_t *conf,
2364  int (*ssl_cert_verify_cb)(rd_kafka_t *rk,
2365  const char *broker_name,
2366  int32_t broker_id,
2367  int *x509_error,
2368  int depth,
2369  const char *buf,
2370  size_t size,
2371  char *errstr,
2372  size_t errstr_size,
2373  void *opaque));
2374 
2375 
2383 typedef enum rd_kafka_cert_type_t {
2387  RD_KAFKA_CERT__CNT,
2389 
2397 typedef enum rd_kafka_cert_enc_t {
2401  RD_KAFKA_CERT_ENC__CNT,
2403 
2404 
2447 RD_EXPORT rd_kafka_conf_res_t
2448 rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf,
2449  rd_kafka_cert_type_t cert_type,
2450  rd_kafka_cert_enc_t cert_enc,
2451  const void *buffer,
2452  size_t size,
2453  char *errstr,
2454  size_t errstr_size);
2455 
2456 
2471 RD_EXPORT
2472 void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf,
2473  void *callback_data);
2474 
2475 
2481 RD_EXPORT
2482 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
2483 
2488 RD_EXPORT
2489 void *rd_kafka_opaque(const rd_kafka_t *rk);
2490 
2491 
2492 
2506 RD_EXPORT
2507 void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf,
2508  rd_kafka_topic_conf_t *tconf);
2509 
2522 RD_EXPORT rd_kafka_topic_conf_t *
2524 
2525 
2545 RD_EXPORT
2546 rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf,
2547  const char *name,
2548  char *dest,
2549  size_t *dest_size);
2550 
2551 
2557 RD_EXPORT
2558 rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf,
2559  const char *name,
2560  char *dest,
2561  size_t *dest_size);
2562 
2563 
2572 RD_EXPORT
2573 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
2574 
2575 
2584 RD_EXPORT
2585 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
2586  size_t *cntp);
2587 
2592 RD_EXPORT
2593 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
2594 
2602 RD_EXPORT
2604 
2621 RD_EXPORT
2622 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
2623 
2624 
2628 RD_EXPORT
2629 rd_kafka_topic_conf_t *
2630 rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf);
2631 
2636 RD_EXPORT
2637 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup(rd_kafka_t *rk);
2638 
2639 
2643 RD_EXPORT
2644 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
2645 
2646 
2655 RD_EXPORT
2656 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
2657  const char *name,
2658  const char *value,
2659  char *errstr,
2660  size_t errstr_size);
2661 
2668 RD_EXPORT
2669 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf,
2670  void *rkt_opaque);
2671 
2672 
2692 RD_EXPORT
2694  rd_kafka_topic_conf_t *topic_conf,
2695  int32_t (*partitioner)(const rd_kafka_topic_t *rkt,
2696  const void *keydata,
2697  size_t keylen,
2698  int32_t partition_cnt,
2699  void *rkt_opaque,
2700  void *msg_opaque));
2701 
2702 
2729  rd_kafka_topic_conf_t *topic_conf,
2730  int (*msg_order_cmp)(const rd_kafka_message_t *a,
2731  const rd_kafka_message_t *b));
2732 
2733 
2741 RD_EXPORT
2742 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
2743  int32_t partition);
2744 
2745 
2746 /*******************************************************************
2747  * *
2748  * Partitioners provided by rdkafka *
2749  * *
2750  *******************************************************************/
2751 
2765 RD_EXPORT
2766 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
2767  const void *key,
2768  size_t keylen,
2769  int32_t partition_cnt,
2770  void *rkt_opaque,
2771  void *msg_opaque);
2772 
2786 RD_EXPORT
2787 int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt,
2788  const void *key,
2789  size_t keylen,
2790  int32_t partition_cnt,
2791  void *rkt_opaque,
2792  void *msg_opaque);
2793 
2809 RD_EXPORT
2810 int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt,
2811  const void *key,
2812  size_t keylen,
2813  int32_t partition_cnt,
2814  void *rkt_opaque,
2815  void *msg_opaque);
2816 
2817 
2831 RD_EXPORT
2832 int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt,
2833  const void *key,
2834  size_t keylen,
2835  int32_t partition_cnt,
2836  void *rkt_opaque,
2837  void *msg_opaque);
2838 
2853 RD_EXPORT
2854 int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt,
2855  const void *key,
2856  size_t keylen,
2857  int32_t partition_cnt,
2858  void *rkt_opaque,
2859  void *msg_opaque);
2860 
2861 
2875 RD_EXPORT
2876 int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt,
2877  const void *key,
2878  size_t keylen,
2879  int32_t partition_cnt,
2880  void *rkt_opaque,
2881  void *msg_opaque);
2882 
2883 
2898 RD_EXPORT
2899 int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt,
2900  const void *key,
2901  size_t keylen,
2902  int32_t partition_cnt,
2903  void *rkt_opaque,
2904  void *msg_opaque);
2905 
2906 
2946 RD_EXPORT
2948  rd_kafka_conf_t *conf,
2949  char *errstr,
2950  size_t errstr_size);
2951 
2952 
2967 RD_EXPORT
2968 void rd_kafka_destroy(rd_kafka_t *rk);
2969 
2970 
2975 RD_EXPORT
2976 void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags);
2977 
2992 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
2993 
2994 
2995 
2999 RD_EXPORT
3000 const char *rd_kafka_name(const rd_kafka_t *rk);
3001 
3002 
3006 RD_EXPORT
3007 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
3008 
3009 
3020 RD_EXPORT
3021 char *rd_kafka_memberid(const rd_kafka_t *rk);
3022 
3023 
3024 
3043 RD_EXPORT
3044 char *rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms);
3045 
3046 
3062 RD_EXPORT
3063 int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms);
3064 
3065 
3087 RD_EXPORT
3088 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk,
3089  const char *topic,
3090  rd_kafka_topic_conf_t *conf);
3091 
3092 
3093 
3102 RD_EXPORT
3103 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
3104 
3105 
3109 RD_EXPORT
3110 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
3111 
3112 
3117 RD_EXPORT
3118 void *rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt);
3119 
3120 
3127 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
3128 
3129 
3157 RD_EXPORT
3158 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
3159 
3160 
3171 RD_EXPORT
3172 void rd_kafka_yield(rd_kafka_t *rk);
3173 
3174 
3175 
3183 RD_EXPORT rd_kafka_resp_err_t
3185  rd_kafka_topic_partition_list_t *partitions);
3186 
3187 
3188 
3196 RD_EXPORT rd_kafka_resp_err_t
3198  rd_kafka_topic_partition_list_t *partitions);
3199 
3200 
3201 
3210 RD_EXPORT rd_kafka_resp_err_t
3212  const char *topic,
3213  int32_t partition,
3214  int64_t *low,
3215  int64_t *high,
3216  int timeout_ms);
3217 
3218 
3236  const char *topic,
3237  int32_t partition,
3238  int64_t *low,
3239  int64_t *high);
3240 
3241 
3242 
3268 RD_EXPORT rd_kafka_resp_err_t
3271  int timeout_ms);
3272 
3273 
3274 
3288 RD_EXPORT
3289 void *rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size);
3290 
3291 
3292 
3306 RD_EXPORT
3307 void *rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size);
3308 
3309 
3310 
3327 RD_EXPORT
3328 void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr);
3329 
3330 
3352 RD_EXPORT
3353 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
3354 
3358 RD_EXPORT
3359 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
3360 
3361 
3368 RD_EXPORT
3369 rd_kafka_queue_t *rd_kafka_queue_get_main(rd_kafka_t *rk);
3370 
3371 
3372 
3382 RD_EXPORT
3383 rd_kafka_queue_t *rd_kafka_queue_get_sasl(rd_kafka_t *rk);
3384 
3385 
3401 RD_EXPORT
3402 rd_kafka_error_t *rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk);
3403 
3404 
3419 RD_EXPORT
3420 rd_kafka_error_t *rd_kafka_sasl_set_credentials(rd_kafka_t *rk,
3421  const char *username,
3422  const char *password);
3423 
3439 RD_EXPORT
3440 rd_kafka_queue_t *rd_kafka_queue_get_consumer(rd_kafka_t *rk);
3441 
3452 RD_EXPORT
3453 rd_kafka_queue_t *rd_kafka_queue_get_partition(rd_kafka_t *rk,
3454  const char *topic,
3455  int32_t partition);
3456 
3479 RD_EXPORT
3480 rd_kafka_queue_t *rd_kafka_queue_get_background(rd_kafka_t *rk);
3481 
3482 
3493 RD_EXPORT
3494 void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
3495 
3514 RD_EXPORT
3516  rd_kafka_queue_t *rkqu);
3517 
3518 
3522 RD_EXPORT
3523 size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu);
3524 
3525 
3543 RD_EXPORT
3544 void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu,
3545  int fd,
3546  const void *payload,
3547  size_t size);
3548 
3565 RD_EXPORT
3566 void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu,
3567  void (*event_cb)(rd_kafka_t *rk,
3568  void *qev_opaque),
3569  void *qev_opaque);
3570 
3571 
3579 RD_EXPORT
3580 void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu);
3581 
3582 
3593 #define RD_KAFKA_OFFSET_BEGINNING \
3594  -2
3596 #define RD_KAFKA_OFFSET_END \
3597  -1
3599 #define RD_KAFKA_OFFSET_STORED \
3600  -1000
3602 #define RD_KAFKA_OFFSET_INVALID -1001
3606 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
3614 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
3615 
3649 RD_EXPORT
3650 int rd_kafka_consume_start(rd_kafka_topic_t *rkt,
3651  int32_t partition,
3652  int64_t offset);
3653 
3668 RD_EXPORT
3669 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt,
3670  int32_t partition,
3671  int64_t offset,
3672  rd_kafka_queue_t *rkqu);
3673 
3687 RD_EXPORT
3688 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
3689 
3690 
3691 
3720 RD_EXPORT
3721 rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt,
3722  int32_t partition,
3723  int64_t offset,
3724  int timeout_ms);
3725 
3726 
3727 
3759 RD_EXPORT rd_kafka_error_t *
3761  rd_kafka_topic_partition_list_t *partitions,
3762  int timeout_ms);
3763 
3764 
3789 RD_EXPORT
3791 rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms);
3792 
3793 
3794 
3820 RD_EXPORT
3821 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt,
3822  int32_t partition,
3823  int timeout_ms,
3824  rd_kafka_message_t **rkmessages,
3825  size_t rkmessages_size);
3826 
3827 
3828 
3862 RD_EXPORT
3863 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt,
3864  int32_t partition,
3865  int timeout_ms,
3866  void (*consume_cb)(rd_kafka_message_t *rkmessage,
3867  void *commit_opaque),
3868  void *commit_opaque);
3869 
3870 
3889 RD_EXPORT
3891  int timeout_ms);
3892 
3898 RD_EXPORT
3899 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
3900  int timeout_ms,
3901  rd_kafka_message_t **rkmessages,
3902  size_t rkmessages_size);
3903 
3913 RD_EXPORT
3915  rd_kafka_queue_t *rkqu,
3916  int timeout_ms,
3917  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque),
3918  void *commit_opaque);
3919 
3920 
3960 RD_EXPORT
3962 rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset);
3963 
3964 
3997 RD_EXPORT rd_kafka_resp_err_t
3998 rd_kafka_offsets_store(rd_kafka_t *rk,
4000 
4001 
4022 RD_EXPORT
4024 
4078 RD_EXPORT rd_kafka_resp_err_t
4079 rd_kafka_subscribe(rd_kafka_t *rk,
4080  const rd_kafka_topic_partition_list_t *topics);
4081 
4082 
4086 RD_EXPORT
4088 
4089 
4099 RD_EXPORT rd_kafka_resp_err_t
4101 
4102 
4103 
4133 RD_EXPORT
4134 rd_kafka_message_t *rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms);
4135 
4153 RD_EXPORT
4155 
4156 
4174 RD_EXPORT
4175 rd_kafka_error_t *rd_kafka_consumer_close_queue(rd_kafka_t *rk,
4176  rd_kafka_queue_t *rkqu);
4177 
4178 
4187 RD_EXPORT
4188 int rd_kafka_consumer_closed(rd_kafka_t *rk);
4189 
4190 
4208 RD_EXPORT rd_kafka_error_t *
4210  const rd_kafka_topic_partition_list_t *partitions);
4211 
4212 
4230 RD_EXPORT rd_kafka_error_t *rd_kafka_incremental_unassign(
4231  rd_kafka_t *rk,
4232  const rd_kafka_topic_partition_list_t *partitions);
4233 
4234 
4248 RD_EXPORT
4249 const char *rd_kafka_rebalance_protocol(rd_kafka_t *rk);
4250 
4251 
4272 RD_EXPORT rd_kafka_resp_err_t
4273 rd_kafka_assign(rd_kafka_t *rk,
4274  const rd_kafka_topic_partition_list_t *partitions);
4275 
4292 RD_EXPORT rd_kafka_resp_err_t
4293 rd_kafka_assignment(rd_kafka_t *rk,
4294  rd_kafka_topic_partition_list_t **partitions);
4295 
4296 
4313 RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk);
4314 
4315 
4336 RD_EXPORT rd_kafka_resp_err_t
4337 rd_kafka_commit(rd_kafka_t *rk,
4338  const rd_kafka_topic_partition_list_t *offsets,
4339  int async);
4340 
4341 
4348 RD_EXPORT rd_kafka_resp_err_t
4350  const rd_kafka_message_t *rkmessage,
4351  int async);
4352 
4353 
4376 RD_EXPORT rd_kafka_resp_err_t
4377 rd_kafka_commit_queue(rd_kafka_t *rk,
4378  const rd_kafka_topic_partition_list_t *offsets,
4379  rd_kafka_queue_t *rkqu,
4380  void (*cb)(rd_kafka_t *rk,
4381  rd_kafka_resp_err_t err,
4383  void *commit_opaque),
4384  void *commit_opaque);
4385 
4386 
4404 RD_EXPORT rd_kafka_resp_err_t
4405 rd_kafka_committed(rd_kafka_t *rk,
4406  rd_kafka_topic_partition_list_t *partitions,
4407  int timeout_ms);
4408 
4409 
4410 
4427 RD_EXPORT rd_kafka_resp_err_t
4429 
4430 
4431 
4443 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4445 
4446 
4456 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4458 
4459 
4472 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4474  int32_t generation_id,
4475  const char *member_id,
4476  const char *group_instance_id);
4477 
4478 
4483 RD_EXPORT void
4484 rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *);
4485 
4486 
4507 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_write(
4508  const rd_kafka_consumer_group_metadata_t *cgmd,
4509  void **bufferp,
4510  size_t *sizep);
4511 
4532 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_read(
4533  rd_kafka_consumer_group_metadata_t **cgmdp,
4534  const void *buffer,
4535  size_t size);
4536 
4552 #define RD_KAFKA_MSG_F_FREE \
4553  0x1
4555 #define RD_KAFKA_MSG_F_COPY \
4556  0x2
4558 #define RD_KAFKA_MSG_F_BLOCK \
4559  0x4
4570 #define RD_KAFKA_MSG_F_PARTITION \
4571  0x8
4676 RD_EXPORT
4677 int rd_kafka_produce(rd_kafka_topic_t *rkt,
4678  int32_t partition,
4679  int msgflags,
4680  void *payload,
4681  size_t len,
4682  const void *key,
4683  size_t keylen,
4684  void *msg_opaque);
4685 
4686 
4700 RD_EXPORT
4702 
4703 
4715 RD_EXPORT
4716 rd_kafka_error_t *
4717 rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt);
4718 
4719 
4749 RD_EXPORT
4750 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt,
4751  int32_t partition,
4752  int msgflags,
4753  rd_kafka_message_t *rkmessages,
4754  int message_cnt);
4755 
4756 
4757 
4782 RD_EXPORT
4783 rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms);
4784 
4785 
4786 
4817 RD_EXPORT
4818 rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags);
4819 
4820 
4828 #define RD_KAFKA_PURGE_F_QUEUE 0x1
4829 
4837 #define RD_KAFKA_PURGE_F_INFLIGHT 0x2
4838 
4839 
4843 #define RD_KAFKA_PURGE_F_NON_BLOCKING 0x4
4844 
4845 
4860 typedef struct rd_kafka_metadata_broker {
4861  int32_t id;
4862  char *host;
4863  int port;
4865 
4869 typedef struct rd_kafka_metadata_partition {
4870  int32_t id;
4872  int32_t leader;
4874  int32_t *replicas;
4875  int isr_cnt;
4876  int32_t *isrs;
4878 
4882 typedef struct rd_kafka_metadata_topic {
4883  char *topic;
4885  struct rd_kafka_metadata_partition *partitions;
4888 
4889 
4893 typedef struct rd_kafka_metadata {
4895  struct rd_kafka_metadata_broker *brokers;
4898  struct rd_kafka_metadata_topic *topics;
4900  int32_t orig_broker_id;
4903 
4924 RD_EXPORT
4926 rd_kafka_metadata(rd_kafka_t *rk,
4927  int all_topics,
4928  rd_kafka_topic_t *only_rkt,
4929  const struct rd_kafka_metadata **metadatap,
4930  int timeout_ms);
4931 
4935 RD_EXPORT
4936 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
4937 
4941 typedef struct rd_kafka_Node_s rd_kafka_Node_t;
4942 
4950 RD_EXPORT
4952 
4963 RD_EXPORT
4964 const char *rd_kafka_Node_host(const rd_kafka_Node_t *node);
4965 
4973 RD_EXPORT
4974 uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node);
4975 
4996  char *member_id;
4997  char *client_id;
4998  char *client_host;
5005 };
5006 
5012 typedef enum {
5013  RD_KAFKA_CONSUMER_GROUP_STATE_UNKNOWN = 0,
5014  RD_KAFKA_CONSUMER_GROUP_STATE_PREPARING_REBALANCE = 1,
5015  RD_KAFKA_CONSUMER_GROUP_STATE_COMPLETING_REBALANCE = 2,
5016  RD_KAFKA_CONSUMER_GROUP_STATE_STABLE = 3,
5017  RD_KAFKA_CONSUMER_GROUP_STATE_DEAD = 4,
5018  RD_KAFKA_CONSUMER_GROUP_STATE_EMPTY = 5,
5019  RD_KAFKA_CONSUMER_GROUP_STATE__CNT
5021 
5026  struct rd_kafka_metadata_broker broker;
5027  char *group;
5029  char *state;
5031  char *protocol;
5034 };
5035 
5044 };
5045 
5046 
5077 RD_EXPORT
5079 rd_kafka_list_groups(rd_kafka_t *rk,
5080  const char *group,
5081  const struct rd_kafka_group_list **grplistp,
5082  int timeout_ms);
5083 
5091 RD_EXPORT
5092 const char *
5094 
5102 RD_EXPORT
5105 
5109 RD_EXPORT
5111 
5112 
5156 RD_EXPORT
5157 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
5158 
5159 
5160 
5173 RD_EXPORT RD_DEPRECATED void
5174 rd_kafka_set_logger(rd_kafka_t *rk,
5175  void (*func)(const rd_kafka_t *rk,
5176  int level,
5177  const char *fac,
5178  const char *buf));
5179 
5180 
5190 RD_EXPORT
5191 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
5192 
5193 
5197 RD_EXPORT
5198 void rd_kafka_log_print(const rd_kafka_t *rk,
5199  int level,
5200  const char *fac,
5201  const char *buf);
5202 
5203 
5209 RD_EXPORT
5210 void rd_kafka_log_syslog(const rd_kafka_t *rk,
5211  int level,
5212  const char *fac,
5213  const char *buf);
5214 
5215 
5238 RD_EXPORT
5239 int rd_kafka_outq_len(rd_kafka_t *rk);
5240 
5241 
5242 
5249 RD_EXPORT
5250 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
5251 
5252 
5253 
5259 RD_EXPORT
5261 
5262 
5275 
5276 
5285 RD_EXPORT
5286 int rd_kafka_wait_destroyed(int timeout_ms);
5287 
5288 
5294 RD_EXPORT
5296 
5297 
5314 RD_EXPORT
5316 
5317 
5334 #define RD_KAFKA_EVENT_NONE 0x0
5335 #define RD_KAFKA_EVENT_DR 0x1
5336 #define RD_KAFKA_EVENT_FETCH 0x2
5337 #define RD_KAFKA_EVENT_LOG 0x4
5338 #define RD_KAFKA_EVENT_ERROR 0x8
5339 #define RD_KAFKA_EVENT_REBALANCE 0x10
5340 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
5341 #define RD_KAFKA_EVENT_STATS 0x40
5342 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
5343 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
5344 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT \
5345  102
5346 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
5347 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT \
5348  104
5349 #define RD_KAFKA_EVENT_DELETERECORDS_RESULT 105
5350 #define RD_KAFKA_EVENT_DELETEGROUPS_RESULT 106
5352 #define RD_KAFKA_EVENT_DELETECONSUMERGROUPOFFSETS_RESULT 107
5354 #define RD_KAFKA_EVENT_OAUTHBEARER_TOKEN_REFRESH 0x100
5355 #define RD_KAFKA_EVENT_BACKGROUND 0x200
5356 #define RD_KAFKA_EVENT_CREATEACLS_RESULT 0x400
5357 #define RD_KAFKA_EVENT_DESCRIBEACLS_RESULT 0x800
5358 #define RD_KAFKA_EVENT_DELETEACLS_RESULT 0x1000
5360 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPS_RESULT 0x2000
5362 #define RD_KAFKA_EVENT_DESCRIBECONSUMERGROUPS_RESULT 0x4000
5364 #define RD_KAFKA_EVENT_LISTCONSUMERGROUPOFFSETS_RESULT 0x8000
5366 #define RD_KAFKA_EVENT_ALTERCONSUMERGROUPOFFSETS_RESULT 0x10000
5368 #define RD_KAFKA_EVENT_INCREMENTALALTERCONFIGS_RESULT 0x20000
5370 #define RD_KAFKA_EVENT_DESCRIBEUSERSCRAMCREDENTIALS_RESULT 0x40000
5372 #define RD_KAFKA_EVENT_ALTERUSERSCRAMCREDENTIALS_RESULT 0x80000
5373 
5380 RD_EXPORT
5381 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev);
5382 
5389 RD_EXPORT
5390 const char *rd_kafka_event_name(const rd_kafka_event_t *rkev);
5391 
5392 
5402 RD_EXPORT
5403 void rd_kafka_event_destroy(rd_kafka_event_t *rkev);
5404 
5405 
5421 RD_EXPORT
5422 const rd_kafka_message_t *rd_kafka_event_message_next(rd_kafka_event_t *rkev);
5423 
5424 
5438 RD_EXPORT
5439 size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev,
5440  const rd_kafka_message_t **rkmessages,
5441  size_t size);
5442 
5443 
5451 RD_EXPORT
5452 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev);
5453 
5454 
5466 RD_EXPORT
5467 const char *rd_kafka_event_config_string(rd_kafka_event_t *rkev);
5468 
5469 
5478 RD_EXPORT
5480 
5481 
5490 RD_EXPORT
5491 const char *rd_kafka_event_error_string(rd_kafka_event_t *rkev);
5492 
5493 
5502 RD_EXPORT
5503 int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev);
5504 
5505 
5529 RD_EXPORT
5530 void *rd_kafka_event_opaque(rd_kafka_event_t *rkev);
5531 
5532 
5541 RD_EXPORT
5542 int rd_kafka_event_log(rd_kafka_event_t *rkev,
5543  const char **fac,
5544  const char **str,
5545  int *level);
5546 
5547 
5559 RD_EXPORT
5560 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev,
5561  char *dst,
5562  size_t dstsize);
5563 
5564 
5577 RD_EXPORT
5578 const char *rd_kafka_event_stats(rd_kafka_event_t *rkev);
5579 
5580 
5592 rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev);
5593 
5594 
5605 RD_EXPORT rd_kafka_topic_partition_t *
5606 rd_kafka_event_topic_partition(rd_kafka_event_t *rkev);
5607 
5608 
5610 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
5612 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
5614 typedef rd_kafka_event_t rd_kafka_CreateAcls_result_t;
5616 typedef rd_kafka_event_t rd_kafka_DescribeAcls_result_t;
5618 typedef rd_kafka_event_t rd_kafka_DeleteAcls_result_t;
5620 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
5622 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
5626 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
5628 typedef rd_kafka_event_t rd_kafka_DeleteRecords_result_t;
5630 typedef rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t;
5634 typedef rd_kafka_event_t rd_kafka_DeleteGroups_result_t;
5645 
5655 RD_EXPORT const rd_kafka_CreateTopics_result_t *
5656 rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev);
5657 
5667 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
5668 rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev);
5669 
5679 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
5681 
5691 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
5692 rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev);
5693 
5705 
5715 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
5717 
5725 RD_EXPORT const rd_kafka_DeleteRecords_result_t *
5726 rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev);
5727 
5740 RD_EXPORT const rd_kafka_ListConsumerGroups_result_t *
5742 
5757 
5767 RD_EXPORT const rd_kafka_DeleteGroups_result_t *
5768 rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev);
5769 
5781 
5789 RD_EXPORT const rd_kafka_CreateAcls_result_t *
5790 rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev);
5791 
5799 RD_EXPORT const rd_kafka_DescribeAcls_result_t *
5800 rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev);
5801 
5809 RD_EXPORT const rd_kafka_DeleteAcls_result_t *
5810 rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev);
5811 
5826 
5841 
5856 
5871 
5881 RD_EXPORT
5882 rd_kafka_event_t *rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms);
5883 
5898 RD_EXPORT
5899 int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms);
5900 
5901 
5946  rd_kafka_conf_t *conf,
5947  void **plug_opaquep,
5948  char *errstr,
5949  size_t errstr_size);
5950 
6032  rd_kafka_conf_t *conf,
6033  const char *name,
6034  const char *val,
6035  char *errstr,
6036  size_t errstr_size,
6037  void *ic_opaque);
6038 
6039 
6062  rd_kafka_conf_t *new_conf,
6063  const rd_kafka_conf_t *old_conf,
6064  size_t filter_cnt,
6065  const char **filter,
6066  void *ic_opaque);
6067 
6068 
6076  void *ic_opaque);
6077 
6078 
6097  rd_kafka_t *rk,
6098  const rd_kafka_conf_t *conf,
6099  void *ic_opaque,
6100  char *errstr,
6101  size_t errstr_size);
6102 
6103 
6112  rd_kafka_interceptor_f_on_destroy_t)(rd_kafka_t *rk, void *ic_opaque);
6113 
6114 
6115 
6137  rd_kafka_t *rk,
6138  rd_kafka_message_t *rkmessage,
6139  void *ic_opaque);
6140 
6164  rd_kafka_t *rk,
6165  rd_kafka_message_t *rkmessage,
6166  void *ic_opaque);
6167 
6168 
6186  rd_kafka_t *rk,
6187  rd_kafka_message_t *rkmessage,
6188  void *ic_opaque);
6189 
6212  rd_kafka_t *rk,
6213  const rd_kafka_topic_partition_list_t *offsets,
6215  void *ic_opaque);
6216 
6217 
6240  rd_kafka_t *rk,
6241  int sockfd,
6242  const char *brokername,
6243  int32_t brokerid,
6244  int16_t ApiKey,
6245  int16_t ApiVersion,
6246  int32_t CorrId,
6247  size_t size,
6248  void *ic_opaque);
6249 
6250 
6277  rd_kafka_t *rk,
6278  int sockfd,
6279  const char *brokername,
6280  int32_t brokerid,
6281  int16_t ApiKey,
6282  int16_t ApiVersion,
6283  int32_t CorrId,
6284  size_t size,
6285  int64_t rtt,
6287  void *ic_opaque);
6288 
6289 
6307  rd_kafka_t *rk,
6308  rd_kafka_thread_type_t thread_type,
6309  const char *thread_name,
6310  void *ic_opaque);
6311 
6312 
6333  rd_kafka_t *rk,
6334  rd_kafka_thread_type_t thread_type,
6335  const char *thread_name,
6336  void *ic_opaque);
6337 
6338 
6354  rd_kafka_t *rk,
6355  int32_t broker_id,
6356  const char *secproto,
6357  const char *name,
6358  int port,
6359  const char *state,
6360  void *ic_opaque);
6361 
6362 
6376  rd_kafka_conf_t *conf,
6377  const char *ic_name,
6379  void *ic_opaque);
6380 
6381 
6395  rd_kafka_conf_t *conf,
6396  const char *ic_name,
6398  void *ic_opaque);
6399 
6414  rd_kafka_conf_t *conf,
6415  const char *ic_name,
6417  void *ic_opaque);
6418 
6419 
6441 RD_EXPORT rd_kafka_resp_err_t
6443  const char *ic_name,
6445  void *ic_opaque);
6446 
6447 
6448 
6462  rd_kafka_t *rk,
6463  const char *ic_name,
6465  void *ic_opaque);
6466 
6467 
6480 RD_EXPORT rd_kafka_resp_err_t
6482  const char *ic_name,
6484  void *ic_opaque);
6485 
6499  rd_kafka_t *rk,
6500  const char *ic_name,
6501  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
6502  void *ic_opaque);
6503 
6504 
6518  rd_kafka_t *rk,
6519  const char *ic_name,
6521  void *ic_opaque);
6522 
6523 
6537  rd_kafka_t *rk,
6538  const char *ic_name,
6540  void *ic_opaque);
6541 
6542 
6556  rd_kafka_t *rk,
6557  const char *ic_name,
6559  void *ic_opaque);
6560 
6561 
6575  rd_kafka_t *rk,
6576  const char *ic_name,
6577  rd_kafka_interceptor_f_on_response_received_t *on_response_received,
6578  void *ic_opaque);
6579 
6580 
6594  rd_kafka_t *rk,
6595  const char *ic_name,
6597  void *ic_opaque);
6598 
6599 
6613  rd_kafka_t *rk,
6614  const char *ic_name,
6616  void *ic_opaque);
6617 
6618 
6631 RD_EXPORT
6633  rd_kafka_t *rk,
6634  const char *ic_name,
6635  rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change,
6636  void *ic_opaque);
6637 
6638 
6639 
6660 RD_EXPORT rd_kafka_resp_err_t
6661 rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres);
6662 
6669 RD_EXPORT const char *
6670 rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres);
6671 
6677 RD_EXPORT const char *
6678 rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres);
6679 
6689 RD_EXPORT const rd_kafka_error_t *
6690 rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres);
6691 
6697 RD_EXPORT const char *
6698 rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres);
6699 
6705 RD_EXPORT const rd_kafka_topic_partition_list_t *
6706 rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres);
6707 
6708 
6755 typedef enum rd_kafka_admin_op_t {
6783 
6796 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
6797 
6817 RD_EXPORT rd_kafka_AdminOptions_t *
6819 
6820 
6825 
6826 
6844 RD_EXPORT rd_kafka_resp_err_t
6846  int timeout_ms,
6847  char *errstr,
6848  size_t errstr_size);
6849 
6850 
6877 RD_EXPORT rd_kafka_resp_err_t
6879  int timeout_ms,
6880  char *errstr,
6881  size_t errstr_size);
6882 
6883 
6902 RD_EXPORT rd_kafka_resp_err_t
6904  int true_or_false,
6905  char *errstr,
6906  size_t errstr_size);
6907 
6908 
6936 RD_EXPORT rd_kafka_resp_err_t
6938  int32_t broker_id,
6939  char *errstr,
6940  size_t errstr_size);
6941 
6942 
6955 RD_EXPORT
6957  rd_kafka_AdminOptions_t *options,
6958  int true_or_false);
6959 
6972 RD_EXPORT
6974  rd_kafka_AdminOptions_t *options,
6975  const rd_kafka_consumer_group_state_t *consumer_group_states,
6976  size_t consumer_group_states_cnt);
6977 
6982 RD_EXPORT void
6984  void *ev_opaque);
6985 
6997 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
6998 
7020 RD_EXPORT rd_kafka_NewTopic_t *rd_kafka_NewTopic_new(const char *topic,
7021  int num_partitions,
7022  int replication_factor,
7023  char *errstr,
7024  size_t errstr_size);
7025 
7031 
7032 
7039  size_t new_topic_cnt);
7040 
7041 
7063 RD_EXPORT rd_kafka_resp_err_t
7065  int32_t partition,
7066  int32_t *broker_ids,
7067  size_t broker_id_cnt,
7068  char *errstr,
7069  size_t errstr_size);
7070 
7083 RD_EXPORT rd_kafka_resp_err_t
7085  const char *name,
7086  const char *value);
7087 
7088 
7107 RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk,
7108  rd_kafka_NewTopic_t **new_topics,
7109  size_t new_topic_cnt,
7110  const rd_kafka_AdminOptions_t *options,
7111  rd_kafka_queue_t *rkqu);
7112 
7113 
7114 /*
7115  * CreateTopics result type and methods
7116  */
7117 
7126 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_CreateTopics_result_topics(
7127  const rd_kafka_CreateTopics_result_t *result,
7128  size_t *cntp);
7129 
7130 
7131 
7132 /*
7133  * DeleteTopics - delete topics from cluster
7134  *
7135  */
7136 
7138 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
7139 
7149 RD_EXPORT rd_kafka_DeleteTopic_t *rd_kafka_DeleteTopic_new(const char *topic);
7150 
7156 
7162 RD_EXPORT void
7164  size_t del_topic_cnt);
7165 
7179 RD_EXPORT
7180 void rd_kafka_DeleteTopics(rd_kafka_t *rk,
7181  rd_kafka_DeleteTopic_t **del_topics,
7182  size_t del_topic_cnt,
7183  const rd_kafka_AdminOptions_t *options,
7184  rd_kafka_queue_t *rkqu);
7185 
7186 
7187 
7188 /*
7189  * DeleteTopics result type and methods
7190  */
7191 
7200 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_DeleteTopics_result_topics(
7201  const rd_kafka_DeleteTopics_result_t *result,
7202  size_t *cntp);
7203 
7204 
7215 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
7216 
7233 RD_EXPORT rd_kafka_NewPartitions_t *
7234 rd_kafka_NewPartitions_new(const char *topic,
7235  size_t new_total_cnt,
7236  char *errstr,
7237  size_t errstr_size);
7238 
7243 RD_EXPORT void
7245 
7251 RD_EXPORT void
7253  size_t new_parts_cnt);
7254 
7278  rd_kafka_NewPartitions_t *new_parts,
7279  int32_t new_partition_idx,
7280  int32_t *broker_ids,
7281  size_t broker_id_cnt,
7282  char *errstr,
7283  size_t errstr_size);
7284 
7285 
7304 RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk,
7305  rd_kafka_NewPartitions_t **new_parts,
7306  size_t new_parts_cnt,
7307  const rd_kafka_AdminOptions_t *options,
7308  rd_kafka_queue_t *rkqu);
7309 
7310 
7311 
7312 /*
7313  * CreatePartitions result type and methods
7314  */
7315 
7324 RD_EXPORT const rd_kafka_topic_result_t **
7326  const rd_kafka_CreatePartitions_result_t *result,
7327  size_t *cntp);
7328 
7364 
7368 
7369 
7373 RD_EXPORT const char *
7375 
7376 
7378 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
7379 
7383 RD_EXPORT const char *
7385 
7390 RD_EXPORT const char *
7392 
7396 RD_EXPORT rd_kafka_ConfigSource_t
7398 
7403 RD_EXPORT int
7405 
7411 RD_EXPORT int
7413 
7421 RD_EXPORT int
7423 
7427 RD_EXPORT int
7429 
7430 
7441 RD_EXPORT const rd_kafka_ConfigEntry_t **
7443  size_t *cntp);
7444 
7445 
7446 
7459 
7475  RD_KAFKA_RESOURCE_PATTERN_TYPE__CNT,
7477 
7483  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SET = 0,
7484  RD_KAFKA_ALTER_CONFIG_OP_TYPE_DELETE = 1,
7485  RD_KAFKA_ALTER_CONFIG_OP_TYPE_APPEND = 2,
7486  RD_KAFKA_ALTER_CONFIG_OP_TYPE_SUBTRACT = 3,
7487  RD_KAFKA_ALTER_CONFIG_OP_TYPE__CNT,
7489 
7494  rd_kafka_ResourcePatternType_t resource_pattern_type);
7495 
7499 RD_EXPORT const char *
7501 
7503 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
7504 
7505 
7514 RD_EXPORT rd_kafka_ConfigResource_t *
7516  const char *resname);
7517 
7522 RD_EXPORT void
7524 
7525 
7531 RD_EXPORT void
7533  size_t config_cnt);
7534 
7535 
7549 RD_EXPORT rd_kafka_resp_err_t
7551  const char *name,
7552  const char *value);
7553 
7554 
7574  rd_kafka_ConfigResource_t *config,
7575  const char *name,
7577  const char *value);
7578 
7579 
7588 RD_EXPORT const rd_kafka_ConfigEntry_t **
7590  size_t *cntp);
7591 
7592 
7593 
7597 RD_EXPORT rd_kafka_ResourceType_t
7599 
7603 RD_EXPORT const char *
7605 
7609 RD_EXPORT rd_kafka_resp_err_t
7611 
7616 RD_EXPORT const char *
7618 
7619 
7620 /*
7621  * AlterConfigs - alter cluster configuration.
7622  *
7623  */
7624 
7625 
7648 RD_EXPORT
7649 void rd_kafka_AlterConfigs(rd_kafka_t *rk,
7650  rd_kafka_ConfigResource_t **configs,
7651  size_t config_cnt,
7652  const rd_kafka_AdminOptions_t *options,
7653  rd_kafka_queue_t *rkqu);
7654 
7655 
7656 /*
7657  * AlterConfigs result type and methods
7658  */
7659 
7674 RD_EXPORT const rd_kafka_ConfigResource_t **
7676  const rd_kafka_AlterConfigs_result_t *result,
7677  size_t *cntp);
7678 
7679 
7680 
7681 /*
7682  * IncrementalAlterConfigs - alter cluster configuration incrementally.
7683  *
7684  */
7685 
7686 
7707 RD_EXPORT
7709  rd_kafka_ConfigResource_t **configs,
7710  size_t config_cnt,
7711  const rd_kafka_AdminOptions_t *options,
7712  rd_kafka_queue_t *rkqu);
7713 
7714 
7715 /*
7716  * IncrementalAlterConfigs result type and methods
7717  */
7718 
7734 RD_EXPORT const rd_kafka_ConfigResource_t **
7737  size_t *cntp);
7738 
7739 
7740 
7741 /*
7742  * DescribeConfigs - retrieve cluster configuration.
7743  *
7744  */
7745 
7746 
7771 RD_EXPORT
7772 void rd_kafka_DescribeConfigs(rd_kafka_t *rk,
7773  rd_kafka_ConfigResource_t **configs,
7774  size_t config_cnt,
7775  const rd_kafka_AdminOptions_t *options,
7776  rd_kafka_queue_t *rkqu);
7777 
7778 
7779 
7780 /*
7781  * DescribeConfigs result type and methods
7782  */
7783 
7792 RD_EXPORT const rd_kafka_ConfigResource_t **
7794  const rd_kafka_DescribeConfigs_result_t *result,
7795  size_t *cntp);
7796 
7797 
7808 typedef struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t;
7809 
7827  const rd_kafka_topic_partition_list_t *before_offsets);
7828 
7833 RD_EXPORT void
7835 
7841 RD_EXPORT void
7843  size_t del_record_cnt);
7844 
7866 RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk,
7867  rd_kafka_DeleteRecords_t **del_records,
7868  size_t del_record_cnt,
7869  const rd_kafka_AdminOptions_t *options,
7870  rd_kafka_queue_t *rkqu);
7871 
7872 
7873 /*
7874  * DeleteRecords result type and methods
7875  */
7876 
7886 RD_EXPORT const rd_kafka_topic_partition_list_t *
7888  const rd_kafka_DeleteRecords_result_t *result);
7889 
7903 typedef struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t;
7904 
7906 typedef struct rd_kafka_ListConsumerGroupsResult_s
7908 
7919 RD_EXPORT
7920 void rd_kafka_ListConsumerGroups(rd_kafka_t *rk,
7921  const rd_kafka_AdminOptions_t *options,
7922  rd_kafka_queue_t *rkqu);
7923 
7934 RD_EXPORT
7936  const rd_kafka_ConsumerGroupListing_t *grplist);
7937 
7946 RD_EXPORT
7948  const rd_kafka_ConsumerGroupListing_t *grplist);
7949 
7957 RD_EXPORT
7959  const rd_kafka_ConsumerGroupListing_t *grplist);
7960 
7972 RD_EXPORT
7976  size_t *cntp);
7977 
7991 RD_EXPORT
7994  size_t *cntp);
7995 
8007 typedef struct rd_kafka_ConsumerGroupDescription_s
8009 
8014 typedef struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t;
8015 
8020 typedef struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t;
8021 
8035 RD_EXPORT
8037  const char **groups,
8038  size_t groups_cnt,
8039  const rd_kafka_AdminOptions_t *options,
8040  rd_kafka_queue_t *rkqu);
8041 
8053 RD_EXPORT
8057  size_t *cntp);
8058 
8059 
8070 RD_EXPORT
8072  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8073 
8084 RD_EXPORT
8086  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8087 
8095 RD_EXPORT
8097  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8098 
8099 
8110 RD_EXPORT
8112  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8113 
8114 
8122 RD_EXPORT
8124  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8125 
8136 RD_EXPORT
8138  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8139 
8147 RD_EXPORT
8149  const rd_kafka_ConsumerGroupDescription_t *grpdesc);
8150 
8163 RD_EXPORT
8165  const rd_kafka_ConsumerGroupDescription_t *grpdesc,
8166  size_t idx);
8167 
8178 RD_EXPORT
8180  const rd_kafka_MemberDescription_t *member);
8181 
8192 RD_EXPORT
8194  const rd_kafka_MemberDescription_t *member);
8195 
8206 RD_EXPORT
8208  const rd_kafka_MemberDescription_t *member);
8209 
8220 RD_EXPORT
8221 const char *
8223 
8234 RD_EXPORT
8236  const rd_kafka_MemberDescription_t *member);
8237 
8248 RD_EXPORT
8250  const rd_kafka_MemberAssignment_t *assignment);
8251 
8263 typedef struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t;
8264 
8274 RD_EXPORT
8276 
8281 RD_EXPORT
8283 
8289 RD_EXPORT void
8291  size_t del_group_cnt);
8292 
8308 RD_EXPORT
8309 void rd_kafka_DeleteGroups(rd_kafka_t *rk,
8310  rd_kafka_DeleteGroup_t **del_groups,
8311  size_t del_group_cnt,
8312  const rd_kafka_AdminOptions_t *options,
8313  rd_kafka_queue_t *rkqu);
8314 
8315 
8316 
8317 /*
8318  * DeleteGroups result type and methods
8319  */
8320 
8329 RD_EXPORT const rd_kafka_group_result_t **rd_kafka_DeleteGroups_result_groups(
8330  const rd_kafka_DeleteGroups_result_t *result,
8331  size_t *cntp);
8332 
8343 typedef struct rd_kafka_ListConsumerGroupOffsets_s
8345 
8360  const char *group_id,
8361  const rd_kafka_topic_partition_list_t *partitions);
8362 
8368  rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets);
8369 
8376  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8377  size_t list_grpoffset_cnt);
8378 
8396 RD_EXPORT
8398  rd_kafka_t *rk,
8399  rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets,
8400  size_t list_grpoffsets_cnt,
8401  const rd_kafka_AdminOptions_t *options,
8402  rd_kafka_queue_t *rkqu);
8403 
8404 
8405 
8406 /*
8407  * ListConsumerGroupOffsets result type and methods
8408  */
8409 
8421 RD_EXPORT const rd_kafka_group_result_t **
8424  size_t *cntp);
8425 
8426 
8427 
8438 typedef struct rd_kafka_AlterConsumerGroupOffsets_s
8440 
8455  const char *group_id,
8456  const rd_kafka_topic_partition_list_t *partitions);
8457 
8463  rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets);
8464 
8471  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
8472  size_t alter_grpoffset_cnt);
8473 
8492 RD_EXPORT
8494  rd_kafka_t *rk,
8495  rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets,
8496  size_t alter_grpoffsets_cnt,
8497  const rd_kafka_AdminOptions_t *options,
8498  rd_kafka_queue_t *rkqu);
8499 
8500 
8501 
8502 /*
8503  * AlterConsumerGroupOffsets result type and methods
8504  */
8505 
8517 RD_EXPORT const rd_kafka_group_result_t **
8520  size_t *cntp);
8521 
8522 
8523 
8534 typedef struct rd_kafka_DeleteConsumerGroupOffsets_s
8536 
8551  const char *group,
8552  const rd_kafka_topic_partition_list_t *partitions);
8553 
8559  rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets);
8560 
8567  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
8568  size_t del_grpoffset_cnt);
8569 
8588 RD_EXPORT
8590  rd_kafka_t *rk,
8591  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
8592  size_t del_grpoffsets_cnt,
8593  const rd_kafka_AdminOptions_t *options,
8594  rd_kafka_queue_t *rkqu);
8595 
8596 
8597 
8598 /*
8599  * DeleteConsumerGroupOffsets result type and methods
8600  */
8601 
8610 RD_EXPORT const rd_kafka_group_result_t **
8613  size_t *cntp);
8614 
8628  RD_KAFKA_SCRAM_MECHANISM_UNKNOWN = 0,
8629  RD_KAFKA_SCRAM_MECHANISM_SHA_256 = 1,
8630  RD_KAFKA_SCRAM_MECHANISM_SHA_512 = 2,
8631  RD_KAFKA_SCRAM_MECHANISM__CNT
8633 
8639 typedef struct rd_kafka_ScramCredentialInfo_s rd_kafka_ScramCredentialInfo_t;
8640 
8644 RD_EXPORT
8646  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
8647 
8651 RD_EXPORT
8653  const rd_kafka_ScramCredentialInfo_t *scram_credential_info);
8654 
8661 typedef struct rd_kafka_UserScramCredentialsDescription_s
8663 
8667 RD_EXPORT
8669  const rd_kafka_UserScramCredentialsDescription_t *description);
8670 
8674 RD_EXPORT
8676  const rd_kafka_UserScramCredentialsDescription_t *description);
8677 
8682 RD_EXPORT
8684  const rd_kafka_UserScramCredentialsDescription_t *description);
8685 
8690 RD_EXPORT
8693  const rd_kafka_UserScramCredentialsDescription_t *description,
8694  size_t idx);
8695 
8705 RD_EXPORT
8709  size_t *cntp);
8710 
8722 RD_EXPORT
8724  rd_kafka_t *rk,
8725  const char **users,
8726  size_t user_cnt,
8727  const rd_kafka_AdminOptions_t *options,
8728  rd_kafka_queue_t *rkqu);
8729 
8733 typedef struct rd_kafka_UserScramCredentialAlteration_s
8735 
8756 RD_EXPORT
8759  rd_kafka_ScramMechanism_t mechanism,
8760  int32_t iterations,
8761  const unsigned char *password,
8762  size_t password_size,
8763  const unsigned char *salt,
8764  size_t salt_size);
8765 
8775 RD_EXPORT
8778  rd_kafka_ScramMechanism_t mechanism);
8779 
8780 
8784 RD_EXPORT
8787 
8791 RD_EXPORT
8794  size_t alteration_cnt);
8795 
8799 typedef struct rd_kafka_AlterUserScramCredentials_result_response_s
8801 
8806 RD_EXPORT
8809 
8814 RD_EXPORT
8815 const rd_kafka_error_t *
8818 
8827 RD_EXPORT
8831  size_t *cntp);
8832 
8846 RD_EXPORT
8848  rd_kafka_t *rk,
8850  size_t alteration_cnt,
8851  const rd_kafka_AdminOptions_t *options,
8852  rd_kafka_queue_t *rkqu);
8853 
8866 typedef struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t;
8867 
8873 
8877 RD_EXPORT const rd_kafka_error_t *
8878 rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres);
8879 
8880 
8888  1,
8897  9,
8899  10,
8901  11,
8903  12,
8904  RD_KAFKA_ACL_OPERATION__CNT
8906 
8910 RD_EXPORT const char *
8912 
8920  1,
8923  RD_KAFKA_ACL_PERMISSION_TYPE__CNT
8925 
8929 RD_EXPORT const char *rd_kafka_AclPermissionType_name(
8930  rd_kafka_AclPermissionType_t acl_permission_type);
8931 
8950 RD_EXPORT rd_kafka_AclBinding_t *
8952  const char *name,
8953  rd_kafka_ResourcePatternType_t resource_pattern_type,
8954  const char *principal,
8955  const char *host,
8956  rd_kafka_AclOperation_t operation,
8957  rd_kafka_AclPermissionType_t permission_type,
8958  char *errstr,
8959  size_t errstr_size);
8960 
8987  rd_kafka_ResourceType_t restype,
8988  const char *name,
8989  rd_kafka_ResourcePatternType_t resource_pattern_type,
8990  const char *principal,
8991  const char *host,
8992  rd_kafka_AclOperation_t operation,
8993  rd_kafka_AclPermissionType_t permission_type,
8994  char *errstr,
8995  size_t errstr_size);
8996 
9000 RD_EXPORT rd_kafka_ResourceType_t
9002 
9008 RD_EXPORT const char *
9010 
9016 RD_EXPORT const char *
9018 
9024 RD_EXPORT const char *
9026 
9030 RD_EXPORT rd_kafka_AclOperation_t
9032 
9038 
9044 
9048 RD_EXPORT const rd_kafka_error_t *
9050 
9051 
9057 
9058 
9064 RD_EXPORT void
9066  size_t acl_bindings_cnt);
9067 
9075 RD_EXPORT const rd_kafka_acl_result_t **
9077  size_t *cntp);
9078 
9095 RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk,
9096  rd_kafka_AclBinding_t **new_acls,
9097  size_t new_acls_cnt,
9098  const rd_kafka_AdminOptions_t *options,
9099  rd_kafka_queue_t *rkqu);
9100 
9114 RD_EXPORT const rd_kafka_AclBinding_t **
9116  size_t *cntp);
9117 
9132 RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk,
9133  rd_kafka_AclBindingFilter_t *acl_filter,
9134  const rd_kafka_AdminOptions_t *options,
9135  rd_kafka_queue_t *rkqu);
9136 
9143 typedef struct rd_kafka_DeleteAcls_result_response_s
9145 
9153 RD_EXPORT const rd_kafka_DeleteAcls_result_response_t **
9155  size_t *cntp);
9156 
9161 RD_EXPORT const rd_kafka_error_t *rd_kafka_DeleteAcls_result_response_error(
9162  const rd_kafka_DeleteAcls_result_response_t *result_response);
9163 
9164 
9171 RD_EXPORT const rd_kafka_AclBinding_t **
9173  const rd_kafka_DeleteAcls_result_response_t *result_response,
9174  size_t *matching_acls_cntp);
9175 
9192 RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk,
9193  rd_kafka_AclBindingFilter_t **del_acls,
9194  size_t del_acls_cnt,
9195  const rd_kafka_AdminOptions_t *options,
9196  rd_kafka_queue_t *rkqu);
9197 
9250 RD_EXPORT
9253  const char *token_value,
9254  int64_t md_lifetime_ms,
9255  const char *md_principal_name,
9256  const char **extensions,
9257  size_t extension_size,
9258  char *errstr,
9259  size_t errstr_size);
9260 
9281 RD_EXPORT
9283  const char *errstr);
9284 
9471 RD_EXPORT
9472 rd_kafka_error_t *rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms);
9473 
9474 
9475 
9520 RD_EXPORT
9521 rd_kafka_error_t *rd_kafka_begin_transaction(rd_kafka_t *rk);
9522 
9523 
9595 RD_EXPORT
9597  rd_kafka_t *rk,
9598  const rd_kafka_topic_partition_list_t *offsets,
9599  const rd_kafka_consumer_group_metadata_t *cgmetadata,
9600  int timeout_ms);
9601 
9602 
9670 RD_EXPORT
9671 rd_kafka_error_t *rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms);
9672 
9673 
9732 RD_EXPORT
9733 rd_kafka_error_t *rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms);
9734 
9735 
9738 /* @cond NO_DOC */
9739 #ifdef __cplusplus
9740 }
9741 #endif
9742 #endif /* _RDKAFKA_H_ */
9743 /* @endcond NO_DOC */
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_new_t *on_new, void *ic_opaque)
Append an on_new() interceptor.
RD_EXPORT const char * rd_kafka_AlterUserScramCredentials_result_response_user(const rd_kafka_AlterUserScramCredentials_result_response_t *response)
Returns the username for a rd_kafka_AlterUserScramCredentials_result_response.
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Murmur2 partitioner (Java compatible).
struct rd_kafka_Node_s rd_kafka_Node_t
Node (broker) information.
Definition: rdkafka.h:4941
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_DeleteConsumerGroupOffsets_result_groups(const rd_kafka_DeleteConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a DeleteConsumerGroupOffsets result.
RD_EXPORT void * rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt)
Get the rkt_opaque pointer that was set in the topic configuration with rd_kafka_topic_conf_set_opaqu...
RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk, rd_kafka_AclBinding_t **new_acls, size_t new_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create acls as specified by the new_acls array of size new_topic_cnt elements.
rd_kafka_resp_err_t
Error codes.
Definition: rdkafka.h:280
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_EXPIRED
Definition: rdkafka.h:566
@ RD_KAFKA_RESP_ERR_POLICY_VIOLATION
Definition: rdkafka.h:516
@ RD_KAFKA_RESP_ERR_TRANSACTION_COORDINATOR_FENCED
Definition: rdkafka.h:537
@ RD_KAFKA_RESP_ERR__AUTO_OFFSET_RESET
Definition: rdkafka.h:406
@ RD_KAFKA_RESP_ERR_RESOURCE_NOT_FOUND
Definition: rdkafka.h:619
@ RD_KAFKA_RESP_ERR__PURGE_QUEUE
Definition: rdkafka.h:382
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_FOR_MESSAGE_FORMAT
Definition: rdkafka.h:514
@ RD_KAFKA_RESP_ERR_OPERATION_NOT_ATTEMPTED
Definition: rdkafka.h:543
@ RD_KAFKA_RESP_ERR_INVALID_TRANSACTION_TIMEOUT
Definition: rdkafka.h:530
@ RD_KAFKA_RESP_ERR_INVALID_PRINCIPAL_TYPE
Definition: rdkafka.h:568
@ RD_KAFKA_RESP_ERR_PREFERRED_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:594
@ RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION
Definition: rdkafka.h:306
@ RD_KAFKA_RESP_ERR__FS
Definition: rdkafka.h:308
@ RD_KAFKA_RESP_ERR__TIMED_OUT
Definition: rdkafka.h:316
@ RD_KAFKA_RESP_ERR_REQUEST_TIMED_OUT
Definition: rdkafka.h:433
@ RD_KAFKA_RESP_ERR_LEADER_NOT_AVAILABLE
Definition: rdkafka.h:427
@ RD_KAFKA_RESP_ERR_INVALID_REQUIRED_ACKS
Definition: rdkafka.h:470
@ RD_KAFKA_RESP_ERR__FATAL
Definition: rdkafka.h:386
@ RD_KAFKA_RESP_ERR__UNDERFLOW
Definition: rdkafka.h:376
@ RD_KAFKA_RESP_ERR__BEGIN
Definition: rdkafka.h:283
@ RD_KAFKA_RESP_ERR__NOENT
Definition: rdkafka.h:374
@ RD_KAFKA_RESP_ERR_UNKNOWN_TOPIC_OR_PART
Definition: rdkafka.h:423
@ RD_KAFKA_RESP_ERR_RECORD_LIST_TOO_LARGE
Definition: rdkafka.h:464
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_EPOCH
Definition: rdkafka.h:522
@ RD_KAFKA_RESP_ERR_NETWORK_EXCEPTION
Definition: rdkafka.h:445
@ RD_KAFKA_RESP_ERR_KAFKA_STORAGE_ERROR
Definition: rdkafka.h:545
@ RD_KAFKA_RESP_ERR_GROUP_ID_NOT_FOUND
Definition: rdkafka.h:572
@ RD_KAFKA_RESP_ERR_TOPIC_ALREADY_EXISTS
Definition: rdkafka.h:500
@ RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
Definition: rdkafka.h:452
@ RD_KAFKA_RESP_ERR_MSG_SIZE_TOO_LARGE
Definition: rdkafka.h:439
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTH_DISABLED
Definition: rdkafka.h:556
@ RD_KAFKA_RESP_ERR_DUPLICATE_RESOURCE
Definition: rdkafka.h:621
@ RD_KAFKA_RESP_ERR_INVALID_CONFIG
Definition: rdkafka.h:508
@ RD_KAFKA_RESP_ERR_FEATURE_UPDATE_FAILED
Definition: rdkafka.h:630
@ RD_KAFKA_RESP_ERR__MAX_POLL_EXCEEDED
Definition: rdkafka.h:392
@ RD_KAFKA_RESP_ERR_INVALID_UPDATE_VERSION
Definition: rdkafka.h:628
@ RD_KAFKA_RESP_ERR__UNKNOWN_GROUP
Definition: rdkafka.h:328
@ RD_KAFKA_RESP_ERR__ISR_INSUFF
Definition: rdkafka.h:320
@ RD_KAFKA_RESP_ERR_SASL_AUTHENTICATION_FAILED
Definition: rdkafka.h:550
@ RD_KAFKA_RESP_ERR_INVALID_PRODUCER_ID_MAPPING
Definition: rdkafka.h:527
@ RD_KAFKA_RESP_ERR__IN_PROGRESS
Definition: rdkafka.h:330
@ RD_KAFKA_RESP_ERR__READ_ONLY
Definition: rdkafka.h:372
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS_AFTER_APPEND
Definition: rdkafka.h:468
@ RD_KAFKA_RESP_ERR__UNKNOWN_BROKER
Definition: rdkafka.h:394
@ RD_KAFKA_RESP_ERR_INCONSISTENT_GROUP_PROTOCOL
Definition: rdkafka.h:474
@ RD_KAFKA_RESP_ERR_NON_EMPTY_GROUP
Definition: rdkafka.h:570
@ RD_KAFKA_RESP_ERR_SECURITY_DISABLED
Definition: rdkafka.h:541
@ RD_KAFKA_RESP_ERR_TOPIC_DELETION_DISABLED
Definition: rdkafka.h:580
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_OWNER_MISMATCH
Definition: rdkafka.h:560
@ RD_KAFKA_RESP_ERR__INVALID_ARG
Definition: rdkafka.h:314
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_VERSION
Definition: rdkafka.h:498
@ RD_KAFKA_RESP_ERR__OUTDATED
Definition: rdkafka.h:352
@ RD_KAFKA_RESP_ERR__FAIL
Definition: rdkafka.h:291
@ RD_KAFKA_RESP_ERR_INVALID_MSG_SIZE
Definition: rdkafka.h:425
@ RD_KAFKA_RESP_ERR_STALE_BROKER_EPOCH
Definition: rdkafka.h:588
@ RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
Definition: rdkafka.h:447
@ RD_KAFKA_RESP_ERR__ASSIGN_PARTITIONS
Definition: rdkafka.h:336
@ RD_KAFKA_RESP_ERR__VALUE_SERIALIZATION
Definition: rdkafka.h:364
@ RD_KAFKA_RESP_ERR_FETCH_SESSION_ID_NOT_FOUND
Definition: rdkafka.h:574
@ RD_KAFKA_RESP_ERR__REVOKE_PARTITIONS
Definition: rdkafka.h:338
@ RD_KAFKA_RESP_ERR__APPLICATION
Definition: rdkafka.h:400
@ RD_KAFKA_RESP_ERR_REPLICA_NOT_AVAILABLE
Definition: rdkafka.h:437
@ RD_KAFKA_RESP_ERR__EXISTING_SUBSCRIPTION
Definition: rdkafka.h:334
@ RD_KAFKA_RESP_ERR_NOT_CONTROLLER
Definition: rdkafka.h:510
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_NOT_FOUND
Definition: rdkafka.h:558
@ RD_KAFKA_RESP_ERR_LISTENER_NOT_FOUND
Definition: rdkafka.h:578
@ RD_KAFKA_RESP_ERR_INVALID_GROUP_ID
Definition: rdkafka.h:476
@ RD_KAFKA_RESP_ERR_UNSTABLE_OFFSET_COMMIT
Definition: rdkafka.h:612
@ RD_KAFKA_RESP_ERR__GAPLESS_GUARANTEE
Definition: rdkafka.h:390
@ RD_KAFKA_RESP_ERR__INVALID_TYPE
Definition: rdkafka.h:378
@ RD_KAFKA_RESP_ERR_OFFSET_NOT_AVAILABLE
Definition: rdkafka.h:590
@ RD_KAFKA_RESP_ERR__PURGE_INFLIGHT
Definition: rdkafka.h:384
@ RD_KAFKA_RESP_ERR__STATE
Definition: rdkafka.h:342
@ RD_KAFKA_RESP_ERR_UNKNOWN_PRODUCER_ID
Definition: rdkafka.h:552
@ RD_KAFKA_RESP_ERR_UNKNOWN_LEADER_EPOCH
Definition: rdkafka.h:584
@ RD_KAFKA_RESP_ERR_UNACCEPTABLE_CREDENTIAL
Definition: rdkafka.h:623
@ RD_KAFKA_RESP_ERR_BROKER_NOT_AVAILABLE
Definition: rdkafka.h:435
@ RD_KAFKA_RESP_ERR_INCONSISTENT_VOTER_SET
Definition: rdkafka.h:626
@ RD_KAFKA_RESP_ERR_ELIGIBLE_LEADERS_NOT_AVAILABLE
Definition: rdkafka.h:601
@ RD_KAFKA_RESP_ERR__RETRY
Definition: rdkafka.h:380
@ RD_KAFKA_RESP_ERR_INVALID_REPLICATION_FACTOR
Definition: rdkafka.h:504
@ RD_KAFKA_RESP_ERR_NOT_COORDINATOR
Definition: rdkafka.h:457
@ RD_KAFKA_RESP_ERR__PREV_IN_PROGRESS
Definition: rdkafka.h:332
@ RD_KAFKA_RESP_ERR_ILLEGAL_GENERATION
Definition: rdkafka.h:472
@ RD_KAFKA_RESP_ERR__BAD_MSG
Definition: rdkafka.h:285
@ RD_KAFKA_RESP_ERR__BAD_COMPRESSION
Definition: rdkafka.h:287
@ RD_KAFKA_RESP_ERR__QUEUE_FULL
Definition: rdkafka.h:318
@ RD_KAFKA_RESP_ERR__WAIT_CACHE
Definition: rdkafka.h:358
@ RD_KAFKA_RESP_ERR__RESOLVE
Definition: rdkafka.h:297
@ RD_KAFKA_RESP_ERR_PRODUCER_FENCED
Definition: rdkafka.h:617
@ RD_KAFKA_RESP_ERR__AUTHENTICATION
Definition: rdkafka.h:348
@ RD_KAFKA_RESP_ERR__NOT_IMPLEMENTED
Definition: rdkafka.h:346
@ RD_KAFKA_RESP_ERR_INVALID_RECORD
Definition: rdkafka.h:610
@ RD_KAFKA_RESP_ERR__UNKNOWN_PROTOCOL
Definition: rdkafka.h:344
@ RD_KAFKA_RESP_ERR__PARTIAL
Definition: rdkafka.h:370
@ RD_KAFKA_RESP_ERR_STALE_CTRL_EPOCH
Definition: rdkafka.h:441
@ RD_KAFKA_RESP_ERR_CLUSTER_AUTHORIZATION_FAILED
Definition: rdkafka.h:490
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_AUTHORIZATION_FAILED
Definition: rdkafka.h:564
@ RD_KAFKA_RESP_ERR_NOT_ENOUGH_REPLICAS
Definition: rdkafka.h:466
@ RD_KAFKA_RESP_ERR_GROUP_MAX_SIZE_REACHED
Definition: rdkafka.h:596
@ RD_KAFKA_RESP_ERR_INVALID_TXN_STATE
Definition: rdkafka.h:524
@ RD_KAFKA_RESP_ERR__DESTROY
Definition: rdkafka.h:289
@ RD_KAFKA_RESP_ERR__INCONSISTENT
Definition: rdkafka.h:388
@ RD_KAFKA_RESP_ERR__KEY_SERIALIZATION
Definition: rdkafka.h:362
@ RD_KAFKA_RESP_ERR_UNKNOWN_MEMBER_ID
Definition: rdkafka.h:478
@ RD_KAFKA_RESP_ERR_MEMBER_ID_REQUIRED
Definition: rdkafka.h:592
@ RD_KAFKA_RESP_ERR_CONCURRENT_TRANSACTIONS
Definition: rdkafka.h:533
@ RD_KAFKA_RESP_ERR__TRANSPORT
Definition: rdkafka.h:293
@ RD_KAFKA_RESP_ERR_INVALID_TIMESTAMP
Definition: rdkafka.h:492
@ RD_KAFKA_RESP_ERR_INVALID_REQUEST
Definition: rdkafka.h:512
@ RD_KAFKA_RESP_ERR_OFFSET_OUT_OF_RANGE
Definition: rdkafka.h:419
@ RD_KAFKA_RESP_ERR__LOG_TRUNCATION
Definition: rdkafka.h:408
@ RD_KAFKA_RESP_ERR_FENCED_INSTANCE_ID
Definition: rdkafka.h:599
@ RD_KAFKA_RESP_ERR_DELEGATION_TOKEN_REQUEST_NOT_ALLOWED
Definition: rdkafka.h:562
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_SASL_MECHANISM
Definition: rdkafka.h:494
@ RD_KAFKA_RESP_ERR__CONFLICT
Definition: rdkafka.h:340
@ RD_KAFKA_RESP_ERR_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:554
@ RD_KAFKA_RESP_ERR_INVALID_COMMIT_OFFSET_SIZE
Definition: rdkafka.h:484
@ RD_KAFKA_RESP_ERR__NOOP
Definition: rdkafka.h:404
@ RD_KAFKA_RESP_ERR_THROTTLING_QUOTA_EXCEEDED
Definition: rdkafka.h:614
@ RD_KAFKA_RESP_ERR__TIMED_OUT_QUEUE
Definition: rdkafka.h:354
@ RD_KAFKA_RESP_ERR_DUPLICATE_SEQUENCE_NUMBER
Definition: rdkafka.h:520
@ RD_KAFKA_RESP_ERR_ELECTION_NOT_NEEDED
Definition: rdkafka.h:603
@ RD_KAFKA_RESP_ERR_TOPIC_AUTHORIZATION_FAILED
Definition: rdkafka.h:486
@ RD_KAFKA_RESP_ERR_TRANSACTIONAL_ID_AUTHORIZATION_FAILED
Definition: rdkafka.h:539
@ RD_KAFKA_RESP_ERR_GROUP_SUBSCRIBED_TO_TOPIC
Definition: rdkafka.h:608
@ RD_KAFKA_RESP_ERR__ASSIGNMENT_LOST
Definition: rdkafka.h:402
@ RD_KAFKA_RESP_ERR__MSG_TIMED_OUT
Definition: rdkafka.h:299
@ RD_KAFKA_RESP_ERR_INVALID_SESSION_TIMEOUT
Definition: rdkafka.h:480
@ RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC
Definition: rdkafka.h:310
@ RD_KAFKA_RESP_ERR_PRINCIPAL_DESERIALIZATION_FAILURE
Definition: rdkafka.h:632
@ RD_KAFKA_RESP_ERR_OFFSET_METADATA_TOO_LARGE
Definition: rdkafka.h:443
@ RD_KAFKA_RESP_ERR__NODE_UPDATE
Definition: rdkafka.h:322
@ RD_KAFKA_RESP_ERR_NO_REASSIGNMENT_IN_PROGRESS
Definition: rdkafka.h:605
@ RD_KAFKA_RESP_ERR__CRIT_SYS_RESOURCE
Definition: rdkafka.h:295
@ RD_KAFKA_RESP_ERR_INVALID_FETCH_SESSION_EPOCH
Definition: rdkafka.h:576
@ RD_KAFKA_RESP_ERR__VALUE_DESERIALIZATION
Definition: rdkafka.h:368
@ RD_KAFKA_RESP_ERR__NO_OFFSET
Definition: rdkafka.h:350
@ RD_KAFKA_RESP_ERR_INVALID_MSG
Definition: rdkafka.h:421
@ RD_KAFKA_RESP_ERR_LOG_DIR_NOT_FOUND
Definition: rdkafka.h:548
@ RD_KAFKA_RESP_ERR__ALL_BROKERS_DOWN
Definition: rdkafka.h:312
@ RD_KAFKA_RESP_ERR_INVALID_REPLICA_ASSIGNMENT
Definition: rdkafka.h:506
@ RD_KAFKA_RESP_ERR__FENCED
Definition: rdkafka.h:398
@ RD_KAFKA_RESP_ERR_INVALID_PARTITIONS
Definition: rdkafka.h:502
@ RD_KAFKA_RESP_ERR__NOT_CONFIGURED
Definition: rdkafka.h:396
@ RD_KAFKA_RESP_ERR__UNSUPPORTED_FEATURE
Definition: rdkafka.h:356
@ RD_KAFKA_RESP_ERR_NO_ERROR
Definition: rdkafka.h:417
@ RD_KAFKA_RESP_ERR__END
Definition: rdkafka.h:411
@ RD_KAFKA_RESP_ERR__INTR
Definition: rdkafka.h:360
@ RD_KAFKA_RESP_ERR_ILLEGAL_SASL_STATE
Definition: rdkafka.h:496
@ RD_KAFKA_RESP_ERR_GROUP_AUTHORIZATION_FAILED
Definition: rdkafka.h:488
@ RD_KAFKA_RESP_ERR_UNSUPPORTED_COMPRESSION_TYPE
Definition: rdkafka.h:586
@ RD_KAFKA_RESP_ERR_OUT_OF_ORDER_SEQUENCE_NUMBER
Definition: rdkafka.h:518
@ RD_KAFKA_RESP_ERR__KEY_DESERIALIZATION
Definition: rdkafka.h:366
@ RD_KAFKA_RESP_ERR__SSL
Definition: rdkafka.h:324
@ RD_KAFKA_RESP_ERR_REBALANCE_IN_PROGRESS
Definition: rdkafka.h:482
@ RD_KAFKA_RESP_ERR_UNKNOWN
Definition: rdkafka.h:415
@ RD_KAFKA_RESP_ERR_FENCED_LEADER_EPOCH
Definition: rdkafka.h:582
@ RD_KAFKA_RESP_ERR__WAIT_COORD
Definition: rdkafka.h:326
@ RD_KAFKA_RESP_ERR__PARTITION_EOF
Definition: rdkafka.h:304
@ RD_KAFKA_RESP_ERR_TOPIC_EXCEPTION
Definition: rdkafka.h:462
RD_EXPORT void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf, void(*throttle_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int throttle_time_ms, void *opaque))
Set throttle callback.
RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic)
Destroy and free a NewTopic object previously created with rd_kafka_NewTopic_new()
RD_EXPORT void rd_kafka_conf_set_socket_cb(rd_kafka_conf_t *conf, int(*socket_cb)(int domain, int type, int protocol, void *opaque))
Set socket callback.
RD_EXPORT void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs, size_t *cntp)
Returns the full list of error codes.
RD_EXPORT const rd_kafka_error_t * rd_kafka_AclBinding_error(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_store(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets)
Store offsets for next auto-commit for one or more partitions.
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupListing_state(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets state for the grplist group.
RD_EXPORT void rd_kafka_DeleteRecords_destroy_array(rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt)
Helper function to destroy all DeleteRecords objects in the del_groups array (of del_group_cnt elemen...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Atomic assignment of partitions to consume.
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:6997
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:7215
RD_EXPORT void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf, void(*log_cb)(const rd_kafka_t *rk, int level, const char *fac, const char *buf))
Set logger callback.
RD_EXPORT void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf, void(*dr_cb)(rd_kafka_t *rk, void *payload, size_t len, rd_kafka_resp_err_t err, void *opaque, void *msg_opaque))
RD_EXPORT const char * rd_kafka_ConfigResource_error_string(const rd_kafka_ConfigResource_t *config)
RD_EXPORT void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf, void *callback_data)
Set callback_data for OpenSSL engine.
RD_EXPORT void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque)
Sets the application's opaque pointer that will be passed to callbacks.
struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t
Definition: rdkafka.h:7808
RD_EXPORT int rd_kafka_thread_cnt(void)
Retrieve the current number of threads in use by librdkafka.
RD_EXPORT const rd_kafka_CreateTopics_result_t * rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev)
Get CreateTopics result.
RD_EXPORT int32_t rd_kafka_ScramCredentialInfo_iterations(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the iterations of a given ScramCredentialInfo.
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_ListConsumerGroupOffsets_result_groups(const rd_kafka_ListConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a ListConsumerGroupOffsets result.
RD_EXPORT void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata)
Release metadata memory.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_request_sent_t(rd_kafka_t *rk, int sockfd, const char *brokername, int32_t brokerid, int16_t ApiKey, int16_t ApiVersion, int32_t CorrId, size_t size, void *ic_opaque)
on_request_sent() is called when a request has been fully written to a broker TCP connections socket.
Definition: rdkafka.h:6239
RD_EXPORT void rd_kafka_topic_partition_list_destroy(rd_kafka_topic_partition_list_t *rkparlist)
Free all resources used by the list and the list itself.
RD_EXPORT const char * rd_kafka_version_str(void)
Returns the librdkafka version as string.
RD_EXPORT const rd_kafka_error_t ** rd_kafka_ListConsumerGroups_result_errors(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of errors from a ListConsumerGroups call result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_commit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_commit_t *on_commit, void *ic_opaque)
Append an on_commit() interceptor.
rd_kafka_event_t rd_kafka_DeleteRecords_result_t
Definition: rdkafka.h:5628
RD_EXPORT const rd_kafka_DescribeConfigs_result_t * rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev)
Get DescribeConfigs result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_consume(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_consume_t *on_consume, void *ic_opaque)
Append an on_consume() interceptor.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscribe(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *topics)
Subscribe to topic set using balanced consumer groups.
RD_EXPORT const rd_kafka_error_t * rd_kafka_AlterUserScramCredentials_result_response_error(const rd_kafka_AlterUserScramCredentials_result_response_t *response)
Returns the error of a rd_kafka_AlterUserScramCredentials_result_response.
struct rd_kafka_DeleteAcls_result_response_s rd_kafka_DeleteAcls_result_response_t
Definition: rdkafka.h:9143
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_partition_list_set_offset(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition, int64_t offset)
Set offset to offset for topic and partition.
RD_EXPORT rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assignment(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **partitions)
Returns the current partition assignment as set by rd_kafka_assign() or rd_kafka_incremental_assign()...
RD_EXPORT void rd_kafka_conf_set_rebalance_cb(rd_kafka_conf_t *conf, void(*rebalance_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *partitions, void *opaque))
Consumer: Set rebalance callback for use with coordinated consumer group balancing.
RD_EXPORT void rd_kafka_yield(rd_kafka_t *rk)
Cancels the current callback dispatcher (rd_kafka_poll(), rd_kafka_consume_callback(),...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_destroy(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_destroy_t *on_destroy, void *ic_opaque)
Append an on_destroy() interceptor.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_find(const rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Find element by topic and partition.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_add(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Add topic+partition to list.
RD_EXPORT void rd_kafka_conf_set_resolve_cb(rd_kafka_conf_t *conf, int(*resolve_cb)(const char *node, const char *service, const struct addrinfo *hints, struct addrinfo **res, void *opaque))
Set address resolution callback.
RD_EXPORT void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf, rd_kafka_topic_conf_t *tconf)
Sets the default topic configuration to use for automatically subscribed topics (e....
RD_EXPORT int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for events served through callbacks for max timeout_ms.
RD_EXPORT const rd_kafka_AclBinding_t ** rd_kafka_DeleteAcls_result_response_matching_acls(const rd_kafka_DeleteAcls_result_response_t *result_response, size_t *matching_acls_cntp)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf)
Creates a copy/duplicate of topic configuration object conf.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_dup_t(rd_kafka_conf_t *new_conf, const rd_kafka_conf_t *old_conf, size_t filter_cnt, const char **filter, void *ic_opaque)
on_conf_dup() is called from rd_kafka_conf_dup() in the order the interceptors were added and is used...
Definition: rdkafka.h:6061
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata(rd_kafka_t *rk)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token(rd_kafka_t *rk, const char *token_value, int64_t md_lifetime_ms, const char *md_principal_name, const char **extensions, size_t extension_size, char *errstr, size_t errstr_size)
Set SASL/OAUTHBEARER token and metadata.
RD_EXPORT size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev, const rd_kafka_message_t **rkmessages, size_t size)
Extacts size message(s) from the event into the pre-allocated array rkmessages.
RD_EXPORT int rd_kafka_ConfigEntry_is_synonym(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_topic_conf_new(void)
Create topic configuration object.
RD_EXPORT void rd_kafka_conf_set_offset_commit_cb(rd_kafka_conf_t *conf, void(*offset_commit_cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *opaque))
Consumer: Set offset commit callback for use with consumer groups.
RD_EXPORT const char * rd_kafka_ConsumerGroupDescription_group_id(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the group id for the grpdesc group.
RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk, rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create additional partitions for the given topics, as specified by the new_parts array of size new_pa...
RD_EXPORT const rd_kafka_DescribeConsumerGroups_result_t * rd_kafka_event_DescribeConsumerGroups_result(rd_kafka_event_t *rkev)
Get DescribeConsumerGroups result.
RD_EXPORT const rd_kafka_UserScramCredentialsDescription_t ** rd_kafka_DescribeUserScramCredentials_result_descriptions(const rd_kafka_DescribeUserScramCredentials_result_t *result, size_t *cntp)
Get an array of descriptions from a DescribeUserScramCredentials result.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_new_t(rd_kafka_t *rk, const rd_kafka_conf_t *conf, void *ic_opaque, char *errstr, size_t errstr_size)
on_new() is called from rd_kafka_new() prior toreturning the newly created client instance to the app...
Definition: rdkafka.h:6096
RD_EXPORT rd_kafka_DeleteTopic_t * rd_kafka_DeleteTopic_new(const char *topic)
Create a new DeleteTopic object. This object is later passed to rd_kafka_DeleteTopics().
RD_EXPORT const char * rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres)
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_DescribeConfigs_result_resources(const rd_kafka_DescribeConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeConfigs result.
RD_EXPORT int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_AclBindingFilter_t * rd_kafka_AclBindingFilter_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBindingFilter object. This object is later passed to rd_kafka_DescribeAcls() or rd_ka...
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_DeleteGroups_result_groups(const rd_kafka_DeleteGroups_result_t *result, size_t *cntp)
Get an array of group results from a DeleteGroups result.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy_array(rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffset_cnt)
Helper function to destroy all AlterConsumerGroupOffsets objects in the alter_grpoffsets array (of al...
rd_kafka_event_t rd_kafka_CreatePartitions_result_t
Definition: rdkafka.h:5620
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...
struct rd_kafka_AlterUserScramCredentials_result_response_s rd_kafka_AlterUserScramCredentials_result_response_t
Result of a single user SCRAM alteration.
Definition: rdkafka.h:8799
RD_EXPORT char * rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms)
Returns the ClusterId as reported in broker metadata.
RD_EXPORT rd_kafka_error_t * rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt)
Produce and send a single message to broker.
RD_EXPORT void rd_kafka_dump(FILE *fp, rd_kafka_t *rk)
Dumps rdkafka's internal state for handle rk to stream fp.
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_unassign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally remove partitions from the current assignment.
RD_EXPORT rd_kafka_error_t * rd_kafka_send_offsets_to_transaction(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, const rd_kafka_consumer_group_metadata_t *cgmetadata, int timeout_ms)
Sends a list of topic partition offsets to the consumer group coordinator for cgmetadata,...
RD_EXPORT const char * rd_kafka_ConsumerGroupDescription_partition_assignor(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the partition assignor for the grpdesc group.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_pause_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Pause producing or consumption for the provided list of partitions.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_config(rd_kafka_NewTopic_t *new_topic, const char *name, const char *value)
Set (broker-side) topic configuration name/value pair.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_broker_state_change(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_broker_state_change_t *on_broker_state_change, void *ic_opaque)
Append an on_broker_state_change() interceptor.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_new(rd_kafka_t *rk)
Create a new message queue.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_background(rd_kafka_t *rk)
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf, size_t filter_cnt, const char **filter)
Same as rd_kafka_conf_dup() but with an array of property name prefixes to filter out (ignore) when c...
rd_kafka_event_t rd_kafka_DescribeConsumerGroups_result_t
Definition: rdkafka.h:5632
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreatePartitions_result_topics(const rd_kafka_CreatePartitions_result_t *result, size_t *cntp)
Get an array of topic results from a CreatePartitions result.
RD_EXPORT rd_kafka_error_t * rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms)
Aborts the ongoing transaction.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk, rd_kafka_resp_err_t err, const char *reason)
Trigger a fatal error for testing purposes.
RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding)
Destroy and free an AclBinding object previously created with rd_kafka_AclBinding_new()
RD_EXPORT rd_kafka_event_t * rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for an event for max timeout_ms.
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:8661
RD_EXPORT void rd_kafka_error_destroy(rd_kafka_error_t *error)
Free and destroy an error object.
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new(const char *group_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
rd_kafka_event_t rd_kafka_DeleteTopics_result_t
Definition: rdkafka.h:5612
RD_EXPORT rd_kafka_AclBinding_t * rd_kafka_AclBinding_new(rd_kafka_ResourceType_t restype, const char *name, rd_kafka_ResourcePatternType_t resource_pattern_type, const char *principal, const char *host, rd_kafka_AclOperation_t operation, rd_kafka_AclPermissionType_t permission_type, char *errstr, size_t errstr_size)
Create a new AclBinding object. This object is later passed to rd_kafka_CreateAcls().
RD_EXPORT void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage)
Frees resources for rkmessage and hands ownership back to rdkafka.
RD_EXPORT void rd_kafka_IncrementalAlterConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Incrementally update the configuration for the specified resources. Updates are not transactional so ...
RD_EXPORT const rd_kafka_DeleteConsumerGroupOffsets_result_t * rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get DeleteConsumerGroupOffsets result.
rd_kafka_event_t rd_kafka_ListConsumerGroupOffsets_result_t
Definition: rdkafka.h:5640
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_dup(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_dup_t *on_conf_dup, void *ic_opaque)
Append an on_conf_dup() interceptor.
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_require_stable_offsets(rd_kafka_AdminOptions_t *options, int true_or_false)
Whether broker should return stable offsets (transaction-committed).
RD_EXPORT rd_kafka_resp_err_t rd_kafka_consumer_close(rd_kafka_t *rk)
Close the consumer.
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk)
Enable SASL OAUTHBEARER refresh callbacks on the librdkafka background thread.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_committed(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Retrieve committed offsets for topics+partitions.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_send_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_send() is called from rd_kafka_produce*() (et.al) prior to the partitioner being called.
Definition: rdkafka.h:6136
RD_EXPORT const rd_kafka_ConsumerGroupListing_t ** rd_kafka_ListConsumerGroups_result_valid(const rd_kafka_ListConsumerGroups_result_t *result, size_t *cntp)
Get an array of valid list groups from a ListConsumerGroups result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs, size_t idx, const char **namep, const void **valuep, size_t *sizep)
Iterator for all headers.
RD_EXPORT const rd_kafka_message_t * rd_kafka_event_message_next(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_error_name(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_error_t * rd_kafka_incremental_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Incrementally add partitions to the current assignment.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_AlterConsumerGroupOffsets_t **alter_grpoffsets, size_t alter_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Alter committed offsets for a set of partitions in a consumer group. This will succeed at the partiti...
RD_EXPORT void rd_kafka_DescribeConsumerGroups(rd_kafka_t *rk, const char **groups, size_t groups_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe groups from cluster as specified by the groups array of size groups_cnt elements.
RD_EXPORT rd_kafka_error_t * rd_kafka_begin_transaction(rd_kafka_t *rk)
Begin a new transaction.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_unsubscribe(rd_kafka_t *rk)
Unsubscribe from the current subscription set.
RD_EXPORT void rd_kafka_log_print(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin (default) log sink: print to stderr.
RD_EXPORT ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume batch of messages from queue.
RD_EXPORT const char * rd_kafka_topic_name(const rd_kafka_topic_t *rkt)
Returns the topic name.
RD_EXPORT const char * rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres)
RD_EXPORT const char ** rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf, size_t *cntp)
Dump the topic configuration properties and values of conf to an array with "key",...
RD_EXPORT void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu, void(*event_cb)(rd_kafka_t *rk, void *qev_opaque), void *qev_opaque)
Enable callback event triggering for queue.
RD_EXPORT int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage, rd_kafka_timestamp_type_t *tstype)
Returns the message timestamp for a consumed message.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_request_sent(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_request_sent_t *on_request_sent, void *ic_opaque)
Append an on_request_sent() interceptor.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size)
Returns the first fatal error set on this client instance, or RD_KAFKA_RESP_ERR_NO_ERROR if no fatal ...
rd_kafka_event_t rd_kafka_CreateAcls_result_t
Definition: rdkafka.h:5614
RD_EXPORT rd_kafka_resp_err_t rd_kafka_query_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high, int timeout_ms)
Query broker for low (oldest/beginning) and high (newest/end) offsets for partition.
rd_kafka_event_t rd_kafka_CreateTopics_result_t
Definition: rdkafka.h:5610
RD_EXPORT int rd_kafka_consume_callback_queue(rd_kafka_queue_t *rkqu, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consume multiple messages from queue with callback.
RD_EXPORT void rd_kafka_conf_set_open_cb(rd_kafka_conf_t *conf, int(*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque))
Set open callback.
rd_kafka_event_t rd_kafka_DeleteAcls_result_t
Definition: rdkafka.h:5618
RD_EXPORT const char * rd_kafka_ResourcePatternType_name(rd_kafka_ResourcePatternType_t resource_pattern_type)
RD_EXPORT const char * rd_kafka_Node_host(const rd_kafka_Node_t *node)
Get the host of node.
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreateTopics_result_topics(const rd_kafka_CreateTopics_result_t *result, size_t *cntp)
Get an array of topic results from a CreateTopics result.
rd_kafka_event_t rd_kafka_AlterUserScramCredentials_result_t
Definition: rdkafka.h:5644
RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt)
Helper function to destroy all NewTopic objects in the new_topics array (of new_topic_cnt elements)....
rd_kafka_event_t rd_kafka_DescribeAcls_result_t
Definition: rdkafka.h:5616
RD_EXPORT const rd_kafka_DeleteTopics_result_t * rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev)
Get DeleteTopics result.
RD_EXPORT void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr)
Free pointer returned by librdkafka.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_broker_state_change_t(rd_kafka_t *rk, int32_t broker_id, const char *secproto, const char *name, int port, const char *state, void *ic_opaque)
on_broker_state_change() is called just after a broker has been created or its state has been changed...
Definition: rdkafka.h:6353
RD_EXPORT const rd_kafka_error_t * rd_kafka_DeleteAcls_result_response_error(const rd_kafka_DeleteAcls_result_response_t *result_response)
rd_kafka_AclOperation_t
Apache Kafka ACL operation types.
Definition: rdkafka.h:8885
@ RD_KAFKA_ACL_OPERATION_ALL
Definition: rdkafka.h:8889
@ RD_KAFKA_ACL_OPERATION_ALTER
Definition: rdkafka.h:8894
@ RD_KAFKA_ACL_OPERATION_CLUSTER_ACTION
Definition: rdkafka.h:8896
@ RD_KAFKA_ACL_OPERATION_IDEMPOTENT_WRITE
Definition: rdkafka.h:8902
@ RD_KAFKA_ACL_OPERATION_DELETE
Definition: rdkafka.h:8893
@ RD_KAFKA_ACL_OPERATION_WRITE
Definition: rdkafka.h:8891
@ RD_KAFKA_ACL_OPERATION_ANY
Definition: rdkafka.h:8887
@ RD_KAFKA_ACL_OPERATION_UNKNOWN
Definition: rdkafka.h:8886
@ RD_KAFKA_ACL_OPERATION_CREATE
Definition: rdkafka.h:8892
@ RD_KAFKA_ACL_OPERATION_ALTER_CONFIGS
Definition: rdkafka.h:8900
@ RD_KAFKA_ACL_OPERATION_DESCRIBE_CONFIGS
Definition: rdkafka.h:8898
@ RD_KAFKA_ACL_OPERATION_READ
Definition: rdkafka.h:8890
@ RD_KAFKA_ACL_OPERATION_DESCRIBE
Definition: rdkafka.h:8895
RD_EXPORT ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, rd_kafka_message_t **rkmessages, size_t rkmessages_size)
Consume up to rkmessages_size from topic rkt and partition putting a pointer to each message in the a...
RD_EXPORT int rd_kafka_event_log(rd_kafka_event_t *rkev, const char **fac, const char **str, int *level)
Extract log message from the event.
RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk)
Check whether the consumer considers the current assignment to have been lost involuntarily....
RD_EXPORT void rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf, int(*connect_cb)(int sockfd, const struct sockaddr *addr, int addrlen, const char *id, void *opaque))
Set connect callback.
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy_array(rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt)
Destroys an array of UserScramCredentialAlteration.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf, rd_kafka_cert_type_t cert_type, rd_kafka_cert_enc_t cert_enc, const void *buffer, size_t size, char *errstr, size_t errstr_size)
Set certificate/key cert_type from the cert_enc encoded memory at buffer of size bytes.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve topic configuration value for property name.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_sasl(rd_kafka_t *rk)
RD_EXPORT int rd_kafka_consume_callback(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque), void *commit_opaque)
Consumes messages from topic rkt and partition, calling the provided callback for each consumed messs...
RD_EXPORT const char * rd_kafka_rebalance_protocol(rd_kafka_t *rk)
The rebalance protocol currently in use. This will be "NONE" if the consumer has not (yet) joined a g...
RD_EXPORT int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Random partitioner.
RD_EXPORT const char * rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage)
Returns the error string for an errored rd_kafka_message_t or NULL if there was no error.
RD_EXPORT void rd_kafka_conf_set_stats_cb(rd_kafka_conf_t *conf, int(*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque))
Set statistics callback in provided conf object.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags)
Purge messages currently handled by the producer instance.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_oauthbearer_set_token_failure(rd_kafka_t *rk, const char *errstr)
SASL/OAUTHBEARER token refresh failure indicator.
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres)
RD_EXPORT int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev, char *dst, size_t dstsize)
Extract log debug context from event.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_set_log_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Forward librdkafka logs (and debug) to the specified queue for serving with one of the ....
RD_EXPORT void rd_kafka_conf_set_background_event_cb(rd_kafka_conf_t *conf, void(*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque))
Generic event callback to be used with the event API to trigger callbacks for rd_kafka_event_t object...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get(const rd_kafka_headers_t *hdrs, size_t idx, const char *name, const void **valuep, size_t *sizep)
Iterator for headers matching name.
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_consumer_group_state_code(const char *name)
Returns a code for a state name.
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_MemberAssignment_partitions(const rd_kafka_MemberAssignment_t *assignment)
Gets assigned partitions of a member assignment.
RD_EXPORT void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst)
Forward/re-route queue src to dst. If dst is NULL the forwarding is removed.
RD_EXPORT const rd_kafka_ConfigEntry_t ** rd_kafka_ConfigResource_configs(const rd_kafka_ConfigResource_t *config, size_t *cntp)
Get an array of config entries from a ConfigResource object.
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy_array(rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffset_cnt)
Helper function to destroy all DeleteConsumerGroupOffsets objects in the del_grpoffsets array (of del...
rd_kafka_cert_type_t
SSL certificate type.
Definition: rdkafka.h:2383
@ RD_KAFKA_CERT_PUBLIC_KEY
Definition: rdkafka.h:2384
@ RD_KAFKA_CERT_PRIVATE_KEY
Definition: rdkafka.h:2385
@ RD_KAFKA_CERT_CA
Definition: rdkafka.h:2386
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_copy(const rd_kafka_headers_t *src)
Make a copy of headers list src.
RD_EXPORT const char * rd_kafka_event_name(const rd_kafka_event_t *rkev)
RD_EXPORT void * rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size)
Allocate memory using the same allocator librdkafka uses.
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets_destroy(rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets)
Destroy and free a DeleteConsumerGroupOffsets object previously created with rd_kafka_DeleteConsumerG...
RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t **del_acls, size_t del_acls_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete acls matching the filteres provided in del_acls array of size del_acls_cnt.
RD_EXPORT size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf)
Gets the default topic configuration as previously set with rd_kafka_conf_set_default_topic_conf() or...
RD_EXPORT void rd_kafka_topic_partition_list_add_range(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t start, int32_t stop)
Add range of partitions from start to stop inclusive.
RD_EXPORT rd_kafka_error_t * rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms)
Commit the current transaction (as started with rd_kafka_begin_transaction()).
RD_EXPORT const rd_kafka_Node_t * rd_kafka_ConsumerGroupDescription_coordinator(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the coordinator for the grpdesc group.
RD_EXPORT rd_kafka_t * rd_kafka_new(rd_kafka_type_t type, rd_kafka_conf_t *conf, char *errstr, size_t errstr_size)
Creates a new Kafka handle and starts its operation according to the specified type (RD_KAFKA_CONSUME...
struct rd_kafka_ListConsumerGroupsResult_s rd_kafka_ListConsumerGroupsResult_t
Definition: rdkafka.h:7906
RD_EXPORT const char * rd_kafka_ConsumerGroupListing_group_id(const rd_kafka_ConsumerGroupListing_t *grplist)
Gets the group id for the grplist group.
RD_EXPORT rd_kafka_message_t * rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms)
Poll the consumer for messages or events.
RD_EXPORT void rd_kafka_conf_set_consume_cb(rd_kafka_conf_t *conf, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque))
Consumer: Set consume callback for use with rd_kafka_consumer_poll()
RD_EXPORT rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, int timeout_ms)
Seek consumer for topic+partition to offset which is either an absolute or logical offset.
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_copy(const rd_kafka_topic_partition_list_t *src)
Make a copy of an existing list.
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata_new_with_genid(const char *group_id, int32_t generation_id, const char *member_id, const char *group_instance_id)
Create a new consumer group metadata object. This is typically only used for writing tests.
RD_EXPORT void rd_kafka_DeleteRecords_destroy(rd_kafka_DeleteRecords_t *del_records)
Destroy and free a DeleteRecords object previously created with rd_kafka_DeleteRecords_new()
RD_EXPORT rd_kafka_AdminOptions_t * rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api)
Create a new AdminOptions object.
RD_EXPORT const rd_kafka_DescribeUserScramCredentials_result_t * rd_kafka_event_DescribeUserScramCredentials_result(rd_kafka_event_t *rkev)
Get DescribeUserScramCredentials result.
RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk, rd_kafka_NewTopic_t **new_topics, size_t new_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Create topics in cluster as specified by the new_topics array of size new_topic_cnt elements.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_error(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewPartitions_set_replica_assignment(rd_kafka_NewPartitions_t *new_parts, int32_t new_partition_idx, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker id) assignment for new_partition_idx to the replica set in broker_ids (of bro...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error)
rd_kafka_conf_res_t() rd_kafka_interceptor_f_on_conf_set_t(rd_kafka_conf_t *conf, const char *name, const char *val, char *errstr, size_t errstr_size, void *ic_opaque)
on_conf_set() is called from rd_kafka_*_conf_set() in the order the interceptors were added.
Definition: rdkafka.h:6031
RD_EXPORT rd_kafka_DeleteGroup_t * rd_kafka_DeleteGroup_new(const char *group)
Create a new DeleteGroup object. This object is later passed to rd_kafka_DeleteGroups().
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs, const char *name, const void **valuep, size_t *sizep)
Find last header in list hdrs matching name.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_list_groups(rd_kafka_t *rk, const char *group, const struct rd_kafka_group_list **grplistp, int timeout_ms)
List and describe client groups in cluster.
rd_kafka_resp_err_t() rd_kafka_plugin_f_conf_init_t(rd_kafka_conf_t *conf, void **plug_opaquep, char *errstr, size_t errstr_size)
Plugin's configuration initializer method called each time the library is referenced from configurati...
Definition: rdkafka.h:5945
RD_EXPORT rd_kafka_resp_err_t rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Retrieve current positions (offsets) for topics+partitions.
RD_EXPORT const char ** rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp)
Dump the configuration properties and values of conf to an array with "key", "value" pairs.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_operation_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the broker's operation timeout, such as the timeout for CreateTopics to complete the creation of...
RD_EXPORT const rd_kafka_ConsumerGroupDescription_t ** rd_kafka_DescribeConsumerGroups_result_groups(const rd_kafka_DescribeConsumerGroups_result_t *result, size_t *cntp)
Get an array of group results from a DescribeConsumerGroups result.
RD_EXPORT const rd_kafka_conf_t * rd_kafka_conf(rd_kafka_t *rk)
RD_EXPORT void rd_kafka_DeleteGroup_destroy(rd_kafka_DeleteGroup_t *del_group)
Destroy and free a DeleteGroup object previously created with rd_kafka_DeleteGroup_new()
RD_EXPORT void * rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size)
Allocate and zero memory using the same allocator librdkafka uses.
RD_EXPORT const char * rd_kafka_ConfigEntry_value(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT const char * rd_kafka_MemberDescription_host(const rd_kafka_MemberDescription_t *member)
Gets host of member.
RD_EXPORT void rd_kafka_log_syslog(const rd_kafka_t *rk, int level, const char *fac, const char *buf)
Builtin log sink: print to syslog.
struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t
Definition: rdkafka.h:7138
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:8639
RD_EXPORT void rd_kafka_topic_conf_set_msg_order_cmp(rd_kafka_topic_conf_t *topic_conf, int(*msg_order_cmp)(const rd_kafka_message_t *a, const rd_kafka_message_t *b))
Producer: Set message queueing order comparator callback.
RD_EXPORT rd_kafka_error_t * rd_kafka_seek_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Seek consumer for partitions in partitions to the per-partition offset in the .offset field of partit...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Store offset offset + 1 for topic rkt partition partition.
rd_kafka_AlterConfigOpType_t
Incremental alter configs operations.
Definition: rdkafka.h:7482
RD_EXPORT void rd_kafka_DeleteGroup_destroy_array(rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt)
Helper function to destroy all DeleteGroup objects in the del_groups array (of del_group_cnt elements...
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_AlterConfigs_result_resources(const rd_kafka_AlterConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a AlterConfigs result.
RD_EXPORT int rd_kafka_produce_batch(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, rd_kafka_message_t *rkmessages, int message_cnt)
Produce multiple messages.
rd_kafka_event_t rd_kafka_ListConsumerGroups_result_t
Definition: rdkafka.h:5630
RD_EXPORT const rd_kafka_MemberDescription_t * rd_kafka_ConsumerGroupDescription_member(const rd_kafka_ConsumerGroupDescription_t *grpdesc, size_t idx)
Gets a member of grpdesc group.
RD_EXPORT rd_kafka_ResourcePatternType_t rd_kafka_AclBinding_resource_pattern_type(const rd_kafka_AclBinding_t *acl)
RD_EXPORT int rd_kafka_ConfigEntry_is_sensitive(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy(rd_kafka_ListConsumerGroupOffsets_t *list_grpoffsets)
Destroy and free a ListConsumerGroupOffsets object previously created with rd_kafka_ListConsumerGroup...
RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk, rd_kafka_AclBindingFilter_t *acl_filter, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe acls matching the filter provided in acl_filter.
RD_EXPORT const char * rd_kafka_event_config_string(rd_kafka_event_t *rkev)
RD_EXPORT const rd_kafka_group_result_t ** rd_kafka_AlterConsumerGroupOffsets_result_groups(const rd_kafka_AlterConsumerGroupOffsets_result_t *result, size_t *cntp)
Get an array of results from a AlterConsumerGroupOffsets result.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_main(rd_kafka_t *rk)
RD_EXPORT const char * rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres)
rd_kafka_admin_op_t
Admin operation enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:6755
@ RD_KAFKA_ADMIN_OP_DESCRIBEUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6778
@ RD_KAFKA_ADMIN_OP_DELETERECORDS
Definition: rdkafka.h:6762
@ RD_KAFKA_ADMIN_OP_CREATETOPICS
Definition: rdkafka.h:6757
@ RD_KAFKA_ADMIN_OP_CREATEACLS
Definition: rdkafka.h:6766
@ RD_KAFKA_ADMIN_OP_INCREMENTALALTERCONFIGS
Definition: rdkafka.h:6776
@ RD_KAFKA_ADMIN_OP_DESCRIBEACLS
Definition: rdkafka.h:6767
@ RD_KAFKA_ADMIN_OP_DESCRIBECONSUMERGROUPS
Definition: rdkafka.h:6770
@ RD_KAFKA_ADMIN_OP_DELETEGROUPS
Definition: rdkafka.h:6763
@ RD_KAFKA_ADMIN_OP_ANY
Definition: rdkafka.h:6756
@ RD_KAFKA_ADMIN_OP_DELETEACLS
Definition: rdkafka.h:6768
@ RD_KAFKA_ADMIN_OP_DELETECONSUMERGROUPOFFSETS
Definition: rdkafka.h:6765
@ RD_KAFKA_ADMIN_OP_DELETETOPICS
Definition: rdkafka.h:6758
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6772
@ RD_KAFKA_ADMIN_OP_LISTCONSUMERGROUPS
Definition: rdkafka.h:6769
@ RD_KAFKA_ADMIN_OP__CNT
Definition: rdkafka.h:6781
@ RD_KAFKA_ADMIN_OP_ALTERCONSUMERGROUPOFFSETS
Definition: rdkafka.h:6774
@ RD_KAFKA_ADMIN_OP_ALTERCONFIGS
Definition: rdkafka.h:6760
@ RD_KAFKA_ADMIN_OP_CREATEPARTITIONS
Definition: rdkafka.h:6759
@ RD_KAFKA_ADMIN_OP_ALTERUSERSCRAMCREDENTIALS
Definition: rdkafka.h:6780
@ RD_KAFKA_ADMIN_OP_DESCRIBECONFIGS
Definition: rdkafka.h:6761
struct rd_kafka_AlterConsumerGroupOffsets_s rd_kafka_AlterConsumerGroupOffsets_t
Definition: rdkafka.h:8438
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_response_received(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_response_received_t *on_response_received, void *ic_opaque)
Append an on_response_received() interceptor.
RD_EXPORT const rd_kafka_error_t * rd_kafka_ConsumerGroupDescription_error(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the error for the grpdesc group.
RD_EXPORT void rd_kafka_NewPartitions_destroy_array(rd_kafka_NewPartitions_t **new_parts, size_t new_parts_cnt)
Helper function to destroy all NewPartitions objects in the new_parts array (of new_parts_cnt element...
RD_EXPORT int rd_kafka_version(void)
Returns the librdkafka version as integer.
RD_EXPORT int rd_kafka_topic_partition_list_del_by_idx(rd_kafka_topic_partition_list_t *rktparlist, int idx)
Delete partition from list by elems[] index.
RD_EXPORT size_t rd_kafka_UserScramCredentialsDescription_scramcredentialinfo_count(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the count of ScramCredentialInfos of a UserScramCredentialsDescription.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_metadata(rd_kafka_t *rk, int all_topics, rd_kafka_topic_t *only_rkt, const struct rd_kafka_metadata **metadatap, int timeout_ms)
Request Metadata from broker.
RD_EXPORT char * rd_kafka_memberid(const rd_kafka_t *rk)
Returns this client's broker-assigned group member id.
RD_EXPORT rd_kafka_error_t * rd_kafka_offset_store_message(rd_kafka_message_t *rkmessage)
Store offset +1 for the consumed message.
RD_EXPORT void * rd_kafka_event_opaque(rd_kafka_event_t *rkev)
struct rd_kafka_UserScramCredentialAlteration_s rd_kafka_UserScramCredentialAlteration_t
A request to alter a user's SASL/SCRAM credentials.
Definition: rdkafka.h:8733
RD_EXPORT const char * rd_kafka_MemberDescription_consumer_id(const rd_kafka_MemberDescription_t *member)
Gets consumer id of member.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_exit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_exit_t *on_thread_exit, void *ic_opaque)
Append an on_thread_exit() interceptor.
RD_EXPORT int rd_kafka_consumer_closed(rd_kafka_t *rk)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_commit_t(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, rd_kafka_resp_err_t err, void *ic_opaque)
on_commit() is called on completed or failed offset commit. It is called from internal librdkafka thr...
Definition: rdkafka.h:6211
RD_EXPORT rd_kafka_UserScramCredentialAlteration_t * rd_kafka_UserScramCredentialDeletion_new(const char *username, rd_kafka_ScramMechanism_t mechanism)
Allocates a new UserScramCredentialDeletion given its fields.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs, const char *name)
Remove all headers for the given key (if any).
RD_EXPORT const rd_kafka_ConfigEntry_t ** rd_kafka_ConfigEntry_synonyms(const rd_kafka_ConfigEntry_t *entry, size_t *cntp)
RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void)
Returns the thread-local system errno.
RD_EXPORT rd_kafka_topic_conf_t * rd_kafka_default_topic_conf_dup(rd_kafka_t *rk)
Creates a copy/duplicate of rk 's default topic configuration object.
RD_EXPORT rd_kafka_DeleteConsumerGroupOffsets_t * rd_kafka_DeleteConsumerGroupOffsets_new(const char *group, const rd_kafka_topic_partition_list_t *partitions)
Create a new DeleteConsumerGroupOffsets object. This object is later passed to rd_kafka_DeleteConsume...
RD_EXPORT rd_kafka_error_t * rd_kafka_sasl_set_credentials(rd_kafka_t *rk, const char *username, const char *password)
Sets SASL credentials used for SASL PLAIN and SCRAM mechanisms by this Kafka client.
RD_EXPORT const char * rd_kafka_AclBinding_host(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_dup(const rd_kafka_conf_t *conf)
Creates a copy/duplicate of configuration object conf.
RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic)
Destroy and free a DeleteTopic object previously created with rd_kafka_DeleteTopic_new()
RD_EXPORT const char * rd_kafka_err2name(rd_kafka_resp_err_t err)
Returns the error code name (enum name).
rd_kafka_consumer_group_state_t
Consumer group state.
Definition: rdkafka.h:5012
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:6276
RD_EXPORT const rd_kafka_IncrementalAlterConfigs_result_t * rd_kafka_event_IncrementalAlterConfigs_result(rd_kafka_event_t *rkev)
Get IncrementalAlterConfigs result.
RD_EXPORT int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, rd_kafka_queue_t *rkqu)
Same as rd_kafka_consume_start() but re-routes incoming messages to the provided queue rkqu (which mu...
int rd_kafka_event_type_t
Event types.
Definition: rdkafka.h:5333
RD_EXPORT size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu)
struct rd_kafka_ConsumerGroupListing_s rd_kafka_ConsumerGroupListing_t
ListConsumerGroups result for a single group.
Definition: rdkafka.h:7903
RD_EXPORT void rd_kafka_conf_dump_free(const char **arr, size_t cnt)
Frees a configuration dump returned from rd_kafka_conf_dump() or `rd_kafka_topic_conf_dump().
RD_EXPORT rd_kafka_message_t * rd_kafka_consume_queue(rd_kafka_queue_t *rkqu, int timeout_ms)
Consume from queue.
RD_EXPORT rd_kafka_error_t * rd_kafka_AdminOptions_set_match_consumer_group_states(rd_kafka_AdminOptions_t *options, const rd_kafka_consumer_group_state_t *consumer_group_states, size_t consumer_group_states_cnt)
Set consumer groups states to query for.
struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t
Definition: rdkafka.h:7378
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_destroy(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_destroy_t *on_conf_destroy, void *ic_opaque)
Append an on_conf_destroy() interceptor.
RD_EXPORT void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt)
Loose application's topic handle refcount as previously created with rd_kafka_topic_new().
RD_EXPORT void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu)
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets_destroy_array(rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffset_cnt)
Helper function to destroy all ListConsumerGroupOffsets objects in the list_grpoffsets array (of list...
struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t
AdminOptions provides a generic mechanism for setting optional parameters for the Admin API requests.
Definition: rdkafka.h:6796
RD_EXPORT int rd_kafka_topic_partition_list_del(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Delete partition from list.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres)
Topic result provides per-topic operation result information.
RD_EXPORT int rd_kafka_error_is_fatal(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_conf_interceptor_add_on_conf_set(rd_kafka_conf_t *conf, const char *ic_name, rd_kafka_interceptor_f_on_conf_set_t *on_conf_set, void *ic_opaque)
Append an on_conf_set() interceptor.
RD_EXPORT const rd_kafka_error_t * rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres)
Group result provides per-group operation result information.
RD_EXPORT const rd_kafka_AlterConfigs_result_t * rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev)
Get AlterConfigs result.
RD_EXPORT void rd_kafka_conf_set_oauthbearer_token_refresh_cb(rd_kafka_conf_t *conf, void(*oauthbearer_token_refresh_cb)(rd_kafka_t *rk, const char *oauthbearer_config, void *opaque))
Set SASL/OAUTHBEARER token refresh callback in provided conf object.
RD_EXPORT void rd_kafka_AlterConsumerGroupOffsets_destroy(rd_kafka_AlterConsumerGroupOffsets_t *alter_grpoffsets)
Destroy and free a AlterConsumerGroupOffsets object previously created with rd_kafka_AlterConsumerGro...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_request_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the overall request timeout, including broker lookup, request transmission, operation time on br...
RD_EXPORT void rd_kafka_message_set_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t *hdrs)
Replace the message's current headers with a new list.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_message(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, int async)
Commit message's offset on broker for the message's partition. The committed offset is the message's ...
rd_kafka_vtype_t
Var-arg tag types.
Definition: rdkafka.h:1116
@ RD_KAFKA_VTYPE_PARTITION
Definition: rdkafka.h:1120
@ RD_KAFKA_VTYPE_HEADERS
Definition: rdkafka.h:1132
@ RD_KAFKA_VTYPE_TOPIC
Definition: rdkafka.h:1118
@ RD_KAFKA_VTYPE_MSGFLAGS
Definition: rdkafka.h:1128
@ RD_KAFKA_VTYPE_RKT
Definition: rdkafka.h:1119
@ RD_KAFKA_VTYPE_HEADER
Definition: rdkafka.h:1130
@ RD_KAFKA_VTYPE_KEY
Definition: rdkafka.h:1122
@ RD_KAFKA_VTYPE_END
Definition: rdkafka.h:1117
@ RD_KAFKA_VTYPE_OPAQUE
Definition: rdkafka.h:1123
@ RD_KAFKA_VTYPE_VALUE
Definition: rdkafka.h:1121
@ RD_KAFKA_VTYPE_TIMESTAMP
Definition: rdkafka.h:1129
struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t
Definition: rdkafka.h:7503
RD_EXPORT const char * rd_kafka_name(const rd_kafka_t *rk)
Returns Kafka handle name.
RD_EXPORT const rd_kafka_DeleteAcls_result_t * rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev)
struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t
Definition: rdkafka.h:8263
RD_EXPORT rd_kafka_resp_err_t rd_kafka_poll_set_consumer(rd_kafka_t *rk)
Redirect the main (rd_kafka_poll()) queue to the KafkaConsumer's queue (rd_kafka_consumer_poll()).
RD_EXPORT const rd_kafka_MemberAssignment_t * rd_kafka_MemberDescription_assignment(const rd_kafka_MemberDescription_t *member)
Gets assignment of member.
RD_EXPORT void rd_kafka_AdminOptions_set_opaque(rd_kafka_AdminOptions_t *options, void *ev_opaque)
Set application opaque value that can be extracted from the result event using rd_kafka_event_opaque(...
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_close_queue(rd_kafka_t *rk, rd_kafka_queue_t *rkqu)
Asynchronously close the consumer.
RD_EXPORT const char * rd_kafka_AclPermissionType_name(rd_kafka_AclPermissionType_t acl_permission_type)
RD_EXPORT RD_DEPRECATED void rd_kafka_set_logger(rd_kafka_t *rk, void(*func)(const rd_kafka_t *rk, int level, const char *fac, const char *buf))
Set logger function.
RD_EXPORT rd_kafka_NewTopic_t * rd_kafka_NewTopic_new(const char *topic, int num_partitions, int replication_factor, char *errstr, size_t errstr_size)
Create a new NewTopic object. This object is later passed to rd_kafka_CreateTopics().
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_new(size_t initial_count)
Create a new headers list.
RD_EXPORT rd_kafka_AlterConsumerGroupOffsets_t * rd_kafka_AlterConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new AlterConsumerGroupOffsets object. This object is later passed to rd_kafka_AlterConsumerG...
RD_EXPORT int rd_kafka_ConsumerGroupDescription_is_simple_consumer_group(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Is the grpdesc group a simple consumer group.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev)
RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs)
Destroy the headers list. The object and any returned value pointers are not usable after this call.
RD_EXPORT int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage)
Returns the broker id of the broker the message was produced to or fetched from.
rd_kafka_event_t rd_kafka_AlterConsumerGroupOffsets_result_t
Definition: rdkafka.h:5638
RD_EXPORT void rd_kafka_ConfigResource_destroy_array(rd_kafka_ConfigResource_t **config, size_t config_cnt)
Helper function to destroy all ConfigResource objects in the configs array (of config_cnt elements)....
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_start(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_start_t *on_thread_start, void *ic_opaque)
Append an on_thread_start() interceptor.
RD_EXPORT rd_kafka_message_t * rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms)
Consume a single message from topic rkt and partition.
RD_EXPORT size_t rd_kafka_ConsumerGroupDescription_member_count(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets the members count of grpdesc group.
RD_EXPORT void rd_kafka_conf_properties_show(FILE *fp)
Prints a table to fp of all supported configuration properties, their default values as well as a des...
RD_EXPORT const rd_kafka_AclBinding_t ** rd_kafka_DescribeAcls_result_acls(const rd_kafka_DescribeAcls_result_t *result, size_t *cntp)
Get an array of resource results from a DescribeAcls result.
RD_EXPORT const rd_kafka_CreateAcls_result_t * rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_acknowledgement_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_acknowledgement() is called to inform interceptors that a message was succesfully delivered or per...
Definition: rdkafka.h:6163
struct rd_kafka_DeleteConsumerGroupOffsets_s rd_kafka_DeleteConsumerGroupOffsets_t
Definition: rdkafka.h:8534
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_new(void)
Create configuration object.
RD_EXPORT int rd_kafka_ConfigEntry_is_read_only(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_thread_start_t(rd_kafka_t *rk, rd_kafka_thread_type_t thread_type, const char *thread_name, void *ic_opaque)
on_thread_start() is called from a newly created librdkafka-managed thread.
Definition: rdkafka.h:6306
RD_EXPORT int rd_kafka_wait_destroyed(int timeout_ms)
Wait for all rd_kafka_t objects to be destroyed.
RD_EXPORT const char * rd_kafka_consumer_group_state_name(rd_kafka_consumer_group_state_t state)
Returns a name for a state code.
rd_kafka_ConfigSource_t
Apache Kafka config sources.
Definition: rdkafka.h:7347
@ RD_KAFKA_CONFIG_SOURCE_UNKNOWN_CONFIG
Definition: rdkafka.h:7350
@ RD_KAFKA_CONFIG_SOURCE__CNT
Definition: rdkafka.h:7366
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_BROKER_CONFIG
Definition: rdkafka.h:7354
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_TOPIC_CONFIG
Definition: rdkafka.h:7352
@ RD_KAFKA_CONFIG_SOURCE_DEFAULT_CONFIG
Definition: rdkafka.h:7363
@ RD_KAFKA_CONFIG_SOURCE_STATIC_BROKER_CONFIG
Definition: rdkafka.h:7360
@ RD_KAFKA_CONFIG_SOURCE_DYNAMIC_DEFAULT_BROKER_CONFIG
Definition: rdkafka.h:7357
RD_EXPORT const rd_kafka_AlterConsumerGroupOffsets_result_t * rd_kafka_event_AlterConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get AlterConsumerGroupOffsets result.
RD_EXPORT int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent partitioner.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_headers(const rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list.
RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk, rd_kafka_DeleteRecords_t **del_records, size_t del_record_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete records (messages) in topic partitions older than the offsets provided.
RD_EXPORT rd_kafka_AclPermissionType_t rd_kafka_AclBinding_permission_type(const rd_kafka_AclBinding_t *acl)
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_destroy_t(rd_kafka_t *rk, void *ic_opaque)
on_destroy() is called from rd_kafka_destroy() or (rd_kafka_new() if rd_kafka_new() fails during init...
Definition: rdkafka.h:6112
RD_EXPORT rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms)
Wait until all outstanding produce requests, et.al, are completed. This should typically be done prio...
RD_EXPORT void rd_kafka_AlterUserScramCredentials(rd_kafka_t *rk, rd_kafka_UserScramCredentialAlteration_t **alterations, size_t alteration_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Alter SASL/SCRAM credentials. This operation is supported by brokers with version 2....
RD_EXPORT int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error)
RD_EXPORT rd_kafka_topic_t * rd_kafka_topic_new(rd_kafka_t *rk, const char *topic, rd_kafka_topic_conf_t *conf)
Creates a new topic handle for topic named topic.
RD_EXPORT void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist)
Release list memory.
rd_kafka_AclPermissionType_t
Apache Kafka ACL permission types.
Definition: rdkafka.h:8917
@ RD_KAFKA_ACL_PERMISSION_TYPE_DENY
Definition: rdkafka.h:8921
@ RD_KAFKA_ACL_PERMISSION_TYPE_ALLOW
Definition: rdkafka.h:8922
@ RD_KAFKA_ACL_PERMISSION_TYPE_UNKNOWN
Definition: rdkafka.h:8918
@ RD_KAFKA_ACL_PERMISSION_TYPE_ANY
Definition: rdkafka.h:8919
rd_kafka_thread_type_t
librdkafka internal thread type.
Definition: rdkafka.h:5270
@ RD_KAFKA_THREAD_MAIN
Definition: rdkafka.h:5271
@ RD_KAFKA_THREAD_BACKGROUND
Definition: rdkafka.h:5272
@ RD_KAFKA_THREAD_BROKER
Definition: rdkafka.h:5273
RD_EXPORT const rd_kafka_ScramCredentialInfo_t * rd_kafka_UserScramCredentialsDescription_scramcredentialinfo(const rd_kafka_UserScramCredentialsDescription_t *description, size_t idx)
Returns the ScramCredentialInfo at index idx of UserScramCredentialsDescription.
RD_EXPORT int rd_kafka_ConsumerGroupListing_is_simple_consumer_group(const rd_kafka_ConsumerGroupListing_t *grplist)
Is the grplist group a simple consumer group.
RD_EXPORT void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags)
Destroy Kafka handle according to specified destroy flags.
RD_EXPORT rd_kafka_AclOperation_t rd_kafka_AclBinding_operation(const rd_kafka_AclBinding_t *acl)
RD_EXPORT const rd_kafka_CreatePartitions_result_t * rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev)
Get CreatePartitions result.
RD_EXPORT void rd_kafka_ListConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_ListConsumerGroupOffsets_t **list_grpoffsets, size_t list_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List committed offsets for a set of partitions in a consumer group.
RD_EXPORT void rd_kafka_conf_set_closesocket_cb(rd_kafka_conf_t *conf, int(*closesocket_cb)(int sockfd, void *opaque))
Set close socket callback.
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random Murmur2 partitioner (Java compatible).
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics)
Returns the current topic subscription.
RD_EXPORT rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk)
Returns Kafka handle type.
RD_EXPORT void rd_kafka_topic_partition_set_leader_epoch(rd_kafka_topic_partition_t *rktpar, int32_t leader_epoch)
Sets the offset leader epoch (use -1 to clear).
rd_kafka_cert_enc_t
SSL certificate encoding.
Definition: rdkafka.h:2397
@ RD_KAFKA_CERT_ENC_PEM
Definition: rdkafka.h:2400
@ RD_KAFKA_CERT_ENC_PKCS12
Definition: rdkafka.h:2398
@ RD_KAFKA_CERT_ENC_DER
Definition: rdkafka.h:2399
RD_EXPORT const char * rd_kafka_err2str(rd_kafka_resp_err_t err)
Returns a human readable representation of a kafka error.
RD_EXPORT int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist)
Adds one or more brokers to the kafka handle's list of initial bootstrap brokers.
RD_EXPORT int rd_kafka_unittest(void)
Run librdkafka's built-in unit-tests.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, int async)
Commit offsets on broker for the provided list of partitions.
RD_EXPORT int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage)
Returns the latency for a produced message measured from the produce() call.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_acknowledgement(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement, void *ic_opaque)
Append an on_acknowledgement() interceptor.
rd_kafka_event_t rd_kafka_DeleteGroups_result_t
Definition: rdkafka.h:5634
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a configuration property.
RD_EXPORT const rd_kafka_AlterUserScramCredentials_result_t * rd_kafka_event_AlterUserScramCredentials_result(rd_kafka_event_t *rkev)
Get AlterUserScramCredentials result.
rd_kafka_event_t rd_kafka_DescribeConfigs_result_t
Definition: rdkafka.h:5626
RD_EXPORT uint16_t rd_kafka_Node_port(const rd_kafka_Node_t *node)
Get the port of node.
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev)
RD_EXPORT int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms)
Returns the current ControllerId as reported in broker metadata.
RD_EXPORT int rd_kafka_ConfigEntry_is_default(const rd_kafka_ConfigEntry_t *entry)
RD_EXPORT rd_kafka_ScramMechanism_t rd_kafka_ScramCredentialInfo_mechanism(const rd_kafka_ScramCredentialInfo_t *scram_credential_info)
Returns the mechanism of a given ScramCredentialInfo.
RD_EXPORT rd_kafka_NewPartitions_t * rd_kafka_NewPartitions_new(const char *topic, size_t new_total_cnt, char *errstr, size_t errstr_size)
Create a new NewPartitions. This object is later passed to rd_kafka_CreatePartitions() to increase th...
RD_EXPORT void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events)
Enable event sourcing. events is a bitmask of RD_KAFKA_EVENT_* of events to enable for consumption by...
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf, const char *name, char *dest, size_t *dest_size)
Retrieve configuration value for property name.
RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox)
Converts the system errno value errnox to a rd_kafka_resp_err_t error code upon failure from the foll...
RD_EXPORT const rd_kafka_AlterUserScramCredentials_result_response_t ** rd_kafka_AlterUserScramCredentials_result_responses(const rd_kafka_AlterUserScramCredentials_result_t *result, size_t *cntp)
Get an array of responses from a AlterUserScramCredentials result.
RD_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
FNV-1a partitioner.
RD_EXPORT rd_kafka_error_t * rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms)
Initialize transactions for the producer instance.
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_event_topic_partition(rd_kafka_event_t *rkev)
rd_kafka_event_t rd_kafka_DeleteConsumerGroupOffsets_result_t
Definition: rdkafka.h:5636
RD_EXPORT void rd_kafka_topic_conf_set_partitioner_cb(rd_kafka_topic_conf_t *topic_conf, int32_t(*partitioner)(const rd_kafka_topic_t *rkt, const void *keydata, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque))
Producer: Set partitioner callback in provided topic conf object.
RD_EXPORT const rd_kafka_acl_result_t ** rd_kafka_CreateAcls_result_acls(const rd_kafka_CreateAcls_result_t *result, size_t *cntp)
Get an array of acl results from a CreateAcls result.
RD_EXPORT const char * rd_kafka_ResourceType_name(rd_kafka_ResourceType_t restype)
RD_EXPORT void rd_kafka_destroy(rd_kafka_t *rk)
Destroy Kafka handle.
RD_EXPORT int rd_kafka_Node_id(const rd_kafka_Node_t *node)
Get the id of node.
RD_EXPORT void rd_kafka_conf_set_dr_msg_cb(rd_kafka_conf_t *conf, void(*dr_msg_cb)(rd_kafka_t *rk, const rd_kafka_message_t *rkmessage, void *opaque))
Producer: Set delivery report callback in provided conf object.
RD_EXPORT const rd_kafka_ListConsumerGroupOffsets_result_t * rd_kafka_event_ListConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get ListConsumerGroupOffsets result.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk,...)
Produce and send a single message to broker.
RD_EXPORT void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf)
Destroys a topic conf object.
RD_EXPORT const char * rd_kafka_MemberDescription_group_instance_id(const rd_kafka_MemberDescription_t *member)
Gets group instance id of member.
RD_EXPORT void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar)
Destroy a rd_kafka_topic_partition_t.
RD_EXPORT int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random FNV-1a partitioner.
RD_EXPORT void rd_kafka_DescribeConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Get configuration for the specified resources in configs.
rd_kafka_type_t
rd_kafka_t handle type.
Definition: rdkafka.h:209
@ RD_KAFKA_CONSUMER
Definition: rdkafka.h:211
@ RD_KAFKA_PRODUCER
Definition: rdkafka.h:210
RD_EXPORT const rd_kafka_topic_partition_list_t * rd_kafka_DeleteRecords_result_offsets(const rd_kafka_DeleteRecords_result_t *result)
Get a list of topic and partition results from a DeleteRecords result. The returned objects will cont...
RD_EXPORT rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a single rd_kafka_topic_conf_t value by property name.
RD_EXPORT void rd_kafka_DescribeUserScramCredentials(rd_kafka_t *rk, const char **users, size_t user_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Describe SASL/SCRAM credentials. This operation is supported by brokers with version 2....
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_consumer(rd_kafka_t *rk)
RD_EXPORT void rd_kafka_set_log_level(rd_kafka_t *rk, int level)
Specifies the maximum logging level emitted by internal kafka logging and debugging.
RD_EXPORT const char * rd_kafka_AclBinding_name(const rd_kafka_AclBinding_t *acl)
RD_EXPORT void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu, int fd, const void *payload, size_t size)
Enable IO event triggering for queue.
rd_kafka_ResourcePatternType_t
Apache Kafka pattern types.
Definition: rdkafka.h:7464
@ RD_KAFKA_RESOURCE_PATTERN_LITERAL
Definition: rdkafka.h:7472
@ RD_KAFKA_RESOURCE_PATTERN_ANY
Definition: rdkafka.h:7468
@ RD_KAFKA_RESOURCE_PATTERN_MATCH
Definition: rdkafka.h:7470
@ RD_KAFKA_RESOURCE_PATTERN_PREFIXED
Definition: rdkafka.h:7474
@ RD_KAFKA_RESOURCE_PATTERN_UNKNOWN
Definition: rdkafka.h:7466
RD_EXPORT rd_kafka_consumer_group_state_t rd_kafka_ConsumerGroupDescription_state(const rd_kafka_ConsumerGroupDescription_t *grpdesc)
Gets state for the grpdesc group.
RD_EXPORT int rd_kafka_error_is_retriable(const rd_kafka_error_t *error)
rd_kafka_ResourceType_t
Apache Kafka resource types.
Definition: rdkafka.h:7451
@ RD_KAFKA_RESOURCE_BROKER
Definition: rdkafka.h:7456
@ RD_KAFKA_RESOURCE__CNT
Definition: rdkafka.h:7457
@ RD_KAFKA_RESOURCE_TOPIC
Definition: rdkafka.h:7454
@ RD_KAFKA_RESOURCE_GROUP
Definition: rdkafka.h:7455
@ RD_KAFKA_RESOURCE_UNKNOWN
Definition: rdkafka.h:7452
@ RD_KAFKA_RESOURCE_ANY
Definition: rdkafka.h:7453
struct rd_kafka_ConsumerGroupDescription_s rd_kafka_ConsumerGroupDescription_t
DescribeConsumerGroups result type.
Definition: rdkafka.h:8007
RD_EXPORT void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf, void(*error_cb)(rd_kafka_t *rk, int err, const char *reason, void *opaque))
Set error callback in provided conf object.
RD_EXPORT int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition)
Stop consuming messages for topic rkt and partition, purging all messages currently in the local queu...
RD_EXPORT void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable)
Enable/disable creation of a queue specific to SASL events and callbacks.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_resume_partitions(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions)
Resume producing consumption for the provided list of partitions.
RD_EXPORT void rd_kafka_ConfigResource_destroy(rd_kafka_ConfigResource_t *config)
Destroy and free a ConfigResource object previously created with rd_kafka_ConfigResource_new()
RD_EXPORT void rd_kafka_DeleteTopic_destroy_array(rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt)
Helper function to destroy all DeleteTopic objects in the del_topics array (of del_topic_cnt elements...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk, const char *topic, int32_t partition, int64_t *low, int64_t *high)
Get last known low (oldest/beginning) and high (newest/end) offsets for partition.
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_IncrementalAlterConfigs_result_resources(const rd_kafka_IncrementalAlterConfigs_result_t *result, size_t *cntp)
Get an array of resource results from a IncrementalAlterConfigs result.
RD_EXPORT int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt, int32_t partition)
Check if partition is available (has a leader broker).
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_read(rd_kafka_consumer_group_metadata_t **cgmdp, const void *buffer, size_t size)
Reads serialized consumer group metadata and returns a consumer group metadata object....
RD_EXPORT void rd_kafka_ListConsumerGroups(rd_kafka_t *rk, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
List the consumer groups available in the cluster.
RD_EXPORT const rd_kafka_DeleteGroups_result_t * rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev)
Get DeleteGroups result.
rd_kafka_msg_status_t
Message persistence status can be used by the application to find out if a produced message was persi...
Definition: rdkafka.h:1592
@ RD_KAFKA_MSG_STATUS_POSSIBLY_PERSISTED
Definition: rdkafka.h:1601
@ RD_KAFKA_MSG_STATUS_PERSISTED
Definition: rdkafka.h:1606
@ RD_KAFKA_MSG_STATUS_NOT_PERSISTED
Definition: rdkafka.h:1596
RD_EXPORT int rd_kafka_outq_len(rd_kafka_t *rk)
Returns the current out queue length.
RD_EXPORT int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms)
Polls the provided kafka handle for events.
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_partition(rd_kafka_t *rk, const char *topic, int32_t partition)
RD_EXPORT const char * rd_kafka_ConfigEntry_name(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_event_t rd_kafka_AlterConfigs_result_t
Definition: rdkafka.h:5622
RD_EXPORT const char * rd_kafka_event_error_string(rd_kafka_event_t *rkev)
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1646
@ RD_KAFKA_CONF_OK
Definition: rdkafka.h:1651
@ RD_KAFKA_CONF_INVALID
Definition: rdkafka.h:1648
@ RD_KAFKA_CONF_UNKNOWN
Definition: rdkafka.h:1647
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs, const char *name, ssize_t name_size, const void *value, ssize_t value_size)
Add header with name name and value val (copied) of size size (not including null-terminator).
RD_EXPORT int32_t rd_kafka_topic_partition_get_leader_epoch(const rd_kafka_topic_partition_t *rktpar)
RD_EXPORT const char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
RD_EXPORT const rd_kafka_DeleteRecords_result_t * rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_AclBinding_principal(const rd_kafka_AclBinding_t *acl)
RD_EXPORT int32_t rd_kafka_message_leader_epoch(const rd_kafka_message_t *rkmessage)
RD_EXPORT const rd_kafka_DescribeAcls_result_t * rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_DeleteRecords_t * rd_kafka_DeleteRecords_new(const rd_kafka_topic_partition_list_t *before_offsets)
Create a new DeleteRecords object. This object is later passed to rd_kafka_DeleteRecords().
RD_EXPORT void rd_kafka_NewPartitions_destroy(rd_kafka_NewPartitions_t *new_parts)
Destroy and free a NewPartitions object previously created with rd_kafka_NewPartitions_new()
RD_EXPORT rd_kafka_resp_err_t rd_kafka_NewTopic_set_replica_assignment(rd_kafka_NewTopic_t *new_topic, int32_t partition, int32_t *broker_ids, size_t broker_id_cnt, char *errstr, size_t errstr_size)
Set the replica (broker) assignment for partition to the replica set in broker_ids (of broker_id_cnt ...
RD_EXPORT void rd_kafka_AlterConfigs(rd_kafka_t *rk, rd_kafka_ConfigResource_t **configs, size_t config_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Update the configuration for the specified resources. Updates are not transactional so they may succe...
RD_EXPORT const char * rd_kafka_get_debug_contexts(void)
Retrieve supported debug contexts for use with the "debug" configuration property....
RD_EXPORT rd_kafka_ConfigResource_t * rd_kafka_ConfigResource_new(rd_kafka_ResourceType_t restype, const char *resname)
Create new ConfigResource object.
RD_EXPORT void rd_kafka_AdminOptions_destroy(rd_kafka_AdminOptions_t *options)
Destroy a AdminOptions object.
RD_EXPORT rd_kafka_conf_res_t rd_kafka_conf_set_ssl_cert_verify_cb(rd_kafka_conf_t *conf, int(*ssl_cert_verify_cb)(rd_kafka_t *rk, const char *broker_name, int32_t broker_id, int *x509_error, int depth, const char *buf, size_t size, char *errstr, size_t errstr_size, void *opaque))
Sets the verification callback of the broker certificate.
RD_EXPORT rd_kafka_UserScramCredentialAlteration_t * rd_kafka_UserScramCredentialUpsertion_new(const char *username, rd_kafka_ScramMechanism_t mechanism, int32_t iterations, const unsigned char *password, size_t password_size, const unsigned char *salt, size_t salt_size)
Allocates a new UserScramCredentialUpsertion given its fields. If salt isn't given a 64 B salt is gen...
RD_EXPORT int rd_kafka_consume_start(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset)
Start consuming messages for topic rkt and partition at offset offset which may either be an absolute...
RD_EXPORT int rd_kafka_produce(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, void *payload, size_t len, const void *key, size_t keylen, void *msg_opaque)
Produce and send a single message to broker.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_offsets_for_times(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *offsets, int timeout_ms)
Look up the offsets for the given partitions by timestamp.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_consume_t(rd_kafka_t *rk, rd_kafka_message_t *rkmessage, void *ic_opaque)
on_consume() is called just prior to passing the message to the application in rd_kafka_consumer_poll...
Definition: rdkafka.h:6185
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_broker(rd_kafka_AdminOptions_t *options, int32_t broker_id, char *errstr, size_t errstr_size)
Override what broker the Admin request will be sent to.
RD_EXPORT void rd_kafka_DeleteTopics(rd_kafka_t *rk, rd_kafka_DeleteTopic_t **del_topics, size_t del_topic_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete topics from cluster as specified by the topics array of size topic_cnt elements.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_last_error(void)
Returns the last error code generated by a legacy API call in the current thread.
RD_EXPORT const char * rd_kafka_UserScramCredentialsDescription_user(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the username of a UserScramCredentialsDescription.
RD_EXPORT const rd_kafka_error_t * rd_kafka_UserScramCredentialsDescription_error(const rd_kafka_UserScramCredentialsDescription_t *description)
Returns the error associated with a UserScramCredentialsDescription.
RD_EXPORT const char * rd_kafka_event_stats(rd_kafka_event_t *rkev)
Extract stats from the event.
struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t
ACL Binding is used to create access control lists.
Definition: rdkafka.h:8866
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:6332
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_validate_only(rd_kafka_AdminOptions_t *options, int true_or_false, char *errstr, size_t errstr_size)
Tell broker to only validate the request, without performing the requested operation (create topics,...
RD_EXPORT rd_kafka_error_t * rd_kafka_ConfigResource_add_incremental_config(rd_kafka_ConfigResource_t *config, const char *name, rd_kafka_AlterConfigOpType_t op_type, const char *value)
Add the value of the configuration entry for a subsequent incremental alter config operation....
RD_EXPORT const rd_kafka_ListConsumerGroups_result_t * rd_kafka_event_ListConsumerGroups_result(rd_kafka_event_t *rkev)
Get ListConsumerGroups result.
rd_kafka_event_t rd_kafka_IncrementalAlterConfigs_result_t
Definition: rdkafka.h:5624
RD_EXPORT const char * rd_kafka_error_string(const rd_kafka_error_t *error)
RD_EXPORT const rd_kafka_error_t * rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres)
RD_EXPORT int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random partitioner.
RD_EXPORT rd_kafka_error_t * rd_kafka_consumer_group_metadata_write(const rd_kafka_consumer_group_metadata_t *cgmd, void **bufferp, size_t *sizep)
Serialize the consumer group metadata to a binary format. This is mainly for client binding use and n...
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_DeleteTopics_result_topics(const rd_kafka_DeleteTopics_result_t *result, size_t *cntp)
Get an array of topic results from a DeleteTopics result.
RD_EXPORT void rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *)
Frees the consumer group metadata object as returned by rd_kafka_consumer_group_metadata().
struct rd_kafka_ListConsumerGroupOffsets_s rd_kafka_ListConsumerGroupOffsets_t
Definition: rdkafka.h:8343
RD_EXPORT void * rd_kafka_opaque(const rd_kafka_t *rk)
Retrieves the opaque pointer previously set with rd_kafka_conf_set_opaque()
RD_EXPORT void rd_kafka_event_destroy(rd_kafka_event_t *rkev)
Destroy an event.
RD_EXPORT const rd_kafka_DeleteAcls_result_response_t ** rd_kafka_DeleteAcls_result_responses(const rd_kafka_DeleteAcls_result_t *result, size_t *cntp)
Get an array of DeleteAcls result responses from a DeleteAcls result.
struct rd_kafka_MemberDescription_s rd_kafka_MemberDescription_t
Member description included in ConsumerGroupDescription.
Definition: rdkafka.h:8014
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_event_t rd_kafka_DescribeUserScramCredentials_result_t
Definition: rdkafka.h:5642
RD_EXPORT void rd_kafka_AclBinding_destroy_array(rd_kafka_AclBinding_t **acl_bindings, size_t acl_bindings_cnt)
Helper function to destroy all AclBinding objects in the acl_bindings array (of acl_bindings_cnt elem...
RD_EXPORT void rd_kafka_conf_destroy(rd_kafka_conf_t *conf)
Destroys a conf object.
RD_EXPORT void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu)
Cancels the current rd_kafka_queue_poll() on rkqu.
RD_EXPORT void rd_kafka_UserScramCredentialAlteration_destroy(rd_kafka_UserScramCredentialAlteration_t *alteration)
Destroys a UserScramCredentialAlteration given its pointer.
RD_EXPORT const char * rd_kafka_MemberDescription_client_id(const rd_kafka_MemberDescription_t *member)
Gets client id of member.
RD_EXPORT void rd_kafka_topic_partition_list_sort(rd_kafka_topic_partition_list_t *rktparlist, int(*cmp)(const void *a, const void *b, void *cmp_opaque), void *cmp_opaque)
Sort list using comparator cmp.
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_AclBinding_restype(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_send(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_send_t *on_send, void *ic_opaque)
Append an on_send() interceptor.
RD_EXPORT const char * rd_kafka_ConfigResource_name(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit_queue(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, rd_kafka_queue_t *rkqu, void(*cb)(rd_kafka_t *rk, rd_kafka_resp_err_t err, rd_kafka_topic_partition_list_t *offsets, void *commit_opaque), void *commit_opaque)
Commit offsets on broker for the provided list of partitions.
rd_kafka_timestamp_type_t
Definition: rdkafka.h:220
@ RD_KAFKA_TIMESTAMP_NOT_AVAILABLE
Definition: rdkafka.h:221
@ RD_KAFKA_TIMESTAMP_CREATE_TIME
Definition: rdkafka.h:222
@ RD_KAFKA_TIMESTAMP_LOG_APPEND_TIME
Definition: rdkafka.h:223
RD_EXPORT rd_kafka_resp_err_t rd_kafka_message_detach_headers(rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list and detach the list from the message making the application the owner of ...
RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs)
Returns the number of header key/value pairs.
RD_EXPORT void rd_kafka_DeleteGroups(rd_kafka_t *rk, rd_kafka_DeleteGroup_t **del_groups, size_t del_group_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete groups from cluster as specified by the del_groups array of size del_group_cnt elements.
RD_EXPORT rd_kafka_ListConsumerGroupOffsets_t * rd_kafka_ListConsumerGroupOffsets_new(const char *group_id, const rd_kafka_topic_partition_list_t *partitions)
Create a new ListConsumerGroupOffsets object. This object is later passed to rd_kafka_ListConsumerGro...
rd_kafka_AclBinding_t rd_kafka_AclBindingFilter_t
ACL Binding filter is used to filter access control lists.
Definition: rdkafka.h:8872
RD_EXPORT void rd_kafka_DeleteConsumerGroupOffsets(rd_kafka_t *rk, rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets, size_t del_grpoffsets_cnt, const rd_kafka_AdminOptions_t *options, rd_kafka_queue_t *rkqu)
Delete committed offsets for a set of partitions in a consumer group. This will succeed at the partit...
RD_EXPORT rd_kafka_ResourceType_t rd_kafka_ConfigResource_type(const rd_kafka_ConfigResource_t *config)
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_topic_partition_list_new(int size)
Create a new list/vector Topic+Partition container.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_set_config(rd_kafka_ConfigResource_t *config, const char *name, const char *value)
Set configuration name value pair.
rd_kafka_ScramMechanism_t
Apache Kafka ScramMechanism values.
Definition: rdkafka.h:8627
RD_EXPORT rd_kafka_error_t * rd_kafka_error_new(rd_kafka_resp_err_t code, const char *fmt,...) RD_FORMAT(printf
Create a new error object with error code and optional human readable error string in fmt.
rd_kafka_resp_err_t() rd_kafka_interceptor_f_on_conf_destroy_t(void *ic_opaque)
on_conf_destroy() is called from rd_kafka_*_conf_destroy() in the order the interceptors were added.
Definition: rdkafka.h:6075
struct rd_kafka_MemberAssignment_s rd_kafka_MemberAssignment_t
Member assignment included in MemberDescription.
Definition: rdkafka.h:8020
RD_EXPORT const char * rd_kafka_AclOperation_name(rd_kafka_AclOperation_t acl_operation)
RD_EXPORT rd_kafka_ConfigSource_t rd_kafka_ConfigEntry_source(const rd_kafka_ConfigEntry_t *entry)
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:643
const char * name
Definition: rdkafka.h:645
rd_kafka_resp_err_t code
Definition: rdkafka.h:644
const char * desc
Definition: rdkafka.h:646
Group information.
Definition: rdkafka.h:5025
struct rd_kafka_metadata_broker broker
Definition: rdkafka.h:5026
int member_cnt
Definition: rdkafka.h:5033
char * state
Definition: rdkafka.h:5029
char * group
Definition: rdkafka.h:5027
struct rd_kafka_group_member_info * members
Definition: rdkafka.h:5032
char * protocol
Definition: rdkafka.h:5031
char * protocol_type
Definition: rdkafka.h:5030
rd_kafka_resp_err_t err
Definition: rdkafka.h:5028
List of groups.
Definition: rdkafka.h:5041
int group_cnt
Definition: rdkafka.h:5043
struct rd_kafka_group_info * groups
Definition: rdkafka.h:5042
Group member information.
Definition: rdkafka.h:4995
char * member_id
Definition: rdkafka.h:4996
int member_assignment_size
Definition: rdkafka.h:5004
int member_metadata_size
Definition: rdkafka.h:5001
void * member_metadata
Definition: rdkafka.h:4999
char * client_host
Definition: rdkafka.h:4998
void * member_assignment
Definition: rdkafka.h:5002
char * client_id
Definition: rdkafka.h:4997
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1438
size_t key_len
Definition: rdkafka.h:1451
size_t len
Definition: rdkafka.h:1446
void * _private
Definition: rdkafka.h:1461
void * key
Definition: rdkafka.h:1449
int64_t offset
Definition: rdkafka.h:1453
void * payload
Definition: rdkafka.h:1442
int32_t partition
Definition: rdkafka.h:1441
rd_kafka_topic_t * rkt
Definition: rdkafka.h:1440
rd_kafka_resp_err_t err
Definition: rdkafka.h:1439
Broker information.
Definition: rdkafka.h:4860
int32_t id
Definition: rdkafka.h:4861
int port
Definition: rdkafka.h:4863
char * host
Definition: rdkafka.h:4862
Partition information.
Definition: rdkafka.h:4869
int32_t leader
Definition: rdkafka.h:4872
int32_t * isrs
Definition: rdkafka.h:4876
int replica_cnt
Definition: rdkafka.h:4873
rd_kafka_resp_err_t err
Definition: rdkafka.h:4871
int isr_cnt
Definition: rdkafka.h:4875
int32_t id
Definition: rdkafka.h:4870
int32_t * replicas
Definition: rdkafka.h:4874
Metadata container.
Definition: rdkafka.h:4893
int32_t orig_broker_id
Definition: rdkafka.h:4900
char * orig_broker_name
Definition: rdkafka.h:4901
int broker_cnt
Definition: rdkafka.h:4894
struct rd_kafka_metadata_topic * topics
Definition: rdkafka.h:4898
int topic_cnt
Definition: rdkafka.h:4897
struct rd_kafka_metadata_broker * brokers
Definition: rdkafka.h:4895
Topic information.
Definition: rdkafka.h:4882
char * topic
Definition: rdkafka.h:4883
struct rd_kafka_metadata_partition * partitions
Definition: rdkafka.h:4885
rd_kafka_resp_err_t err
Definition: rdkafka.h:4886
int partition_cnt
Definition: rdkafka.h:4884
A growable list of Topic+Partitions.
Definition: rdkafka.h:949
int cnt
Definition: rdkafka.h:950
int size
Definition: rdkafka.h:951
rd_kafka_topic_partition_t * elems
Definition: rdkafka.h:952
Topic+Partition place holder.
Definition: rdkafka.h:898
int64_t offset
Definition: rdkafka.h:901
size_t metadata_size
Definition: rdkafka.h:903
void * opaque
Definition: rdkafka.h:904
char * topic
Definition: rdkafka.h:899
void * metadata
Definition: rdkafka.h:902
int32_t partition
Definition: rdkafka.h:900
rd_kafka_resp_err_t err
Definition: rdkafka.h:905
void * _private
Definition: rdkafka.h:906
VTYPE + argument container for use with rd_kafka_produce_va()
Definition: rdkafka.h:1142
rd_kafka_vtype_t vtype
Definition: rdkafka.h:1143