librdkafka
The Apache Kafka C/C++ client library
 All Data Structures Files Functions Variables Typedefs Enumerations Enumerator Macros Pages
rdkafka.h
Go to the documentation of this file.
1 /*
2  * librdkafka - Apache Kafka C library
3  *
4  * Copyright (c) 2012-2020 Magnus Edenhill
5  * All rights reserved.
6  *
7  * Redistribution and use in source and binary forms, with or without
8  * modification, are permitted provided that the following conditions are met:
9  *
10  * 1. Redistributions of source code must retain the above copyright notice,
11  * this list of conditions and the following disclaimer.
12  * 2. Redistributions in binary form must reproduce the above copyright notice,
13  * this list of conditions and the following disclaimer in the documentation
14  * and/or other materials provided with the distribution.
15  *
16  * THIS SOFTWARE IS PROVIDED BY THE COPYRIGHT HOLDERS AND CONTRIBUTORS "AS IS"
17  * AND ANY EXPRESS OR IMPLIED WARRANTIES, INCLUDING, BUT NOT LIMITED TO, THE
18  * IMPLIED WARRANTIES OF MERCHANTABILITY AND FITNESS FOR A PARTICULAR PURPOSE
19  * ARE DISCLAIMED. IN NO EVENT SHALL THE COPYRIGHT OWNER OR CONTRIBUTORS BE
20  * LIABLE FOR ANY DIRECT, INDIRECT, INCIDENTAL, SPECIAL, EXEMPLARY, OR
21  * CONSEQUENTIAL DAMAGES (INCLUDING, BUT NOT LIMITED TO, PROCUREMENT OF
22  * SUBSTITUTE GOODS OR SERVICES; LOSS OF USE, DATA, OR PROFITS; OR BUSINESS
23  * INTERRUPTION) HOWEVER CAUSED AND ON ANY THEORY OF LIABILITY, WHETHER IN
24  * CONTRACT, STRICT LIABILITY, OR TORT (INCLUDING NEGLIGENCE OR OTHERWISE)
25  * ARISING IN ANY WAY OUT OF THE USE OF THIS SOFTWARE, EVEN IF ADVISED OF THE
26  * POSSIBILITY OF SUCH DAMAGE.
27  */
28 
43 /* @cond NO_DOC */
44 #ifndef _RDKAFKA_H_
45 #define _RDKAFKA_H_
46 
47 #include <stdio.h>
48 #include <inttypes.h>
49 #include <sys/types.h>
50 
51 #ifdef __cplusplus
52 extern "C" {
53 #if 0
54 } /* Restore indent */
55 #endif
56 #endif
57 
58 #ifdef _WIN32
59 #include <basetsd.h>
60 #ifndef WIN32_MEAN_AND_LEAN
61 #define WIN32_MEAN_AND_LEAN
62 #endif
63 #include <winsock2.h> /* for sockaddr, .. */
64 #ifndef _SSIZE_T_DEFINED
65 #define _SSIZE_T_DEFINED
66 typedef SSIZE_T ssize_t;
67 #endif
68 #define RD_UNUSED
69 #define RD_INLINE __inline
70 #define RD_DEPRECATED __declspec(deprecated)
71 #define RD_FORMAT(...)
72 #undef RD_EXPORT
73 #ifdef LIBRDKAFKA_STATICLIB
74 #define RD_EXPORT
75 #else
76 #ifdef LIBRDKAFKA_EXPORTS
77 #define RD_EXPORT __declspec(dllexport)
78 #else
79 #define RD_EXPORT __declspec(dllimport)
80 #endif
81 #ifndef LIBRDKAFKA_TYPECHECKS
82 #define LIBRDKAFKA_TYPECHECKS 0
83 #endif
84 #endif
85 
86 #else
87 #include <sys/socket.h> /* for sockaddr, .. */
88 
89 #define RD_UNUSED __attribute__((unused))
90 #define RD_INLINE inline
91 #define RD_EXPORT
92 #define RD_DEPRECATED __attribute__((deprecated))
93 
94 #if defined(__clang__) || defined(__GNUC__) || defined(__GNUG__)
95 #define RD_FORMAT(...) __attribute__((format(__VA_ARGS__)))
96 #else
97 #define RD_FORMAT(...)
98 #endif
99 
100 #ifndef LIBRDKAFKA_TYPECHECKS
101 #define LIBRDKAFKA_TYPECHECKS 1
102 #endif
103 #endif
104 
105 
111 #if LIBRDKAFKA_TYPECHECKS
112 #define _LRK_TYPECHECK(RET, TYPE, ARG) \
113  ({ \
114  if (0) { \
115  TYPE __t RD_UNUSED = (ARG); \
116  } \
117  RET; \
118  })
119 
120 #define _LRK_TYPECHECK2(RET, TYPE, ARG, TYPE2, ARG2) \
121  ({ \
122  if (0) { \
123  TYPE __t RD_UNUSED = (ARG); \
124  TYPE2 __t2 RD_UNUSED = (ARG2); \
125  } \
126  RET; \
127  })
128 
129 #define _LRK_TYPECHECK3(RET, TYPE, ARG, TYPE2, ARG2, TYPE3, ARG3) \
130  ({ \
131  if (0) { \
132  TYPE __t RD_UNUSED = (ARG); \
133  TYPE2 __t2 RD_UNUSED = (ARG2); \
134  TYPE3 __t3 RD_UNUSED = (ARG3); \
135  } \
136  RET; \
137  })
138 #else
139 #define _LRK_TYPECHECK(RET, TYPE, ARG) (RET)
140 #define _LRK_TYPECHECK2(RET, TYPE, ARG, TYPE2, ARG2) (RET)
141 #define _LRK_TYPECHECK3(RET, TYPE, ARG, TYPE2, ARG2, TYPE3, ARG3) (RET)
142 #endif
143 
144 /* @endcond */
145 
146 
168 #define RD_KAFKA_VERSION 0x010900ff
169 
178 RD_EXPORT
179 int rd_kafka_version(void);
180 
186 RD_EXPORT
187 const char *rd_kafka_version_str(void);
188 
207 typedef enum rd_kafka_type_t {
211 
212 
223 
224 
225 
232 RD_EXPORT
233 const char *rd_kafka_get_debug_contexts(void);
234 
242 #define RD_KAFKA_DEBUG_CONTEXTS \
243  "all,generic,broker,topic,metadata,feature,queue,msg,protocol,cgrp," \
244  "security,fetch,interceptor,plugin,consumer,admin,eos,mock,assignor," \
245  "conf"
246 
247 
248 /* @cond NO_DOC */
249 /* Private types to provide ABI compatibility */
250 typedef struct rd_kafka_s rd_kafka_t;
251 typedef struct rd_kafka_topic_s rd_kafka_topic_t;
252 typedef struct rd_kafka_conf_s rd_kafka_conf_t;
253 typedef struct rd_kafka_topic_conf_s rd_kafka_topic_conf_t;
254 typedef struct rd_kafka_queue_s rd_kafka_queue_t;
255 typedef struct rd_kafka_op_s rd_kafka_event_t;
256 typedef struct rd_kafka_topic_result_s rd_kafka_topic_result_t;
257 typedef struct rd_kafka_consumer_group_metadata_s
258  rd_kafka_consumer_group_metadata_t;
259 typedef struct rd_kafka_error_s rd_kafka_error_t;
260 typedef struct rd_kafka_headers_s rd_kafka_headers_t;
261 typedef struct rd_kafka_group_result_s rd_kafka_group_result_t;
262 typedef struct rd_kafka_acl_result_s rd_kafka_acl_result_t;
263 /* @endcond */
264 
265 
278 typedef enum {
279  /* Internal errors to rdkafka: */
405 
408 
409  /* Kafka broker errors: */
443 #define RD_KAFKA_RESP_ERR_GROUP_LOAD_IN_PROGRESS \
444  RD_KAFKA_RESP_ERR_COORDINATOR_LOAD_IN_PROGRESS
445 
448 #define RD_KAFKA_RESP_ERR_GROUP_COORDINATOR_NOT_AVAILABLE \
449  RD_KAFKA_RESP_ERR_COORDINATOR_NOT_AVAILABLE
450 
453 #define RD_KAFKA_RESP_ERR_NOT_COORDINATOR_FOR_GROUP \
454  RD_KAFKA_RESP_ERR_NOT_COORDINATOR
455 
627 
628  RD_KAFKA_RESP_ERR_END_ALL,
630 
631 
639  const char *name;
640  const char *desc;
641 };
642 
643 
647 RD_EXPORT
648 void rd_kafka_get_err_descs(const struct rd_kafka_err_desc **errdescs,
649  size_t *cntp);
650 
651 
652 
658 RD_EXPORT
659 const char *rd_kafka_err2str(rd_kafka_resp_err_t err);
660 
661 
662 
668 RD_EXPORT
669 const char *rd_kafka_err2name(rd_kafka_resp_err_t err);
670 
671 
697 RD_EXPORT
699 
700 
725 RD_EXPORT RD_DEPRECATED rd_kafka_resp_err_t rd_kafka_errno2err(int errnox);
726 
727 
740 RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void);
741 
742 
743 
770 RD_EXPORT
772 rd_kafka_fatal_error(rd_kafka_t *rk, char *errstr, size_t errstr_size);
773 
774 
792 RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk,
794  const char *reason);
795 
796 
801 RD_EXPORT
802 rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error);
803 
812 RD_EXPORT
813 const char *rd_kafka_error_name(const rd_kafka_error_t *error);
814 
821 RD_EXPORT
822 const char *rd_kafka_error_string(const rd_kafka_error_t *error);
823 
824 
829 RD_EXPORT
830 int rd_kafka_error_is_fatal(const rd_kafka_error_t *error);
831 
832 
837 RD_EXPORT
838 int rd_kafka_error_is_retriable(const rd_kafka_error_t *error);
839 
840 
851 RD_EXPORT
852 int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error);
853 
859 RD_EXPORT
860 void rd_kafka_error_destroy(rd_kafka_error_t *error);
861 
862 
871 RD_EXPORT
872 rd_kafka_error_t *rd_kafka_error_new(rd_kafka_resp_err_t code,
873  const char *fmt,
874  ...) RD_FORMAT(printf, 2, 3);
875 
876 
892 typedef struct rd_kafka_topic_partition_s {
893  char *topic;
894  int32_t partition;
895  int64_t offset;
896  void *metadata;
897  size_t metadata_size;
898  void *opaque;
900  void *_private;
903 
904 
909 RD_EXPORT
911 
912 
917 typedef struct rd_kafka_topic_partition_list_s {
918  int cnt;
919  int size;
922 
923 
938 RD_EXPORT
940 
941 
945 RD_EXPORT
948 
958 RD_EXPORT
961  const char *topic,
962  int32_t partition);
963 
964 
973 RD_EXPORT
976  const char *topic,
977  int32_t start,
978  int32_t stop);
979 
980 
981 
993 RD_EXPORT
996  const char *topic,
997  int32_t partition);
998 
999 
1007 RD_EXPORT
1009  rd_kafka_topic_partition_list_t *rktparlist,
1010  int idx);
1011 
1012 
1020 RD_EXPORT
1023 
1024 
1025 
1033 RD_EXPORT
1035  rd_kafka_topic_partition_list_t *rktparlist,
1036  const char *topic,
1037  int32_t partition,
1038  int64_t offset);
1039 
1040 
1041 
1047 RD_EXPORT
1049  const rd_kafka_topic_partition_list_t *rktparlist,
1050  const char *topic,
1051  int32_t partition);
1052 
1053 
1063 RD_EXPORT void rd_kafka_topic_partition_list_sort(
1064  rd_kafka_topic_partition_list_t *rktparlist,
1065  int (*cmp)(const void *a, const void *b, void *cmp_opaque),
1066  void *cmp_opaque);
1067 
1068 
1086 typedef enum rd_kafka_vtype_t {
1104 
1105 
1112 typedef struct rd_kafka_vu_s {
1115  union {
1116  const char *cstr;
1117  rd_kafka_topic_t *rkt;
1118  int i;
1119  int32_t i32;
1120  int64_t i64;
1121  struct {
1122  void *ptr;
1123  size_t size;
1124  } mem;
1125  struct {
1126  const char *name;
1127  const void *val;
1128  ssize_t size;
1129  } header;
1130  rd_kafka_headers_t *headers;
1131  void *ptr;
1132  char _pad[64];
1133  } u;
1134 } rd_kafka_vu_t;
1135 
1144 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
1145 
1151 #define RD_KAFKA_V_TOPIC(topic) \
1152  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
1153  (const char *)topic
1154 
1159 #define RD_KAFKA_V_RKT(rkt) \
1160  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
1161  (rd_kafka_topic_t *)rkt
1162 
1167 #define RD_KAFKA_V_PARTITION(partition) \
1168  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
1169  (int32_t)partition
1170 
1175 #define RD_KAFKA_V_VALUE(VALUE, LEN) \
1176  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
1177  (void *)VALUE, (size_t)LEN
1178 
1183 #define RD_KAFKA_V_KEY(KEY, LEN) \
1184  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
1185  (void *)KEY, (size_t)LEN
1186 
1193 #define RD_KAFKA_V_OPAQUE(msg_opaque) \
1194  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, msg_opaque), \
1195  (void *)msg_opaque
1196 
1202 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
1203  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), (int)msgflags
1204 
1210 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
1211  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
1212  (int64_t)timestamp
1213 
1221 #define RD_KAFKA_V_HEADER(NAME, VALUE, LEN) \
1222  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
1223  const void *, VALUE, ssize_t, LEN), \
1224  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
1225 
1237 #define RD_KAFKA_V_HEADERS(HDRS) \
1238  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
1239  (rd_kafka_headers_t *)HDRS
1240 
1241 
1272 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new(size_t initial_count);
1273 
1278 RD_EXPORT void rd_kafka_headers_destroy(rd_kafka_headers_t *hdrs);
1279 
1283 RD_EXPORT rd_kafka_headers_t *
1284 rd_kafka_headers_copy(const rd_kafka_headers_t *src);
1285 
1303 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs,
1304  const char *name,
1305  ssize_t name_size,
1306  const void *value,
1307  ssize_t value_size);
1308 
1316 RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_remove(rd_kafka_headers_t *hdrs,
1317  const char *name);
1318 
1319 
1337 RD_EXPORT rd_kafka_resp_err_t
1338 rd_kafka_header_get_last(const rd_kafka_headers_t *hdrs,
1339  const char *name,
1340  const void **valuep,
1341  size_t *sizep);
1342 
1356 RD_EXPORT rd_kafka_resp_err_t
1357 rd_kafka_header_get(const rd_kafka_headers_t *hdrs,
1358  size_t idx,
1359  const char *name,
1360  const void **valuep,
1361  size_t *sizep);
1362 
1363 
1371 RD_EXPORT rd_kafka_resp_err_t
1372 rd_kafka_header_get_all(const rd_kafka_headers_t *hdrs,
1373  size_t idx,
1374  const char **namep,
1375  const void **valuep,
1376  size_t *sizep);
1377 
1378 
1379 
1392 // FIXME: This doesn't show up in docs for some reason
1393 // "Compound rd_kafka_message_t is not documented."
1394 
1408 typedef struct rd_kafka_message_s {
1410  rd_kafka_topic_t *rkt;
1411  int32_t partition;
1412  void *payload;
1416  size_t len;
1419  void *key;
1421  size_t key_len;
1423  int64_t offset;
1431  void *_private;
1438 
1439 
1443 RD_EXPORT
1445 
1446 
1447 
1454 RD_EXPORT
1455 const char *rd_kafka_message_errstr(const rd_kafka_message_t *rkmessage);
1456 
1457 
1469 RD_EXPORT
1470 int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage,
1471  rd_kafka_timestamp_type_t *tstype);
1472 
1473 
1474 
1481 RD_EXPORT
1482 int64_t rd_kafka_message_latency(const rd_kafka_message_t *rkmessage);
1483 
1484 
1491 RD_EXPORT
1492 int32_t rd_kafka_message_broker_id(const rd_kafka_message_t *rkmessage);
1493 
1494 
1511 RD_EXPORT rd_kafka_resp_err_t
1513  rd_kafka_headers_t **hdrsp);
1514 
1526 RD_EXPORT rd_kafka_resp_err_t
1528  rd_kafka_headers_t **hdrsp);
1529 
1530 
1543 RD_EXPORT
1545  rd_kafka_headers_t *hdrs);
1546 
1547 
1553 RD_EXPORT size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs);
1554 
1555 
1561 typedef enum {
1566 
1571 
1577 
1578 
1585 RD_EXPORT rd_kafka_msg_status_t
1587 
1603 typedef enum {
1610 
1611 
1645 RD_EXPORT
1646 rd_kafka_conf_t *rd_kafka_conf_new(void);
1647 
1648 
1652 RD_EXPORT
1653 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1654 
1655 
1662 RD_EXPORT
1663 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1664 
1665 
1670 RD_EXPORT
1671 rd_kafka_conf_t *rd_kafka_conf_dup_filter(const rd_kafka_conf_t *conf,
1672  size_t filter_cnt,
1673  const char **filter);
1674 
1675 
1676 
1685 RD_EXPORT
1686 const rd_kafka_conf_t *rd_kafka_conf(rd_kafka_t *rk);
1687 
1688 
1708 RD_EXPORT
1709 rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf,
1710  const char *name,
1711  const char *value,
1712  char *errstr,
1713  size_t errstr_size);
1714 
1715 
1721 RD_EXPORT
1722 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1723 
1724 
1761  rd_kafka_conf_t *conf,
1762  void (*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque));
1763 
1764 
1768 RD_EXPORT
1769 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1770  void (*dr_cb)(rd_kafka_t *rk,
1771  void *payload,
1772  size_t len,
1773  rd_kafka_resp_err_t err,
1774  void *opaque,
1775  void *msg_opaque));
1776 
1807 RD_EXPORT
1809  rd_kafka_conf_t *conf,
1810  void (*dr_msg_cb)(rd_kafka_t *rk,
1811  const rd_kafka_message_t *rkmessage,
1812  void *opaque));
1813 
1814 
1822 RD_EXPORT
1824  rd_kafka_conf_t *conf,
1825  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque));
1826 
1931 RD_EXPORT
1933  rd_kafka_conf_t *conf,
1934  void (*rebalance_cb)(rd_kafka_t *rk,
1935  rd_kafka_resp_err_t err,
1936  rd_kafka_topic_partition_list_t *partitions,
1937  void *opaque));
1938 
1939 
1940 
1958 RD_EXPORT
1960  rd_kafka_conf_t *conf,
1961  void (*offset_commit_cb)(rd_kafka_t *rk,
1962  rd_kafka_resp_err_t err,
1964  void *opaque));
1965 
1966 
1989 RD_EXPORT
1990 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
1991  void (*error_cb)(rd_kafka_t *rk,
1992  int err,
1993  const char *reason,
1994  void *opaque));
1995 
2013 RD_EXPORT
2014 void rd_kafka_conf_set_throttle_cb(rd_kafka_conf_t *conf,
2015  void (*throttle_cb)(rd_kafka_t *rk,
2016  const char *broker_name,
2017  int32_t broker_id,
2018  int throttle_time_ms,
2019  void *opaque));
2020 
2021 
2038 RD_EXPORT
2039 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
2040  void (*log_cb)(const rd_kafka_t *rk,
2041  int level,
2042  const char *fac,
2043  const char *buf));
2044 
2045 
2068 RD_EXPORT
2070  rd_kafka_conf_t *conf,
2071  int (*stats_cb)(rd_kafka_t *rk, char *json, size_t json_len, void *opaque));
2072 
2122 RD_EXPORT
2124  rd_kafka_conf_t *conf,
2125  void (*oauthbearer_token_refresh_cb)(rd_kafka_t *rk,
2126  const char *oauthbearer_config,
2127  void *opaque));
2128 
2156 RD_EXPORT
2157 void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable);
2158 
2159 
2177 RD_EXPORT
2179  rd_kafka_conf_t *conf,
2180  int (*socket_cb)(int domain, int type, int protocol, void *opaque));
2181 
2182 
2183 
2199 RD_EXPORT void
2200 rd_kafka_conf_set_connect_cb(rd_kafka_conf_t *conf,
2201  int (*connect_cb)(int sockfd,
2202  const struct sockaddr *addr,
2203  int addrlen,
2204  const char *id,
2205  void *opaque));
2206 
2217 RD_EXPORT void rd_kafka_conf_set_closesocket_cb(
2218  rd_kafka_conf_t *conf,
2219  int (*closesocket_cb)(int sockfd, void *opaque));
2220 
2221 
2222 
2223 #ifndef _WIN32
2224 
2241 RD_EXPORT
2243  rd_kafka_conf_t *conf,
2244  int (*open_cb)(const char *pathname, int flags, mode_t mode, void *opaque));
2245 #endif
2246 
2247 
2289 RD_EXPORT
2291  rd_kafka_conf_t *conf,
2292  int (*ssl_cert_verify_cb)(rd_kafka_t *rk,
2293  const char *broker_name,
2294  int32_t broker_id,
2295  int *x509_error,
2296  int depth,
2297  const char *buf,
2298  size_t size,
2299  char *errstr,
2300  size_t errstr_size,
2301  void *opaque));
2302 
2303 
2311 typedef enum rd_kafka_cert_type_t {
2315  RD_KAFKA_CERT__CNT,
2317 
2325 typedef enum rd_kafka_cert_enc_t {
2329  RD_KAFKA_CERT_ENC__CNT,
2331 
2332 
2367 RD_EXPORT rd_kafka_conf_res_t
2368 rd_kafka_conf_set_ssl_cert(rd_kafka_conf_t *conf,
2369  rd_kafka_cert_type_t cert_type,
2370  rd_kafka_cert_enc_t cert_enc,
2371  const void *buffer,
2372  size_t size,
2373  char *errstr,
2374  size_t errstr_size);
2375 
2376 
2391 RD_EXPORT
2392 void rd_kafka_conf_set_engine_callback_data(rd_kafka_conf_t *conf,
2393  void *callback_data);
2394 
2395 
2401 RD_EXPORT
2402 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
2403 
2408 RD_EXPORT
2409 void *rd_kafka_opaque(const rd_kafka_t *rk);
2410 
2411 
2412 
2426 RD_EXPORT
2427 void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf,
2428  rd_kafka_topic_conf_t *tconf);
2429 
2442 RD_EXPORT rd_kafka_topic_conf_t *
2443 rd_kafka_conf_get_default_topic_conf(rd_kafka_conf_t *conf);
2444 
2445 
2465 RD_EXPORT
2466 rd_kafka_conf_res_t rd_kafka_conf_get(const rd_kafka_conf_t *conf,
2467  const char *name,
2468  char *dest,
2469  size_t *dest_size);
2470 
2471 
2477 RD_EXPORT
2478 rd_kafka_conf_res_t rd_kafka_topic_conf_get(const rd_kafka_topic_conf_t *conf,
2479  const char *name,
2480  char *dest,
2481  size_t *dest_size);
2482 
2483 
2492 RD_EXPORT
2493 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
2494 
2495 
2504 RD_EXPORT
2505 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
2506  size_t *cntp);
2507 
2512 RD_EXPORT
2513 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
2514 
2522 RD_EXPORT
2523 void rd_kafka_conf_properties_show(FILE *fp);
2524 
2542 RD_EXPORT
2543 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
2544 
2545 
2549 RD_EXPORT
2550 rd_kafka_topic_conf_t *
2551 rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t *conf);
2552 
2557 RD_EXPORT
2558 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup(rd_kafka_t *rk);
2559 
2560 
2564 RD_EXPORT
2565 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
2566 
2567 
2576 RD_EXPORT
2577 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
2578  const char *name,
2579  const char *value,
2580  char *errstr,
2581  size_t errstr_size);
2582 
2589 RD_EXPORT
2590 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf,
2591  void *rkt_opaque);
2592 
2593 
2613 RD_EXPORT
2615  rd_kafka_topic_conf_t *topic_conf,
2616  int32_t (*partitioner)(const rd_kafka_topic_t *rkt,
2617  const void *keydata,
2618  size_t keylen,
2619  int32_t partition_cnt,
2620  void *rkt_opaque,
2621  void *msg_opaque));
2622 
2623 
2650  rd_kafka_topic_conf_t *topic_conf,
2651  int (*msg_order_cmp)(const rd_kafka_message_t *a,
2652  const rd_kafka_message_t *b));
2653 
2654 
2662 RD_EXPORT
2663 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
2664  int32_t partition);
2665 
2666 
2667 /*******************************************************************
2668  * *
2669  * Partitioners provided by rdkafka *
2670  * *
2671  *******************************************************************/
2672 
2686 RD_EXPORT
2687 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
2688  const void *key,
2689  size_t keylen,
2690  int32_t partition_cnt,
2691  void *rkt_opaque,
2692  void *msg_opaque);
2693 
2707 RD_EXPORT
2708 int32_t rd_kafka_msg_partitioner_consistent(const rd_kafka_topic_t *rkt,
2709  const void *key,
2710  size_t keylen,
2711  int32_t partition_cnt,
2712  void *rkt_opaque,
2713  void *msg_opaque);
2714 
2730 RD_EXPORT
2731 int32_t rd_kafka_msg_partitioner_consistent_random(const rd_kafka_topic_t *rkt,
2732  const void *key,
2733  size_t keylen,
2734  int32_t partition_cnt,
2735  void *rkt_opaque,
2736  void *msg_opaque);
2737 
2738 
2752 RD_EXPORT
2753 int32_t rd_kafka_msg_partitioner_murmur2(const rd_kafka_topic_t *rkt,
2754  const void *key,
2755  size_t keylen,
2756  int32_t partition_cnt,
2757  void *rkt_opaque,
2758  void *msg_opaque);
2759 
2774 RD_EXPORT
2775 int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt,
2776  const void *key,
2777  size_t keylen,
2778  int32_t partition_cnt,
2779  void *rkt_opaque,
2780  void *msg_opaque);
2781 
2782 
2796 RD_EXPORT
2797 int32_t rd_kafka_msg_partitioner_fnv1a(const rd_kafka_topic_t *rkt,
2798  const void *key,
2799  size_t keylen,
2800  int32_t partition_cnt,
2801  void *rkt_opaque,
2802  void *msg_opaque);
2803 
2804 
2819 RD_EXPORT
2820 int32_t rd_kafka_msg_partitioner_fnv1a_random(const rd_kafka_topic_t *rkt,
2821  const void *key,
2822  size_t keylen,
2823  int32_t partition_cnt,
2824  void *rkt_opaque,
2825  void *msg_opaque);
2826 
2827 
2867 RD_EXPORT
2868 rd_kafka_t *rd_kafka_new(rd_kafka_type_t type,
2869  rd_kafka_conf_t *conf,
2870  char *errstr,
2871  size_t errstr_size);
2872 
2873 
2888 RD_EXPORT
2889 void rd_kafka_destroy(rd_kafka_t *rk);
2890 
2891 
2896 RD_EXPORT
2897 void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags);
2898 
2913 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
2914 
2915 
2916 
2920 RD_EXPORT
2921 const char *rd_kafka_name(const rd_kafka_t *rk);
2922 
2923 
2927 RD_EXPORT
2928 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
2929 
2930 
2941 RD_EXPORT
2942 char *rd_kafka_memberid(const rd_kafka_t *rk);
2943 
2944 
2945 
2964 RD_EXPORT
2965 char *rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms);
2966 
2967 
2983 RD_EXPORT
2984 int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms);
2985 
2986 
3008 RD_EXPORT
3009 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk,
3010  const char *topic,
3011  rd_kafka_topic_conf_t *conf);
3012 
3013 
3014 
3023 RD_EXPORT
3024 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
3025 
3026 
3030 RD_EXPORT
3031 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
3032 
3033 
3038 RD_EXPORT
3039 void *rd_kafka_topic_opaque(const rd_kafka_topic_t *rkt);
3040 
3041 
3048 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
3049 
3050 
3078 RD_EXPORT
3079 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
3080 
3081 
3092 RD_EXPORT
3093 void rd_kafka_yield(rd_kafka_t *rk);
3094 
3095 
3096 
3104 RD_EXPORT rd_kafka_resp_err_t
3105 rd_kafka_pause_partitions(rd_kafka_t *rk,
3106  rd_kafka_topic_partition_list_t *partitions);
3107 
3108 
3109 
3117 RD_EXPORT rd_kafka_resp_err_t
3118 rd_kafka_resume_partitions(rd_kafka_t *rk,
3119  rd_kafka_topic_partition_list_t *partitions);
3120 
3121 
3122 
3131 RD_EXPORT rd_kafka_resp_err_t
3132 rd_kafka_query_watermark_offsets(rd_kafka_t *rk,
3133  const char *topic,
3134  int32_t partition,
3135  int64_t *low,
3136  int64_t *high,
3137  int timeout_ms);
3138 
3139 
3156 RD_EXPORT rd_kafka_resp_err_t rd_kafka_get_watermark_offsets(rd_kafka_t *rk,
3157  const char *topic,
3158  int32_t partition,
3159  int64_t *low,
3160  int64_t *high);
3161 
3162 
3163 
3189 RD_EXPORT rd_kafka_resp_err_t
3190 rd_kafka_offsets_for_times(rd_kafka_t *rk,
3192  int timeout_ms);
3193 
3194 
3195 
3209 RD_EXPORT
3210 void *rd_kafka_mem_calloc(rd_kafka_t *rk, size_t num, size_t size);
3211 
3212 
3213 
3227 RD_EXPORT
3228 void *rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size);
3229 
3230 
3231 
3248 RD_EXPORT
3249 void rd_kafka_mem_free(rd_kafka_t *rk, void *ptr);
3250 
3251 
3273 RD_EXPORT
3274 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
3275 
3279 RD_EXPORT
3280 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
3281 
3282 
3289 RD_EXPORT
3290 rd_kafka_queue_t *rd_kafka_queue_get_main(rd_kafka_t *rk);
3291 
3292 
3293 
3303 RD_EXPORT
3304 rd_kafka_queue_t *rd_kafka_queue_get_sasl(rd_kafka_t *rk);
3305 
3306 
3322 RD_EXPORT
3323 rd_kafka_error_t *rd_kafka_sasl_background_callbacks_enable(rd_kafka_t *rk);
3324 
3325 
3335 RD_EXPORT
3336 rd_kafka_queue_t *rd_kafka_queue_get_consumer(rd_kafka_t *rk);
3337 
3348 RD_EXPORT
3349 rd_kafka_queue_t *rd_kafka_queue_get_partition(rd_kafka_t *rk,
3350  const char *topic,
3351  int32_t partition);
3352 
3375 RD_EXPORT
3376 rd_kafka_queue_t *rd_kafka_queue_get_background(rd_kafka_t *rk);
3377 
3378 
3389 RD_EXPORT
3390 void rd_kafka_queue_forward(rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
3391 
3410 RD_EXPORT
3412  rd_kafka_queue_t *rkqu);
3413 
3414 
3418 RD_EXPORT
3419 size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu);
3420 
3421 
3439 RD_EXPORT
3440 void rd_kafka_queue_io_event_enable(rd_kafka_queue_t *rkqu,
3441  int fd,
3442  const void *payload,
3443  size_t size);
3444 
3461 RD_EXPORT
3462 void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu,
3463  void (*event_cb)(rd_kafka_t *rk,
3464  void *qev_opaque),
3465  void *qev_opaque);
3466 
3467 
3475 RD_EXPORT
3476 void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu);
3477 
3478 
3489 #define RD_KAFKA_OFFSET_BEGINNING \
3490  -2
3492 #define RD_KAFKA_OFFSET_END \
3493  -1
3495 #define RD_KAFKA_OFFSET_STORED \
3496  -1000
3498 #define RD_KAFKA_OFFSET_INVALID -1001
3502 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
3503 
3510 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
3511 
3545 RD_EXPORT
3546 int rd_kafka_consume_start(rd_kafka_topic_t *rkt,
3547  int32_t partition,
3548  int64_t offset);
3549 
3564 RD_EXPORT
3565 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt,
3566  int32_t partition,
3567  int64_t offset,
3568  rd_kafka_queue_t *rkqu);
3569 
3583 RD_EXPORT
3584 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
3585 
3586 
3587 
3616 RD_EXPORT
3617 rd_kafka_resp_err_t rd_kafka_seek(rd_kafka_topic_t *rkt,
3618  int32_t partition,
3619  int64_t offset,
3620  int timeout_ms);
3621 
3622 
3623 
3655 RD_EXPORT rd_kafka_error_t *
3656 rd_kafka_seek_partitions(rd_kafka_t *rk,
3657  rd_kafka_topic_partition_list_t *partitions,
3658  int timeout_ms);
3659 
3660 
3685 RD_EXPORT
3687 rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition, int timeout_ms);
3688 
3689 
3690 
3716 RD_EXPORT
3717 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt,
3718  int32_t partition,
3719  int timeout_ms,
3720  rd_kafka_message_t **rkmessages,
3721  size_t rkmessages_size);
3722 
3723 
3724 
3758 RD_EXPORT
3759 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt,
3760  int32_t partition,
3761  int timeout_ms,
3762  void (*consume_cb)(rd_kafka_message_t *rkmessage,
3763  void *commit_opaque),
3764  void *commit_opaque);
3765 
3766 
3783 RD_EXPORT
3784 rd_kafka_message_t *rd_kafka_consume_queue(rd_kafka_queue_t *rkqu,
3785  int timeout_ms);
3786 
3792 RD_EXPORT
3793 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
3794  int timeout_ms,
3795  rd_kafka_message_t **rkmessages,
3796  size_t rkmessages_size);
3797 
3807 RD_EXPORT
3809  rd_kafka_queue_t *rkqu,
3810  int timeout_ms,
3811  void (*consume_cb)(rd_kafka_message_t *rkmessage, void *commit_opaque),
3812  void *commit_opaque);
3813 
3814 
3849 RD_EXPORT
3851 rd_kafka_offset_store(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset);
3852 
3853 
3883 RD_EXPORT rd_kafka_resp_err_t
3884 rd_kafka_offsets_store(rd_kafka_t *rk,
3939 RD_EXPORT rd_kafka_resp_err_t
3940 rd_kafka_subscribe(rd_kafka_t *rk,
3941  const rd_kafka_topic_partition_list_t *topics);
3942 
3943 
3947 RD_EXPORT
3949 
3950 
3960 RD_EXPORT rd_kafka_resp_err_t
3961 rd_kafka_subscription(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **topics);
3962 
3963 
3964 
3994 RD_EXPORT
3995 rd_kafka_message_t *rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms);
3996 
4014 RD_EXPORT
4016 
4017 
4035 RD_EXPORT
4036 rd_kafka_error_t *rd_kafka_consumer_close_queue(rd_kafka_t *rk,
4037  rd_kafka_queue_t *rkqu);
4038 
4039 
4048 RD_EXPORT
4049 int rd_kafka_consumer_closed(rd_kafka_t *rk);
4050 
4051 
4069 RD_EXPORT rd_kafka_error_t *
4070 rd_kafka_incremental_assign(rd_kafka_t *rk,
4071  const rd_kafka_topic_partition_list_t *partitions);
4072 
4073 
4091 RD_EXPORT rd_kafka_error_t *rd_kafka_incremental_unassign(
4092  rd_kafka_t *rk,
4093  const rd_kafka_topic_partition_list_t *partitions);
4094 
4095 
4109 RD_EXPORT
4110 const char *rd_kafka_rebalance_protocol(rd_kafka_t *rk);
4111 
4112 
4133 RD_EXPORT rd_kafka_resp_err_t
4134 rd_kafka_assign(rd_kafka_t *rk,
4135  const rd_kafka_topic_partition_list_t *partitions);
4136 
4153 RD_EXPORT rd_kafka_resp_err_t
4154 rd_kafka_assignment(rd_kafka_t *rk,
4155  rd_kafka_topic_partition_list_t **partitions);
4156 
4157 
4174 RD_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk);
4175 
4176 
4197 RD_EXPORT rd_kafka_resp_err_t
4198 rd_kafka_commit(rd_kafka_t *rk,
4199  const rd_kafka_topic_partition_list_t *offsets,
4200  int async);
4201 
4202 
4209 RD_EXPORT rd_kafka_resp_err_t
4210 rd_kafka_commit_message(rd_kafka_t *rk,
4211  const rd_kafka_message_t *rkmessage,
4212  int async);
4213 
4214 
4237 RD_EXPORT rd_kafka_resp_err_t
4238 rd_kafka_commit_queue(rd_kafka_t *rk,
4239  const rd_kafka_topic_partition_list_t *offsets,
4240  rd_kafka_queue_t *rkqu,
4241  void (*cb)(rd_kafka_t *rk,
4242  rd_kafka_resp_err_t err,
4244  void *commit_opaque),
4245  void *commit_opaque);
4246 
4247 
4265 RD_EXPORT rd_kafka_resp_err_t
4266 rd_kafka_committed(rd_kafka_t *rk,
4267  rd_kafka_topic_partition_list_t *partitions,
4268  int timeout_ms);
4269 
4270 
4271 
4288 RD_EXPORT rd_kafka_resp_err_t
4289 rd_kafka_position(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions);
4290 
4291 
4292 
4304 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4305 rd_kafka_consumer_group_metadata(rd_kafka_t *rk);
4306 
4307 
4317 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4318 rd_kafka_consumer_group_metadata_new(const char *group_id);
4319 
4320 
4333 RD_EXPORT rd_kafka_consumer_group_metadata_t *
4335  int32_t generation_id,
4336  const char *member_id,
4337  const char *group_instance_id);
4338 
4339 
4344 RD_EXPORT void
4345 rd_kafka_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *);
4346 
4347 
4368 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_write(
4369  const rd_kafka_consumer_group_metadata_t *cgmd,
4370  void **bufferp,
4371  size_t *sizep);
4372 
4393 RD_EXPORT rd_kafka_error_t *rd_kafka_consumer_group_metadata_read(
4394  rd_kafka_consumer_group_metadata_t **cgmdp,
4395  const void *buffer,
4396  size_t size);
4397 
4413 #define RD_KAFKA_MSG_F_FREE \
4414  0x1
4416 #define RD_KAFKA_MSG_F_COPY \
4417  0x2
4419 #define RD_KAFKA_MSG_F_BLOCK \
4420  0x4
4431 #define RD_KAFKA_MSG_F_PARTITION \
4432  0x8
4537 RD_EXPORT
4538 int rd_kafka_produce(rd_kafka_topic_t *rkt,
4539  int32_t partition,
4540  int msgflags,
4541  void *payload,
4542  size_t len,
4543  const void *key,
4544  size_t keylen,
4545  void *msg_opaque);
4546 
4547 
4561 RD_EXPORT
4562 rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk, ...);
4563 
4564 
4576 RD_EXPORT
4577 rd_kafka_error_t *
4578 rd_kafka_produceva(rd_kafka_t *rk, const rd_kafka_vu_t *vus, size_t cnt);
4579 
4580 
4610 RD_EXPORT
4611 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt,
4612  int32_t partition,
4613  int msgflags,
4614  rd_kafka_message_t *rkmessages,
4615  int message_cnt);
4616 
4617 
4618 
4643 RD_EXPORT
4644 rd_kafka_resp_err_t rd_kafka_flush(rd_kafka_t *rk, int timeout_ms);
4645 
4646 
4647 
4678 RD_EXPORT
4679 rd_kafka_resp_err_t rd_kafka_purge(rd_kafka_t *rk, int purge_flags);
4680 
4681 
4689 #define RD_KAFKA_PURGE_F_QUEUE 0x1
4690 
4698 #define RD_KAFKA_PURGE_F_INFLIGHT 0x2
4699 
4700 
4704 #define RD_KAFKA_PURGE_F_NON_BLOCKING 0x4
4706 
4721 typedef struct rd_kafka_metadata_broker {
4722  int32_t id;
4723  char *host;
4724  int port;
4726 
4730 typedef struct rd_kafka_metadata_partition {
4731  int32_t id;
4732  rd_kafka_resp_err_t err;
4733  int32_t leader;
4734  int replica_cnt;
4735  int32_t *replicas;
4736  int isr_cnt;
4737  int32_t *isrs;
4743 typedef struct rd_kafka_metadata_topic {
4744  char *topic;
4745  int partition_cnt;
4746  struct rd_kafka_metadata_partition *partitions;
4754 typedef struct rd_kafka_metadata {
4755  int broker_cnt;
4756  struct rd_kafka_metadata_broker *brokers;
4758  int topic_cnt;
4759  struct rd_kafka_metadata_topic *topics;
4761  int32_t orig_broker_id;
4762  char *orig_broker_name;
4764 
4765 
4786 RD_EXPORT
4788 rd_kafka_metadata(rd_kafka_t *rk,
4789  int all_topics,
4790  rd_kafka_topic_t *only_rkt,
4791  const struct rd_kafka_metadata **metadatap,
4792  int timeout_ms);
4793 
4797 RD_EXPORT
4798 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
4799 
4800 
4821  char *member_id;
4822  char *client_id;
4823  char *client_host;
4824  void *member_metadata;
4826  int member_metadata_size;
4827  void *member_assignment;
4830 };
4831 
4835 struct rd_kafka_group_info {
4836  struct rd_kafka_metadata_broker broker;
4837  char *group;
4839  char *state;
4841  char *protocol;
4844 };
4854 };
4884 RD_EXPORT
4886 rd_kafka_list_groups(rd_kafka_t *rk,
4887  const char *group,
4888  const struct rd_kafka_group_list **grplistp,
4889  int timeout_ms);
4890 
4894 RD_EXPORT
4895 void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist);
4896 
4897 
4941 RD_EXPORT
4942 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
4943 
4944 
4945 
4958 RD_EXPORT RD_DEPRECATED void
4959 rd_kafka_set_logger(rd_kafka_t *rk,
4960  void (*func)(const rd_kafka_t *rk,
4961  int level,
4962  const char *fac,
4963  const char *buf));
4964 
4965 
4975 RD_EXPORT
4976 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
4977 
4978 
4982 RD_EXPORT
4983 void rd_kafka_log_print(const rd_kafka_t *rk,
4984  int level,
4985  const char *fac,
4986  const char *buf);
4987 
4988 
4994 RD_EXPORT
4995 void rd_kafka_log_syslog(const rd_kafka_t *rk,
4996  int level,
4997  const char *fac,
4998  const char *buf);
4999 
5000 
5023 RD_EXPORT
5024 int rd_kafka_outq_len(rd_kafka_t *rk);
5025 
5026 
5027 
5034 RD_EXPORT
5035 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
5036 
5037 
5038 
5044 RD_EXPORT
5045 int rd_kafka_thread_cnt(void);
5046 
5047 
5055 typedef enum rd_kafka_thread_type_t {
5060 
5061 
5070 RD_EXPORT
5071 int rd_kafka_wait_destroyed(int timeout_ms);
5079 RD_EXPORT
5080 int rd_kafka_unittest(void);
5081 
5082 
5099 RD_EXPORT
5101 
5102 
5118 typedef int rd_kafka_event_type_t;
5119 #define RD_KAFKA_EVENT_NONE 0x0
5120 #define RD_KAFKA_EVENT_DR 0x1
5121 #define RD_KAFKA_EVENT_FETCH 0x2
5122 #define RD_KAFKA_EVENT_LOG 0x4
5123 #define RD_KAFKA_EVENT_ERROR 0x8
5124 #define RD_KAFKA_EVENT_REBALANCE 0x10
5125 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
5126 #define RD_KAFKA_EVENT_STATS 0x40
5127 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
5128 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
5129 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT \
5130  102
5131 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
5132 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT \
5133  104
5134 #define RD_KAFKA_EVENT_DELETERECORDS_RESULT 105
5135 #define RD_KAFKA_EVENT_DELETEGROUPS_RESULT 106
5137 #define RD_KAFKA_EVENT_DELETECONSUMERGROUPOFFSETS_RESULT 107
5139 #define RD_KAFKA_EVENT_OAUTHBEARER_TOKEN_REFRESH 0x100
5140 #define RD_KAFKA_EVENT_BACKGROUND 0x200
5141 #define RD_KAFKA_EVENT_CREATEACLS_RESULT 0x400
5142 #define RD_KAFKA_EVENT_DESCRIBEACLS_RESULT 0x800
5143 #define RD_KAFKA_EVENT_DELETEACLS_RESULT 0x1000
5151 RD_EXPORT
5152 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev);
5160 RD_EXPORT
5161 const char *rd_kafka_event_name(const rd_kafka_event_t *rkev);
5162 
5163 
5173 RD_EXPORT
5174 void rd_kafka_event_destroy(rd_kafka_event_t *rkev);
5175 
5176 
5192 RD_EXPORT
5193 const rd_kafka_message_t *rd_kafka_event_message_next(rd_kafka_event_t *rkev);
5194 
5195 
5209 RD_EXPORT
5210 size_t rd_kafka_event_message_array(rd_kafka_event_t *rkev,
5211  const rd_kafka_message_t **rkmessages,
5212  size_t size);
5213 
5214 
5222 RD_EXPORT
5223 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev);
5224 
5225 
5237 RD_EXPORT
5238 const char *rd_kafka_event_config_string(rd_kafka_event_t *rkev);
5239 
5240 
5249 RD_EXPORT
5250 rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev);
5251 
5252 
5261 RD_EXPORT
5262 const char *rd_kafka_event_error_string(rd_kafka_event_t *rkev);
5263 
5264 
5273 RD_EXPORT
5274 int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev);
5275 
5276 
5295 RD_EXPORT
5296 void *rd_kafka_event_opaque(rd_kafka_event_t *rkev);
5297 
5298 
5307 RD_EXPORT
5308 int rd_kafka_event_log(rd_kafka_event_t *rkev,
5309  const char **fac,
5310  const char **str,
5311  int *level);
5312 
5313 
5325 RD_EXPORT
5326 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev,
5327  char *dst,
5328  size_t dstsize);
5329 
5330 
5343 RD_EXPORT
5344 const char *rd_kafka_event_stats(rd_kafka_event_t *rkev);
5345 
5346 
5358 rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev);
5359 
5360 
5371 RD_EXPORT rd_kafka_topic_partition_t *
5372 rd_kafka_event_topic_partition(rd_kafka_event_t *rkev);
5373 
5374 
5376 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
5378 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
5380 typedef rd_kafka_event_t rd_kafka_CreateAcls_result_t;
5382 typedef rd_kafka_event_t rd_kafka_DescribeAcls_result_t;
5384 typedef rd_kafka_event_t rd_kafka_DeleteAcls_result_t;
5386 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
5388 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
5390 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
5392 typedef rd_kafka_event_t rd_kafka_DeleteRecords_result_t;
5394 typedef rd_kafka_event_t rd_kafka_DeleteGroups_result_t;
5397 
5407 RD_EXPORT const rd_kafka_CreateTopics_result_t *
5408 rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev);
5409 
5419 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
5420 rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev);
5421 
5431 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
5432 rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev);
5433 
5443 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
5444 rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev);
5445 
5455 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
5456 rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev);
5457 
5465 RD_EXPORT const rd_kafka_DeleteRecords_result_t *
5466 rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev);
5467 
5477 RD_EXPORT const rd_kafka_DeleteGroups_result_t *
5478 rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev);
5479 
5489 RD_EXPORT const rd_kafka_DeleteConsumerGroupOffsets_result_t *
5490 rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev);
5491 
5499 RD_EXPORT const rd_kafka_CreateAcls_result_t *
5500 rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev);
5501 
5509 RD_EXPORT const rd_kafka_DescribeAcls_result_t *
5510 rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev);
5511 
5519 RD_EXPORT const rd_kafka_DeleteAcls_result_t *
5520 rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev);
5521 
5531 RD_EXPORT
5532 rd_kafka_event_t *rd_kafka_queue_poll(rd_kafka_queue_t *rkqu, int timeout_ms);
5533 
5548 RD_EXPORT
5549 int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms);
5550 
5551 
5595  rd_kafka_conf_t *conf,
5596  void **plug_opaquep,
5597  char *errstr,
5598  size_t errstr_size);
5599 
5681  rd_kafka_conf_t *conf,
5682  const char *name,
5683  const char *val,
5684  char *errstr,
5685  size_t errstr_size,
5686  void *ic_opaque);
5687 
5688 
5711  rd_kafka_conf_t *new_conf,
5712  const rd_kafka_conf_t *old_conf,
5713  size_t filter_cnt,
5714  const char **filter,
5715  void *ic_opaque);
5716 
5717 
5725  void *ic_opaque);
5727 
5746  rd_kafka_t *rk,
5747  const rd_kafka_conf_t *conf,
5748  void *ic_opaque,
5749  char *errstr,
5750  size_t errstr_size);
5751 
5752 
5760 typedef rd_kafka_resp_err_t(
5761  rd_kafka_interceptor_f_on_destroy_t)(rd_kafka_t *rk, void *ic_opaque);
5762 
5763 
5764 
5786  rd_kafka_t *rk,
5787  rd_kafka_message_t *rkmessage,
5788  void *ic_opaque);
5789 
5813  rd_kafka_t *rk,
5814  rd_kafka_message_t *rkmessage,
5815  void *ic_opaque);
5816 
5817 
5835  rd_kafka_t *rk,
5836  rd_kafka_message_t *rkmessage,
5837  void *ic_opaque);
5838 
5861  rd_kafka_t *rk,
5862  const rd_kafka_topic_partition_list_t *offsets,
5864  void *ic_opaque);
5865 
5866 
5889  rd_kafka_t *rk,
5890  int sockfd,
5891  const char *brokername,
5892  int32_t brokerid,
5893  int16_t ApiKey,
5894  int16_t ApiVersion,
5895  int32_t CorrId,
5896  size_t size,
5897  void *ic_opaque);
5898 
5899 
5926  rd_kafka_t *rk,
5927  int sockfd,
5928  const char *brokername,
5929  int32_t brokerid,
5930  int16_t ApiKey,
5931  int16_t ApiVersion,
5932  int32_t CorrId,
5933  size_t size,
5934  int64_t rtt,
5936  void *ic_opaque);
5937 
5938 
5956  rd_kafka_t *rk,
5957  rd_kafka_thread_type_t thread_type,
5958  const char *thread_name,
5959  void *ic_opaque);
5960 
5961 
5982  rd_kafka_t *rk,
5983  rd_kafka_thread_type_t thread_type,
5984  const char *thread_name,
5985  void *ic_opaque);
5986 
5987 
5988 
6002  rd_kafka_conf_t *conf,
6003  const char *ic_name,
6005  void *ic_opaque);
6006 
6007 
6021  rd_kafka_conf_t *conf,
6022  const char *ic_name,
6024  void *ic_opaque);
6025 
6040  rd_kafka_conf_t *conf,
6041  const char *ic_name,
6043  void *ic_opaque);
6044 
6045 
6067 RD_EXPORT rd_kafka_resp_err_t
6068 rd_kafka_conf_interceptor_add_on_new(rd_kafka_conf_t *conf,
6069  const char *ic_name,
6071  void *ic_opaque);
6072 
6073 
6074 
6088  rd_kafka_t *rk,
6089  const char *ic_name,
6091  void *ic_opaque);
6092 
6093 
6106 RD_EXPORT rd_kafka_resp_err_t
6107 rd_kafka_interceptor_add_on_send(rd_kafka_t *rk,
6108  const char *ic_name,
6110  void *ic_opaque);
6111 
6125  rd_kafka_t *rk,
6126  const char *ic_name,
6127  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
6128  void *ic_opaque);
6129 
6130 
6144  rd_kafka_t *rk,
6145  const char *ic_name,
6147  void *ic_opaque);
6148 
6149 
6163  rd_kafka_t *rk,
6164  const char *ic_name,
6166  void *ic_opaque);
6167 
6168 
6182  rd_kafka_t *rk,
6183  const char *ic_name,
6185  void *ic_opaque);
6186 
6187 
6201  rd_kafka_t *rk,
6202  const char *ic_name,
6203  rd_kafka_interceptor_f_on_response_received_t *on_response_received,
6204  void *ic_opaque);
6205 
6206 
6220  rd_kafka_t *rk,
6221  const char *ic_name,
6223  void *ic_opaque);
6224 
6225 
6239  rd_kafka_t *rk,
6240  const char *ic_name,
6242  void *ic_opaque);
6243 
6244 
6245 
6266 RD_EXPORT rd_kafka_resp_err_t
6267 rd_kafka_topic_result_error(const rd_kafka_topic_result_t *topicres);
6268 
6275 RD_EXPORT const char *
6276 rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres);
6277 
6283 RD_EXPORT const char *
6284 rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres);
6285 
6295 RD_EXPORT const rd_kafka_error_t *
6296 rd_kafka_group_result_error(const rd_kafka_group_result_t *groupres);
6297 
6303 RD_EXPORT const char *
6304 rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres);
6305 
6311 RD_EXPORT const rd_kafka_topic_partition_list_t *
6312 rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres);
6313 
6314 
6361 typedef enum rd_kafka_admin_op_t {
6362  RD_KAFKA_ADMIN_OP_ANY = 0,
6390 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
6411 RD_EXPORT rd_kafka_AdminOptions_t *
6412 rd_kafka_AdminOptions_new(rd_kafka_t *rk, rd_kafka_admin_op_t for_api);
6413 
6414 
6419 
6420 
6439 RD_EXPORT rd_kafka_resp_err_t
6441  int timeout_ms,
6442  char *errstr,
6443  size_t errstr_size);
6444 
6445 
6472 RD_EXPORT rd_kafka_resp_err_t
6474  int timeout_ms,
6475  char *errstr,
6476  size_t errstr_size);
6477 
6478 
6497 RD_EXPORT rd_kafka_resp_err_t
6499  int true_or_false,
6500  char *errstr,
6501  size_t errstr_size);
6502 
6503 
6529 RD_EXPORT rd_kafka_resp_err_t
6531  int32_t broker_id,
6532  char *errstr,
6533  size_t errstr_size);
6534 
6535 
6536 
6541 RD_EXPORT void
6543  void *ev_opaque);
6544 
6545 
6546 
6547 /*
6548  * CreateTopics - create topics in cluster.
6549  *
6550  */
6551 
6552 
6554 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
6555 
6577 RD_EXPORT rd_kafka_NewTopic_t *rd_kafka_NewTopic_new(const char *topic,
6578  int num_partitions,
6579  int replication_factor,
6580  char *errstr,
6581  size_t errstr_size);
6582 
6587 RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic);
6588 
6589 
6595 RD_EXPORT void rd_kafka_NewTopic_destroy_array(rd_kafka_NewTopic_t **new_topics,
6596  size_t new_topic_cnt);
6597 
6598 
6620 RD_EXPORT rd_kafka_resp_err_t
6622  int32_t partition,
6623  int32_t *broker_ids,
6624  size_t broker_id_cnt,
6625  char *errstr,
6626  size_t errstr_size);
6627 
6640 RD_EXPORT rd_kafka_resp_err_t
6642  const char *name,
6643  const char *value);
6644 
6645 
6664 RD_EXPORT void rd_kafka_CreateTopics(rd_kafka_t *rk,
6665  rd_kafka_NewTopic_t **new_topics,
6666  size_t new_topic_cnt,
6667  const rd_kafka_AdminOptions_t *options,
6668  rd_kafka_queue_t *rkqu);
6669 
6670 
6671 /*
6672  * CreateTopics result type and methods
6673  */
6674 
6683 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_CreateTopics_result_topics(
6684  const rd_kafka_CreateTopics_result_t *result,
6685  size_t *cntp);
6686 
6687 
6688 
6689 /*
6690  * DeleteTopics - delete topics from cluster
6691  *
6692  */
6693 
6695 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
6696 
6706 RD_EXPORT rd_kafka_DeleteTopic_t *rd_kafka_DeleteTopic_new(const char *topic);
6707 
6712 RD_EXPORT void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic);
6713 
6719 RD_EXPORT void
6721  size_t del_topic_cnt);
6722 
6736 RD_EXPORT
6737 void rd_kafka_DeleteTopics(rd_kafka_t *rk,
6738  rd_kafka_DeleteTopic_t **del_topics,
6739  size_t del_topic_cnt,
6740  const rd_kafka_AdminOptions_t *options,
6741  rd_kafka_queue_t *rkqu);
6742 
6743 
6744 
6745 /*
6746  * DeleteTopics result type and methods
6747  */
6748 
6757 RD_EXPORT const rd_kafka_topic_result_t **rd_kafka_DeleteTopics_result_topics(
6758  const rd_kafka_DeleteTopics_result_t *result,
6759  size_t *cntp);
6760 
6761 
6762 
6763 /*
6764  * CreatePartitions - add partitions to topic.
6765  *
6766  */
6767 
6769 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
6770 
6787 RD_EXPORT rd_kafka_NewPartitions_t *
6788 rd_kafka_NewPartitions_new(const char *topic,
6789  size_t new_total_cnt,
6790  char *errstr,
6791  size_t errstr_size);
6792 
6797 RD_EXPORT void
6799 
6805 RD_EXPORT void
6807  size_t new_parts_cnt);
6808 
6832  rd_kafka_NewPartitions_t *new_parts,
6833  int32_t new_partition_idx,
6834  int32_t *broker_ids,
6835  size_t broker_id_cnt,
6836  char *errstr,
6837  size_t errstr_size);
6838 
6839 
6858 RD_EXPORT void rd_kafka_CreatePartitions(rd_kafka_t *rk,
6859  rd_kafka_NewPartitions_t **new_parts,
6860  size_t new_parts_cnt,
6861  const rd_kafka_AdminOptions_t *options,
6862  rd_kafka_queue_t *rkqu);
6863 
6864 
6865 
6866 /*
6867  * CreatePartitions result type and methods
6868  */
6869 
6878 RD_EXPORT const rd_kafka_topic_result_t **
6880  const rd_kafka_CreatePartitions_result_t *result,
6881  size_t *cntp);
6882 
6883 
6884 
6885 /*
6886  * Cluster, broker, topic configuration entries, sources, etc.
6887  *
6888  */
6889 
6899 typedef enum rd_kafka_ConfigSource_t {
6916 
6921 
6925 RD_EXPORT const char *
6927 
6930 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
6935 RD_EXPORT const char *
6937 
6942 RD_EXPORT const char *
6944 
6948 RD_EXPORT rd_kafka_ConfigSource_t
6950 
6955 RD_EXPORT int
6957 
6963 RD_EXPORT int
6965 
6973 RD_EXPORT int
6975 
6979 RD_EXPORT int
6981 
6982 
6993 RD_EXPORT const rd_kafka_ConfigEntry_t **
6995  size_t *cntp);
6996 
6997 
6998 
7003 typedef enum rd_kafka_ResourceType_t {
7005  RD_KAFKA_RESOURCE_ANY = 1,
7011 
7016 typedef enum rd_kafka_ResourcePatternType_t {
7027  RD_KAFKA_RESOURCE_PATTERN_TYPE__CNT,
7029 
7033 RD_EXPORT const char *rd_kafka_ResourcePatternType_name(
7034  rd_kafka_ResourcePatternType_t resource_pattern_type);
7035 
7039 RD_EXPORT const char *
7041 
7043 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
7044 
7045 
7054 RD_EXPORT rd_kafka_ConfigResource_t *
7056  const char *resname);
7057 
7062 RD_EXPORT void
7064 
7065 
7071 RD_EXPORT void
7073  size_t config_cnt);
7074 
7075 
7089 RD_EXPORT rd_kafka_resp_err_t
7091  const char *name,
7092  const char *value);
7093 
7094 
7103 RD_EXPORT const rd_kafka_ConfigEntry_t **
7105  size_t *cntp);
7106 
7107 
7108 
7112 RD_EXPORT rd_kafka_ResourceType_t
7114 
7118 RD_EXPORT const char *
7120 
7124 RD_EXPORT rd_kafka_resp_err_t
7126 
7131 RD_EXPORT const char *
7133 
7134 
7135 /*
7136  * AlterConfigs - alter cluster configuration.
7137  *
7138  */
7139 
7140 
7161 RD_EXPORT
7162 void rd_kafka_AlterConfigs(rd_kafka_t *rk,
7163  rd_kafka_ConfigResource_t **configs,
7164  size_t config_cnt,
7165  const rd_kafka_AdminOptions_t *options,
7166  rd_kafka_queue_t *rkqu);
7167 
7168 
7169 /*
7170  * AlterConfigs result type and methods
7171  */
7172 
7187 RD_EXPORT const rd_kafka_ConfigResource_t **
7189  const rd_kafka_AlterConfigs_result_t *result,
7190  size_t *cntp);
7191 
7192 
7193 
7194 /*
7195  * DescribeConfigs - retrieve cluster configuration.
7196  *
7197  */
7198 
7199 
7224 RD_EXPORT
7225 void rd_kafka_DescribeConfigs(rd_kafka_t *rk,
7226  rd_kafka_ConfigResource_t **configs,
7227  size_t config_cnt,
7228  const rd_kafka_AdminOptions_t *options,
7229  rd_kafka_queue_t *rkqu);
7230 
7231 
7232 
7233 /*
7234  * DescribeConfigs result type and methods
7235  */
7236 
7245 RD_EXPORT const rd_kafka_ConfigResource_t **
7247  const rd_kafka_DescribeConfigs_result_t *result,
7248  size_t *cntp);
7249 
7250 
7251 /*
7252  * DeleteRecords - delete records (messages) from partitions
7253  *
7254  *
7255  */
7256 
7258 typedef struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t;
7259 
7277  const rd_kafka_topic_partition_list_t *before_offsets);
7278 
7283 RD_EXPORT void
7285 
7291 RD_EXPORT void
7293  size_t del_record_cnt);
7294 
7316 RD_EXPORT void rd_kafka_DeleteRecords(rd_kafka_t *rk,
7317  rd_kafka_DeleteRecords_t **del_records,
7318  size_t del_record_cnt,
7319  const rd_kafka_AdminOptions_t *options,
7320  rd_kafka_queue_t *rkqu);
7321 
7322 
7323 /*
7324  * DeleteRecords result type and methods
7325  */
7326 
7336 RD_EXPORT const rd_kafka_topic_partition_list_t *
7338  const rd_kafka_DeleteRecords_result_t *result);
7339 
7340 /*
7341  * DeleteGroups - delete groups from cluster
7342  *
7343  *
7344  */
7345 
7347 typedef struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t;
7348 
7358 RD_EXPORT rd_kafka_DeleteGroup_t *rd_kafka_DeleteGroup_new(const char *group);
7359 
7364 RD_EXPORT void rd_kafka_DeleteGroup_destroy(rd_kafka_DeleteGroup_t *del_group);
7365 
7371 RD_EXPORT void
7373  size_t del_group_cnt);
7374 
7388 RD_EXPORT
7389 void rd_kafka_DeleteGroups(rd_kafka_t *rk,
7390  rd_kafka_DeleteGroup_t **del_groups,
7391  size_t del_group_cnt,
7392  const rd_kafka_AdminOptions_t *options,
7393  rd_kafka_queue_t *rkqu);
7394 
7395 
7396 
7397 /*
7398  * DeleteGroups result type and methods
7399  */
7400 
7409 RD_EXPORT const rd_kafka_group_result_t **rd_kafka_DeleteGroups_result_groups(
7410  const rd_kafka_DeleteGroups_result_t *result,
7411  size_t *cntp);
7412 
7413 
7414 /*
7415  * DeleteConsumerGroupOffsets - delete groups from cluster
7416  *
7417  *
7418  */
7419 
7421 typedef struct rd_kafka_DeleteConsumerGroupOffsets_s
7423 
7438  const char *group,
7439  const rd_kafka_topic_partition_list_t *partitions);
7440 
7446  rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets);
7447 
7454  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
7455  size_t del_grpoffset_cnt);
7456 
7475 RD_EXPORT
7477  rd_kafka_t *rk,
7478  rd_kafka_DeleteConsumerGroupOffsets_t **del_grpoffsets,
7479  size_t del_grpoffsets_cnt,
7480  const rd_kafka_AdminOptions_t *options,
7481  rd_kafka_queue_t *rkqu);
7482 
7483 
7484 
7485 /*
7486  * DeleteConsumerGroupOffsets result type and methods
7487  */
7488 
7497 RD_EXPORT const rd_kafka_group_result_t **
7499  const rd_kafka_DeleteConsumerGroupOffsets_result_t *result,
7500  size_t *cntp);
7501 
7507 typedef struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t;
7508 
7514 
7518 RD_EXPORT const rd_kafka_error_t *
7519 rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres);
7520 
7521 
7531 typedef enum rd_kafka_AclOperation_t {
7534  1,
7543  9,
7545  10,
7547  11,
7549  12,
7550  RD_KAFKA_ACL_OPERATION__CNT
7556 RD_EXPORT const char *
7570 typedef enum rd_kafka_AclPermissionType_t {
7573  1,
7576  RD_KAFKA_ACL_PERMISSION_TYPE__CNT
7578 
7582 RD_EXPORT const char *rd_kafka_AclPermissionType_name(
7583  rd_kafka_AclPermissionType_t acl_permission_type);
7584 
7605 RD_EXPORT rd_kafka_AclBinding_t *
7607  const char *name,
7608  rd_kafka_ResourcePatternType_t resource_pattern_type,
7609  const char *principal,
7610  const char *host,
7611  rd_kafka_AclOperation_t operation,
7612  rd_kafka_AclPermissionType_t permission_type,
7613  char *errstr,
7614  size_t errstr_size);
7615 
7641 RD_EXPORT rd_kafka_AclBindingFilter_t *rd_kafka_AclBindingFilter_new(
7642  rd_kafka_ResourceType_t restype,
7643  const char *name,
7644  rd_kafka_ResourcePatternType_t resource_pattern_type,
7645  const char *principal,
7646  const char *host,
7647  rd_kafka_AclOperation_t operation,
7648  rd_kafka_AclPermissionType_t permission_type,
7649  char *errstr,
7650  size_t errstr_size);
7651 
7655 RD_EXPORT rd_kafka_ResourceType_t
7657 
7663 RD_EXPORT const char *
7665 
7671 RD_EXPORT const char *
7673 
7679 RD_EXPORT const char *
7681 
7685 RD_EXPORT rd_kafka_AclOperation_t
7687 
7693 
7699 
7703 RD_EXPORT const rd_kafka_error_t *
7705 
7706 
7711 RD_EXPORT void rd_kafka_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding);
7712 
7713 
7719 RD_EXPORT void
7721  size_t acl_bindings_cnt);
7722 
7730 RD_EXPORT const rd_kafka_acl_result_t **
7731 rd_kafka_CreateAcls_result_acls(const rd_kafka_CreateAcls_result_t *result,
7732  size_t *cntp);
7733 
7750 RD_EXPORT void rd_kafka_CreateAcls(rd_kafka_t *rk,
7751  rd_kafka_AclBinding_t **new_acls,
7752  size_t new_acls_cnt,
7753  const rd_kafka_AdminOptions_t *options,
7754  rd_kafka_queue_t *rkqu);
7755 
7769 RD_EXPORT const rd_kafka_AclBinding_t **
7770 rd_kafka_DescribeAcls_result_acls(const rd_kafka_DescribeAcls_result_t *result,
7771  size_t *cntp);
7772 
7787 RD_EXPORT void rd_kafka_DescribeAcls(rd_kafka_t *rk,
7788  rd_kafka_AclBindingFilter_t *acl_filter,
7789  const rd_kafka_AdminOptions_t *options,
7790  rd_kafka_queue_t *rkqu);
7791 
7798 typedef struct rd_kafka_DeleteAcls_result_response_s
7800 
7808 RD_EXPORT const rd_kafka_DeleteAcls_result_response_t **
7809 rd_kafka_DeleteAcls_result_responses(const rd_kafka_DeleteAcls_result_t *result,
7810  size_t *cntp);
7811 
7816 RD_EXPORT const rd_kafka_error_t *rd_kafka_DeleteAcls_result_response_error(
7817  const rd_kafka_DeleteAcls_result_response_t *result_response);
7818 
7819 
7826 RD_EXPORT const rd_kafka_AclBinding_t **
7828  const rd_kafka_DeleteAcls_result_response_t *result_response,
7829  size_t *matching_acls_cntp);
7830 
7847 RD_EXPORT void rd_kafka_DeleteAcls(rd_kafka_t *rk,
7848  rd_kafka_AclBindingFilter_t **del_acls,
7849  size_t del_acls_cnt,
7850  const rd_kafka_AdminOptions_t *options,
7851  rd_kafka_queue_t *rkqu);
7852 
7905 RD_EXPORT
7907 rd_kafka_oauthbearer_set_token(rd_kafka_t *rk,
7908  const char *token_value,
7909  int64_t md_lifetime_ms,
7910  const char *md_principal_name,
7911  const char **extensions,
7912  size_t extension_size,
7913  char *errstr,
7914  size_t errstr_size);
7915 
7936 RD_EXPORT
7938  const char *errstr);
7939 
8120 RD_EXPORT
8121 rd_kafka_error_t *rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms);
8122 
8123 
8124 
8169 RD_EXPORT
8170 rd_kafka_error_t *rd_kafka_begin_transaction(rd_kafka_t *rk);
8171 
8172 
8237 RD_EXPORT
8238 rd_kafka_error_t *rd_kafka_send_offsets_to_transaction(
8239  rd_kafka_t *rk,
8240  const rd_kafka_topic_partition_list_t *offsets,
8241  const rd_kafka_consumer_group_metadata_t *cgmetadata,
8242  int timeout_ms);
8243 
8244 
8308 RD_EXPORT
8309 rd_kafka_error_t *rd_kafka_commit_transaction(rd_kafka_t *rk, int timeout_ms);
8310 
8311 
8367 RD_EXPORT
8368 rd_kafka_error_t *rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms);
8369 
8370 
8373 /* @cond NO_DOC */
8374 #ifdef __cplusplus
8375 }
8376 #endif
8377 #endif /* _RDKAFKA_H_ */
8378 /* @endcond NO_DOC */
void * _private
Definition: rdkafka.h:900
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_resp_err_t
Error codes.
Definition: rdkafka.h:278
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:5801
rd_kafka_topic_t * rkt
Definition: rdkafka.h:1410
RD_EXPORT void * rd_kafka_mem_malloc(rd_kafka_t *rk, size_t size)
Allocate memory using the same allocator librdkafka uses.
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.
Definition: rdkafka.h:2314
RD_EXPORT rd_kafka_error_t * rd_kafka_init_transactions(rd_kafka_t *rk, int timeout_ms)
Initialize transactions for the producer instance.
Definition: rdkafka.h:413
RD_EXPORT const char * rd_kafka_ConfigEntry_name(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_ConfigSource_t
Apache Kafka config sources.
Definition: rdkafka.h:6915
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 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_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 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...
Definition: rdkafka.h:342
RD_EXPORT const rd_kafka_DeleteTopics_result_t * rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev)
Get DeleteTopics result.
rd_kafka_resp_err_t err
Definition: rdkafka.h:1409
RD_EXPORT rd_kafka_resp_err_t rd_kafka_test_fatal_error(rd_kafka_t *rk, rd_kafka_resp_err_t err, const char *reason)
Trigger a fatal error for testing purposes.
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1603
int member_cnt
Definition: rdkafka.h:4859
RD_EXPORT char * rd_kafka_memberid(const rd_kafka_t *rk)
Returns this client's broker-assigned group member id.
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.
int cnt
Definition: rdkafka.h:918
Definition: rdkafka.h:360
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_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 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 int rd_kafka_consumer_closed(rd_kafka_t *rk)
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 const char * rd_kafka_ConfigResource_name(const rd_kafka_ConfigResource_t *config)
RD_EXPORT int rd_kafka_thread_cnt(void)
Retrieve the current number of threads in use by librdkafka.
RD_EXPORT void rd_kafka_conf_set_consume_cb(rd_kafka_conf_t *conf, void(*consume_cb)(rd_kafka_message_t *rkmessage, void *opaque))
Consumer: Set consume callback for use with rd_kafka_consumer_poll()
rd_kafka_topic_partition_t * elems
Definition: rdkafka.h:920
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_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_ConfigEntry_t ** rd_kafka_ConfigEntry_synonyms(const rd_kafka_ConfigEntry_t *entry, size_t *cntp)
Definition: rdkafka.h:2312
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_DescribeAcls_result_t * rd_kafka_event_DescribeAcls_result(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_event_name(const rd_kafka_event_t *rkev)
Definition: rdkafka.h:402
Definition: rdkafka.h:1099
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 int rd_kafka_unittest(void)
Run librdkafka's built-in unit-tests.
Definition: rdkafka.h:7021
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 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf, const char *name, const char *value, char *errstr, size_t errstr_size)
Sets a single rd_kafka_topic_conf_t value by property name.
rd_kafka_admin_op_t
Admin operation enum name for use with rd_kafka_AdminOptions_new()
Definition: rdkafka.h:6377
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 size_t rd_kafka_header_cnt(const rd_kafka_headers_t *hdrs)
Returns the number of header key/value pairs.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_add(rd_kafka_headers_t *hdrs, const char *name, ssize_t name_size, const void *value, ssize_t value_size)
Add header with name name and value val (copied) of size size (not including null-terminator).
rd_kafka_event_t rd_kafka_DeleteTopics_result_t
Definition: rdkafka.h:5394
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 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 void rd_kafka_message_destroy(rd_kafka_message_t *rkmessage)
Frees resources for rkmessage and hands ownership back to rdkafka.
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.
struct rd_kafka_DeleteConsumerGroupOffsets_s rd_kafka_DeleteConsumerGroupOffsets_t
Definition: rdkafka.h:7437
Definition: rdkafka.h:316
Definition: rdkafka.h:613
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.
Definition: rdkafka.h:372
RD_EXPORT const char * rd_kafka_error_name(const rd_kafka_error_t *error)
Definition: rdkafka.h:492
Definition: rdkafka.h:1565
Definition: rdkafka.h:1088
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.
Definition: rdkafka.h:330
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_consumer(rd_kafka_t *rk)
RD_EXPORT const char * rd_kafka_ConfigResource_error_string(const rd_kafka_ConfigResource_t *config)
char * state
Definition: rdkafka.h:4855
RD_EXPORT int rd_kafka_ConfigEntry_is_read_only(const rd_kafka_ConfigEntry_t *entry)
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_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:5904
size_t key_len
Definition: rdkafka.h:1421
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.
Definition: rdkafka.h:310
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_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 rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev)
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_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_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.
Definition: rdkafka.h:1605
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.
Definition: rdkafka.h:576
Definition: rdkafka.h:582
int member_assignment_size
Definition: rdkafka.h:4845
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 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...
Definition: rdkafka.h:7560
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.
Group information.
Definition: rdkafka.h:4851
char * group
Definition: rdkafka.h:4853
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...
Partition information.
Definition: rdkafka.h:4746
Definition: rdkafka.h:297
Definition: rdkafka.h:2313
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. This is mainly for client binding use and not for application use.
Definition: rdkafka.h:378
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.
Definition: rdkafka.h:518
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_error_t * rd_kafka_abort_transaction(rd_kafka_t *rk, int timeout_ms)
Aborts the ongoing transaction.
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 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 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_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:5726
RD_EXPORT const char * rd_kafka_version_str(void)
Returns the librdkafka version as string.
RD_EXPORT void * rd_kafka_opaque(const rd_kafka_t *rk)
Retrieves the opaque pointer previously set with rd_kafka_conf_set_opaque()
char * client_id
Definition: rdkafka.h:4838
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.
const char * name
Definition: rdkafka.h:639
Definition: rdkafka.h:306
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 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.
Definition: rdkafka.h:542
char * client_host
Definition: rdkafka.h:4839
Definition: rdkafka.h:546
Definition: rdkafka.h:2328
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.
struct rd_kafka_group_info * groups
Definition: rdkafka.h:4868
RD_EXPORT const char * rd_kafka_AclOperation_name(rd_kafka_AclOperation_t acl_operation)
rd_kafka_event_t rd_kafka_CreateTopics_result_t
Definition: rdkafka.h:5392
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_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...
Definition: rdkafka.h:1093
Definition: rdkafka.h:6385
Definition: rdkafka.h:2327
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...
Definition: rdkafka.h:7549
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.
Definition: rdkafka.h:326
rd_kafka_AclBinding_t rd_kafka_AclBindingFilter_t
ACL Binding filter is used to filter access control lists.
Definition: rdkafka.h:7529
RD_EXPORT void rd_kafka_conf_enable_sasl_queue(rd_kafka_conf_t *conf, int enable)
Enable/disable creation of a queue specific to SASL events and callbacks.
rd_kafka_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:5761
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.
Definition: rdkafka.h:6390
RD_EXPORT const char * rd_kafka_AclPermissionType_name(rd_kafka_AclPermissionType_t acl_permission_type)
char * protocol_type
Definition: rdkafka.h:4856
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:5828
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:5876
Definition: rdkafka.h:334
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:5610
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...
Definition: rdkafka.h:7562
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 rd_kafka_ResourceType_t rd_kafka_AclBinding_restype(const rd_kafka_AclBinding_t *acl)
rd_kafka_event_t rd_kafka_CreateAcls_result_t
Definition: rdkafka.h:5396
RD_EXPORT int rd_kafka_outq_len(rd_kafka_t *rk)
Returns the current out queue length.
int group_cnt
Definition: rdkafka.h:4869
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...
Definition: rdkafka.h:352
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_consume(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_consume_t *on_consume, void *ic_opaque)
Append an on_consume() interceptor.
rd_kafka_vtype_t
Var-arg tag types.
Definition: rdkafka.h:1086
RD_EXPORT const rd_kafka_message_t * rd_kafka_event_message_next(rd_kafka_event_t *rkev)
RD_EXPORT int rd_kafka_error_txn_requires_abort(const rd_kafka_error_t *error)
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...
Definition: rdkafka.h:392
Definition: rdkafka.h:5074
RD_EXPORT const rd_kafka_DeleteConsumerGroupOffsets_result_t * rd_kafka_event_DeleteConsumerGroupOffsets_result(rd_kafka_event_t *rkev)
Get DeleteConsumerGroupOffsets result.
Definition: rdkafka.h:388
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_EXPORT void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf, void(*error_cb)(rd_kafka_t *rk, int err, const char *reason, void *opaque))
Set error callback in provided conf object.
rd_kafka_resp_err_t err
Definition: rdkafka.h:4854
rd_kafka_ResourceType_t
Apache Kafka resource types.
Definition: rdkafka.h:7019
size_t len
Definition: rdkafka.h:1416
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, int async)
Commit offsets on broker for the provided list of partitions.
rd_kafka_event_t rd_kafka_AlterConfigs_result_t
Definition: rdkafka.h:5404
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:5971
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.g., through pattern-matched topics). The topic config object is not usable after this call.
RD_EXPORT rd_kafka_consumer_group_metadata_t * rd_kafka_consumer_group_metadata(rd_kafka_t *rk)
Definition: rdkafka.h:7590
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, and marks the offsets as part part of the current transaction. These offsets will be considered committed only if the transaction is committed successfully.
Definition: rdkafka.h:470
RD_EXPORT rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk)
Returns Kafka handle type.
Definition: rdkafka.h:7034
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 const rd_kafka_error_t * rd_kafka_AclBinding_error(const rd_kafka_AclBinding_t *acl)
Definition: rdkafka.h:386
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:5696
Definition: rdkafka.h:320
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_background(rd_kafka_t *rk)
RD_EXPORT const char * rd_kafka_event_error_string(rd_kafka_event_t *rkev)
Definition: rdkafka.h:472
Definition: rdkafka.h:510
RD_EXPORT rd_kafka_ConfigSource_t rd_kafka_ConfigEntry_source(const rd_kafka_ConfigEntry_t *entry)
struct rd_kafka_DeleteRecords_s rd_kafka_DeleteRecords_t
Definition: rdkafka.h:7274
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 rd_kafka_topic_partition_t * rd_kafka_event_topic_partition(rd_kafka_event_t *rkev)
struct rd_kafka_metadata_broker broker
Definition: rdkafka.h:4852
RD_EXPORT void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu)
RD_EXPORT size_t rd_kafka_queue_length(rd_kafka_queue_t *rkqu)
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.
Definition: rdkafka.h:338
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_error(const rd_kafka_ConfigResource_t *config)
Definition: rdkafka.h:7025
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.
Definition: rdkafka.h:7551
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.
RD_EXPORT const rd_kafka_error_t * rd_kafka_DeleteAcls_result_response_error(const rd_kafka_DeleteAcls_result_response_t *result_response)
Definition: rdkafka.h:451
RD_EXPORT rd_kafka_resp_err_t rd_kafka_header_get(const rd_kafka_headers_t *hdrs, size_t idx, const char *name, const void **valuep, size_t *sizep)
Iterator for headers matching name.
rd_kafka_resp_err_t code
Definition: rdkafka.h:638
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_EXPORT const rd_kafka_CreateAcls_result_t * rd_kafka_event_CreateAcls_result(rd_kafka_event_t *rkev)
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 void rd_kafka_destroy(rd_kafka_t *rk)
Destroy Kafka handle.
VTYPE + argument container for use with rd_kafka_produce_va()
Definition: rdkafka.h:1112
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:6570
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 rd_kafka_DescribeConfigs_result_t * rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev)
Get DescribeConfigs result.
Definition: rdkafka.h:1098
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 rd_kafka_resp_err_t rd_kafka_metadata(rd_kafka_t *rk, int all_topics, rd_kafka_topic_t *only_rkt, const struct rd_kafka_metadata **metadatap, int timeout_ms)
Request Metadata from broker.
rd_kafka_event_t rd_kafka_CreatePartitions_result_t
Definition: rdkafka.h:5402
RD_EXPORT int rd_kafka_wait_destroyed(int timeout_ms)
Wait for all rd_kafka_t objects to be destroyed.
Definition: rdkafka.h:6931
RD_EXPORT const char * rd_kafka_error_string(const rd_kafka_error_t *error)
Definition: rdkafka.h:404
Definition: rdkafka.h:456
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_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().
Definition: rdkafka.h:435
RD_EXPORT void rd_kafka_AdminOptions_destroy(rd_kafka_AdminOptions_t *options)
Destroy a AdminOptions object.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_thread_exit(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_thread_exit_t *on_thread_exit, void *ic_opaque)
Append an on_thread_exit() interceptor.
rd_kafka_event_t rd_kafka_DescribeConfigs_result_t
Definition: rdkafka.h:5406
rd_kafka_AclPermissionType_t
Apache Kafka ACL permission types.
Definition: rdkafka.h:7586
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.
Definition: rdkafka.h:390
int64_t offset
Definition: rdkafka.h:1423
Definition: rdkafka.h:7558
RD_EXPORT void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf, void *rkt_opaque)
Sets the application's opaque pointer that will be passed to all topic callbacks as the rkt_opaque ar...
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_new(size_t initial_count)
Create a new headers list.
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.
Definition: rdkafka.h:439
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 const rd_kafka_DeleteRecords_result_t * rd_kafka_event_DeleteRecords_result(rd_kafka_event_t *rkev)
Definition: rdkafka.h:7554
Definition: rdkafka.h:6384
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 const rd_kafka_DeleteAcls_result_t * rd_kafka_event_DeleteAcls_result(rd_kafka_event_t *rkev)
RD_EXPORT RD_DEPRECATED int rd_kafka_errno(void)
Returns the thread-local system errno.
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_commit_transaction(rd_kafka_t *rk, int timeout_ms)
Commit the current transaction (as started with rd_kafka_begin_transaction()).
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.
Definition: rdkafka.h:6380
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:5777
Definition: rdkafka.h:421
Definition: rdkafka.h:6378
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 const char * rd_kafka_ResourcePatternType_name(rd_kafka_ResourcePatternType_t resource_pattern_type)
Definition: rdkafka.h:293
Definition: rdkafka.h:411
Definition: rdkafka.h:6381
Definition: rdkafka.h:376
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.
Definition: rdkafka.h:281
RD_EXPORT void rd_kafka_destroy_flags(rd_kafka_t *rk, int flags)
Destroy Kafka handle according to specified destroy flags.
Group member information.
Definition: rdkafka.h:4836
void * key
Definition: rdkafka.h:1419
struct rd_kafka_AclBinding_s rd_kafka_AclBinding_t
ACL Binding is used to create access control lists.
Definition: rdkafka.h:7523
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_kafka_topic_conf_t * rd_kafka_topic_conf_new(void)
Create topic configuration object.
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 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 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk)
Dumps rdkafka's internal state for handle rk to stream fp.
Definition: rdkafka.h:1090
Definition: rdkafka.h:7040
Definition: rdkafka.h:308
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.
A growable list of Topic+Partitions.
Definition: rdkafka.h:917
int rd_kafka_event_type_t
Event types.
Definition: rdkafka.h:5134
Topic information.
Definition: rdkafka.h:4759
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 rd_kafka_queue_t * rd_kafka_queue_get_main(rd_kafka_t *rk)
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.
Definition: rdkafka.h:295
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 const rd_kafka_AlterConfigs_result_t * rd_kafka_event_AlterConfigs_result(rd_kafka_event_t *rkev)
Get AlterConfigs result.
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.
Definition: rdkafka.h:7022
Definition: rdkafka.h:7591
Definition: rdkafka.h:1089
RD_EXPORT void rd_kafka_NewTopic_destroy(rd_kafka_NewTopic_t *new_topic)
Destroy and free a NewTopic object previously created with rd_kafka_NewTopic_new() ...
rd_kafka_event_t rd_kafka_DeleteConsumerGroupOffsets_result_t
Definition: rdkafka.h:5412
int32_t partition
Definition: rdkafka.h:894
Definition: rdkafka.h:291
Definition: rdkafka.h:322
Definition: rdkafka.h:289
void * opaque
Definition: rdkafka.h:898
const char * desc
Definition: rdkafka.h:640
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:6785
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 rd_kafka_resp_err_t rd_kafka_assign(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *partitions)
Atomic assignment of partitions to consume.
RD_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 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 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.
Definition: rdkafka.h:382
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_queue_t * rd_kafka_queue_get_partition(rd_kafka_t *rk, const char *topic, int32_t partition)
RD_EXPORT int rd_kafka_topic_partition_list_del_by_idx(rd_kafka_topic_partition_list_t *rktparlist, int idx)
Delete partition from list by elems[] index.
rd_kafka_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:5740
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 rd_kafka_CreatePartitions_result_t * rd_kafka_event_CreatePartitions_result(rd_kafka_event_t *rkev)
Get CreatePartitions result.
Definition: rdkafka.h:7555
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 const char * rd_kafka_topic_result_name(const rd_kafka_topic_result_t *topicres)
Definition: rdkafka.h:572
Definition: rdkafka.h:433
Definition: rdkafka.h:7564
Definition: rdkafka.h:7588
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.
Definition: rdkafka.h:358
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_mem_free(rd_kafka_t *rk, void *ptr)
Free pointer returned by librdkafka.
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 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_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 void rd_kafka_error_destroy(rd_kafka_error_t *error)
Free and destroy an error object.
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 char * rd_kafka_clusterid(rd_kafka_t *rk, int timeout_ms)
Returns the ClusterId as reported in broker metadata.
RD_EXPORT const char * rd_kafka_ResourceType_name(rd_kafka_ResourceType_t restype)
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.
Definition: rdkafka.h:398
Definition: rdkafka.h:7556
RD_EXPORT void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist)
Release list memory.
RD_EXPORT void rd_kafka_queue_yield(rd_kafka_queue_t *rkqu)
Cancels the current rd_kafka_queue_poll() on rkqu.
Definition: rdkafka.h:502
RD_EXPORT void rd_kafka_event_destroy(rd_kafka_event_t *rkev)
Destroy an event.
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_committed(rd_kafka_t *rk, rd_kafka_topic_partition_list_t *partitions, int timeout_ms)
Retrieve committed offsets for topics+partitions.
RD_EXPORT rd_kafka_ConfigResource_t * rd_kafka_ConfigResource_new(rd_kafka_ResourceType_t restype, const char *resname)
Create new ConfigResource object.
Definition: rdkafka.h:221
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 int rd_kafka_event_debug_contexts(rd_kafka_event_t *rkev, char *dst, size_t dstsize)
Extract log debug context from event.
Definition: rdkafka.h:400
RD_EXPORT int rd_kafka_event_error_is_fatal(rd_kafka_event_t *rkev)
RD_EXPORT rd_kafka_conf_t * rd_kafka_conf_new(void)
Create configuration object.
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_DeleteAcls_result_response_s rd_kafka_DeleteAcls_result_response_t
Definition: rdkafka.h:7814
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 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 const char * rd_kafka_err2name(rd_kafka_resp_err_t err)
Returns the error code name (enum name).
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...
Definition: rdkafka.h:1570
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 int rd_kafka_version(void)
Returns the librdkafka version as integer.
Definition: rdkafka.h:490
Definition: rdkafka.h:7042
void * member_assignment
Definition: rdkafka.h:4843
RD_EXPORT const rd_kafka_conf_t * rd_kafka_conf(rd_kafka_t *rk)
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 int rd_kafka_topic_partition_list_del(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Delete partition from list.
int size
Definition: rdkafka.h:919
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.
Definition: rdkafka.h:1575
RD_EXPORT void rd_kafka_topic_partition_destroy(rd_kafka_topic_partition_t *rktpar)
Destroy a rd_kafka_topic_partition_t.
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.
Definition: rdkafka.h:380
Definition: rdkafka.h:1608
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_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_consumer_group_metadata_destroy(rd_kafka_consumer_group_metadata_t *)
Frees the consumer group metadata object as returned by rd_kafka_consumer_group_metadata().
RD_EXPORT rd_kafka_ResourcePatternType_t rd_kafka_AclBinding_resource_pattern_type(const rd_kafka_AclBinding_t *acl)
Definition: rdkafka.h:7552
RD_EXPORT rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev)
Definition: rdkafka.h:350
char * topic
Definition: rdkafka.h:893
RD_EXPORT const char * rd_kafka_get_debug_contexts(void)
Retrieve supported debug contexts for use with the "debug" configuration property. (runtime)
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)...
Definition: rdkafka.h:356
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 rd_kafka_DeleteGroups_result_t * rd_kafka_event_DeleteGroups_result(rd_kafka_event_t *rkev)
Get DeleteGroups result.
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.
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:6406
Definition: rdkafka.h:1087
RD_EXPORT int rd_kafka_queue_poll_callback(rd_kafka_queue_t *rkqu, int timeout_ms)
Poll a queue for events served through callbacks for max timeout_ms.
rd_kafka_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:5850
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...
Definition: rdkafka.h:368
RD_EXPORT rd_kafka_error_t * rd_kafka_begin_transaction(rd_kafka_t *rk)
Begin a new transaction.
Definition: rdkafka.h:6391
RD_EXPORT int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms)
Polls the provided kafka handle for events.
Definition: rdkafka.h:5073
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 const rd_kafka_error_t * rd_kafka_acl_result_error(const rd_kafka_acl_result_t *aclres)
Definition: rdkafka.h:208
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 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_ConfigResource_s rd_kafka_ConfigResource_t
Definition: rdkafka.h:7059
RD_EXPORT const char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
Definition: rdkafka.h:7548
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_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.
Definition: rdkafka.h:506
Definition: rdkafka.h:460
RD_EXPORT const char * rd_kafka_event_config_string(rd_kafka_event_t *rkev)
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 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 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 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 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.
Definition: rdkafka.h:209
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_EXPORT int rd_kafka_assignment_lost(rd_kafka_t *rk)
Check whether the consumer considers the current assignment to have been lost involuntarily. This method is only applicable for use with a high level subscribing consumer. Assignments are revoked immediately when determined to have been lost, so this method is only useful when reacting to a RD_KAFKA_EVENT_REBALANCE event or from within a rebalance_cb. Partitions that have been lost may already be owned by other members in the group and therefore commiting offsets, for example, may fail.
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() ...
Definition: rdkafka.h:220
Definition: rdkafka.h:318
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:5941
Definition: rdkafka.h:7024
rd_kafka_event_t rd_kafka_DeleteGroups_result_t
Definition: rdkafka.h:5410
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 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))
Definition: rdkafka.h:314
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.
Definition: rdkafka.h:7023
Definition: rdkafka.h:415
Definition: rdkafka.h:7557
struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t
Definition: rdkafka.h:6946
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_get_sasl(rd_kafka_t *rk)
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_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 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.
Definition: rdkafka.h:287
Metadata container.
Definition: rdkafka.h:4770
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.
Definition: rdkafka.h:539
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:5997
Definition: rdkafka.h:348
RD_EXPORT rd_kafka_resp_err_t rd_kafka_unsubscribe(rd_kafka_t *rk)
Unsubscribe from the current subscription set.
rd_kafka_event_t rd_kafka_DeleteRecords_result_t
Definition: rdkafka.h:5408
RD_EXPORT void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata)
Release metadata memory.
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_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_EXPORT rd_kafka_AclOperation_t rd_kafka_AclBinding_operation(const rd_kafka_AclBinding_t *acl)
rd_kafka_cert_enc_t
SSL certificate encoding.
Definition: rdkafka.h:2325
Definition: rdkafka.h:7553
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 rd_kafka_ResourceType_t rd_kafka_ConfigResource_type(const rd_kafka_ConfigResource_t *config)
Definition: rdkafka.h:586
Definition: rdkafka.h:340
RD_EXPORT rd_kafka_resp_err_t rd_kafka_producev(rd_kafka_t *rk,...)
Produce and send a single message to broker.
Definition: rdkafka.h:396
Definition: rdkafka.h:504
Definition: rdkafka.h:7036
RD_EXPORT rd_kafka_AclPermissionType_t rd_kafka_AclBinding_permission_type(const rd_kafka_AclBinding_t *acl)
RD_EXPORT const char * rd_kafka_AclBinding_principal(const rd_kafka_AclBinding_t *acl)
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 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 const rd_kafka_CreateTopics_result_t * rd_kafka_event_CreateTopics_result(rd_kafka_event_t *rkev)
Get CreateTopics result.
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 const rd_kafka_topic_partition_list_t * rd_kafka_group_result_partitions(const rd_kafka_group_result_t *groupres)
RD_EXPORT size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev)
rd_kafka_event_t rd_kafka_DescribeAcls_result_t
Definition: rdkafka.h:5398
Definition: rdkafka.h:1604
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.
Definition: rdkafka.h:312
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 char * rd_kafka_group_result_name(const rd_kafka_group_result_t *groupres)
RD_EXPORT int32_t rd_kafka_msg_partitioner_murmur2_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *rkt_opaque, void *msg_opaque)
Consistent-Random Murmur2 partitioner (Java compatible).
rd_kafka_thread_type_t
librdkafka internal thread type.
Definition: rdkafka.h:5071
Definition: rdkafka.h:219
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.
Definition: rdkafka.h:336
RD_EXPORT int rd_kafka_ConfigEntry_is_synonym(const rd_kafka_ConfigEntry_t *entry)
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 rd_kafka_DeleteTopic_t * rd_kafka_DeleteTopic_new(const char *topic)
Create a new DeleteTopic object. This object is later passed to rd_kafka_DeleteTopics().
rd_kafka_timestamp_type_t
Definition: rdkafka.h:218
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_event_opaque(rd_kafka_event_t *rkev)
RD_EXPORT const char * rd_kafka_topic_result_error_string(const rd_kafka_topic_result_t *topicres)
Definition: rdkafka.h:2326
struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t
Definition: rdkafka.h:6711
rd_kafka_vtype_t vtype
Definition: rdkafka.h:1113
Definition: rdkafka.h:593
RD_EXPORT int rd_kafka_error_is_fatal(const rd_kafka_error_t *error)
RD_EXPORT const char * rd_kafka_name(const rd_kafka_t *rk)
Returns Kafka handle name.
RD_EXPORT const char * rd_kafka_AclBinding_name(const rd_kafka_AclBinding_t *acl)
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1408
Definition: rdkafka.h:324
Definition: rdkafka.h:7587
Definition: rdkafka.h:384
rd_kafka_ResourcePatternType_t
Apache Kafka pattern types.
Definition: rdkafka.h:7032
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_AclBinding_destroy(rd_kafka_AclBinding_t *acl_binding)
Destroy and free an AclBinding object previously created with rd_kafka_AclBinding_new() ...
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_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 conusmer group. This will succeed at the partit...
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().
int32_t partition
Definition: rdkafka.h:1411
Definition: rdkafka.h:407
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 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_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.
Definition: rdkafka.h:328
RD_EXPORT rd_kafka_message_t * rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms)
Poll the consumer for messages or events.
Definition: rdkafka.h:566
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_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()...
Definition: rdkafka.h:304
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 void rd_kafka_DeleteTopic_destroy(rd_kafka_DeleteTopic_t *del_topic)
Destroy and free a DeleteTopic object previously created with rd_kafka_DeleteTopic_new() ...
Definition: rdkafka.h:1102
char * member_id
Definition: rdkafka.h:4837
List of groups.
Definition: rdkafka.h:4867
struct rd_kafka_DeleteGroup_s rd_kafka_DeleteGroup_t
Definition: rdkafka.h:7363
Definition: rdkafka.h:535
Definition: rdkafka.h:1092
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreatePartitions_result_topics(const rd_kafka_CreatePartitions_result_t *result, size_t *cntp)
Get an array of topic results from a CreatePartitions result.
RD_EXPORT const char * rd_kafka_err2str(rd_kafka_resp_err_t err)
Returns a human readable representation of a kafka error.
RD_EXPORT int rd_kafka_ConfigEntry_is_sensitive(const rd_kafka_ConfigEntry_t *entry)
Definition: rdkafka.h:564
RD_EXPORT const char * rd_kafka_AclBinding_host(const rd_kafka_AclBinding_t *acl)
RD_EXPORT rd_kafka_resp_err_t rd_kafka_error_code(const rd_kafka_error_t *error)
void * metadata
Definition: rdkafka.h:896
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.
Definition: rdkafka.h:6918
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_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...
int64_t offset
Definition: rdkafka.h:895
RD_EXPORT rd_kafka_topic_partition_list_t * rd_kafka_event_topic_partition_list(rd_kafka_event_t *rkev)
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...
Definition: rdkafka.h:285
RD_EXPORT void rd_kafka_conf_destroy(rd_kafka_conf_t *conf)
Destroys a conf object.
Definition: rdkafka.h:597
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.
Definition: rdkafka.h:346
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 void rd_kafka_DeleteGroup_destroy(rd_kafka_DeleteGroup_t *del_group)
Destroy and free a DeleteGroup object previously created with rd_kafka_DeleteGroup_new() ...
Broker information.
Definition: rdkafka.h:4737
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:637
Definition: rdkafka.h:486
RD_EXPORT const char * rd_kafka_ConfigEntry_value(const rd_kafka_ConfigEntry_t *entry)
Definition: rdkafka.h:466
Definition: rdkafka.h:6379
rd_kafka_resp_err_t err
Definition: rdkafka.h:899
Definition: rdkafka.h:344
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 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 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.
Definition: rdkafka.h:370
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 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 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...
Topic+Partition place holder.
Definition: rdkafka.h:892
Definition: rdkafka.h:283
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.
Definition: rdkafka.h:611
RD_EXPORT int rd_kafka_error_is_retriable(const rd_kafka_error_t *error)
Definition: rdkafka.h:374
struct rd_kafka_group_member_info * members
Definition: rdkafka.h:4858
rd_kafka_event_t rd_kafka_DeleteAcls_result_t
Definition: rdkafka.h:5400
size_t metadata_size
Definition: rdkafka.h:897
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", "value" pairs.
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 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 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...
int member_metadata_size
Definition: rdkafka.h:4842
rd_kafka_AclOperation_t
Apache Kafka ACL operation types.
Definition: rdkafka.h:7547
Definition: rdkafka.h:6382
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.
Definition: rdkafka.h:1091
Definition: rdkafka.h:6934
Definition: rdkafka.h:7020
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_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:1561
rd_kafka_type_t
rd_kafka_t handle type.
Definition: rdkafka.h:207
Definition: rdkafka.h:7038
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 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.
Definition: rdkafka.h:394
Definition: rdkafka.h:427
RD_EXPORT void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf)
Destroys a topic conf object.
Definition: rdkafka.h:6389
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_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.
Definition: rdkafka.h:496
void * payload
Definition: rdkafka.h:1412
Definition: rdkafka.h:6383
RD_EXPORT void rd_kafka_yield(rd_kafka_t *rk)
Cancels the current callback dispatcher (rd_kafka_poll(), rd_kafka_consume_callback(), etc).
Definition: rdkafka.h:417
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 const char * rd_kafka_event_stats(rd_kafka_event_t *rkev)
Extract stats from the 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 ...
void * member_metadata
Definition: rdkafka.h:4840
RD_EXPORT rd_kafka_resp_err_t rd_kafka_consumer_close(rd_kafka_t *rk)
Close the consumer.
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_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 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().
void * _private
Definition: rdkafka.h:1431
RD_EXPORT int rd_kafka_ConfigEntry_is_default(const rd_kafka_ConfigEntry_t *entry)
Definition: rdkafka.h:302
RD_EXPORT const char * rd_kafka_topic_name(const rd_kafka_topic_t *rkt)
Returns the topic name.
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 ...
char * protocol
Definition: rdkafka.h:4857
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_DeleteConsumerGroupOffsets_destroy(rd_kafka_DeleteConsumerGroupOffsets_t *del_grpoffsets)
Destroy and free a DeleteConsumerGroupOffsets object previously created with rd_kafka_DeleteConsumerG...
rd_kafka_cert_type_t
SSL certificate type.
Definition: rdkafka.h:2311
Definition: rdkafka.h:5072
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_EXPORT rd_kafka_message_t * rd_kafka_consume_queue(rd_kafka_queue_t *rkqu, int timeout_ms)
Consume from queue.
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 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 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt, int32_t partition, int64_t offset, rd_kafka_queue_t *rkqu)
Same as rd_kafka_consume_start() but re-routes incoming messages to the provided queue rkqu (which mu...
RD_EXPORT rd_kafka_queue_t * rd_kafka_queue_new(rd_kafka_t *rk)
Create a new message queue.
Definition: rdkafka.h:604
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.
Definition: rdkafka.h:6388
Definition: rdkafka.h:1100
Definition: rdkafka.h:615
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...