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-2018 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 _MSC_VER
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 typedef SSIZE_T ssize_t;
65 #define RD_UNUSED
66 #define RD_INLINE __inline
67 #define RD_DEPRECATED __declspec(deprecated)
68 #undef RD_EXPORT
69 #ifdef LIBRDKAFKA_STATICLIB
70 #define RD_EXPORT
71 #else
72 #ifdef LIBRDKAFKA_EXPORTS
73 #define RD_EXPORT __declspec(dllexport)
74 #else
75 #define RD_EXPORT __declspec(dllimport)
76 #endif
77 #ifndef LIBRDKAFKA_TYPECHECKS
78 #define LIBRDKAFKA_TYPECHECKS 0
79 #endif
80 #endif
81 
82 #else
83 #include <sys/socket.h> /* for sockaddr, .. */
84 
85 #define RD_UNUSED __attribute__((unused))
86 #define RD_INLINE inline
87 #define RD_EXPORT
88 #define RD_DEPRECATED __attribute__((deprecated))
89 
90 #ifndef LIBRDKAFKA_TYPECHECKS
91 #define LIBRDKAFKA_TYPECHECKS 1
92 #endif
93 #endif
94 
95 
101 #if LIBRDKAFKA_TYPECHECKS
102 #define _LRK_TYPECHECK(RET,TYPE,ARG) \
103  ({ if (0) { TYPE __t RD_UNUSED = (ARG); } RET; })
104 
105 #define _LRK_TYPECHECK2(RET,TYPE,ARG,TYPE2,ARG2) \
106  ({ \
107  if (0) { \
108  TYPE __t RD_UNUSED = (ARG); \
109  TYPE2 __t2 RD_UNUSED = (ARG2); \
110  } \
111  RET; })
112 
113 #define _LRK_TYPECHECK3(RET,TYPE,ARG,TYPE2,ARG2,TYPE3,ARG3) \
114  ({ \
115  if (0) { \
116  TYPE __t RD_UNUSED = (ARG); \
117  TYPE2 __t2 RD_UNUSED = (ARG2); \
118  TYPE3 __t3 RD_UNUSED = (ARG3); \
119  } \
120  RET; })
121 #else
122 #define _LRK_TYPECHECK(RET,TYPE,ARG) (RET)
123 #define _LRK_TYPECHECK2(RET,TYPE,ARG,TYPE2,ARG2) (RET)
124 #define _LRK_TYPECHECK3(RET,TYPE,ARG,TYPE2,ARG2,TYPE3,ARG3) (RET)
125 #endif
126 
127 /* @endcond */
128 
129 
151 #define RD_KAFKA_VERSION 0x000b06ff
152 
161 RD_EXPORT
162 int rd_kafka_version(void);
163 
169 RD_EXPORT
170 const char *rd_kafka_version_str (void);
171 
190 typedef enum rd_kafka_type_t {
194 
195 
206 
207 
208 
215 RD_EXPORT
216 const char *rd_kafka_get_debug_contexts(void);
217 
225 #define RD_KAFKA_DEBUG_CONTEXTS \
226  "all,generic,broker,topic,metadata,feature,queue,msg,protocol,cgrp,security,fetch,interceptor,plugin,consumer,admin"
227 
228 
229 /* @cond NO_DOC */
230 /* Private types to provide ABI compatibility */
231 typedef struct rd_kafka_s rd_kafka_t;
232 typedef struct rd_kafka_topic_s rd_kafka_topic_t;
233 typedef struct rd_kafka_conf_s rd_kafka_conf_t;
234 typedef struct rd_kafka_topic_conf_s rd_kafka_topic_conf_t;
235 typedef struct rd_kafka_queue_s rd_kafka_queue_t;
236 typedef struct rd_kafka_op_s rd_kafka_event_t;
237 typedef struct rd_kafka_topic_result_s rd_kafka_topic_result_t;
238 /* @endcond */
239 
240 
253 typedef enum {
254  /* Internal errors to rdkafka: */
350 
353 
354  /* Kafka broker errors: */
474 
475  RD_KAFKA_RESP_ERR_END_ALL,
477 
478 
486  const char *name;
487  const char *desc;
488 };
489 
490 
494 RD_EXPORT
495 void rd_kafka_get_err_descs (const struct rd_kafka_err_desc **errdescs,
496  size_t *cntp);
497 
498 
499 
500 
506 RD_EXPORT
507 const char *rd_kafka_err2str (rd_kafka_resp_err_t err);
508 
509 
510 
516 RD_EXPORT
517 const char *rd_kafka_err2name (rd_kafka_resp_err_t err);
518 
519 
545 RD_EXPORT
547 
548 
573 RD_EXPORT RD_DEPRECATED
575 
576 
589 RD_EXPORT RD_DEPRECATED
590 int rd_kafka_errno (void);
591 
592 
593 
609 typedef struct rd_kafka_topic_partition_s {
610  char *topic;
611  int32_t partition;
612  int64_t offset;
613  void *metadata;
614  size_t metadata_size;
615  void *opaque;
617  void *_private;
620 
621 
626 RD_EXPORT
628 
629 
634 typedef struct rd_kafka_topic_partition_list_s {
635  int cnt;
636  int size;
639 
640 
655 RD_EXPORT
657 
658 
662 RD_EXPORT
663 void
665 
675 RD_EXPORT
678  const char *topic, int32_t partition);
679 
680 
689 RD_EXPORT
690 void
692  *rktparlist,
693  const char *topic,
694  int32_t start, int32_t stop);
695 
696 
697 
709 RD_EXPORT
710 int
712  const char *topic, int32_t partition);
713 
714 
722 RD_EXPORT
723 int
726  int idx);
727 
728 
736 RD_EXPORT
739 
740 
741 
742 
750 RD_EXPORT
753  const char *topic, int32_t partition, int64_t offset);
754 
755 
756 
762 RD_EXPORT
765  const char *topic, int32_t partition);
766 
767 
775 RD_EXPORT void
777  int (*cmp) (const void *a, const void *b,
778  void *opaque),
779  void *opaque);
780 
781 
799 typedef enum rd_kafka_vtype_t {
813 
814 
823 #define RD_KAFKA_V_END RD_KAFKA_VTYPE_END
824 
828 #define RD_KAFKA_V_TOPIC(topic) \
829  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TOPIC, const char *, topic), \
830  (const char *)topic
831 
834 #define RD_KAFKA_V_RKT(rkt) \
835  _LRK_TYPECHECK(RD_KAFKA_VTYPE_RKT, rd_kafka_topic_t *, rkt), \
836  (rd_kafka_topic_t *)rkt
837 
840 #define RD_KAFKA_V_PARTITION(partition) \
841  _LRK_TYPECHECK(RD_KAFKA_VTYPE_PARTITION, int32_t, partition), \
842  (int32_t)partition
843 
846 #define RD_KAFKA_V_VALUE(VALUE,LEN) \
847  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_VALUE, void *, VALUE, size_t, LEN), \
848  (void *)VALUE, (size_t)LEN
849 
852 #define RD_KAFKA_V_KEY(KEY,LEN) \
853  _LRK_TYPECHECK2(RD_KAFKA_VTYPE_KEY, const void *, KEY, size_t, LEN), \
854  (void *)KEY, (size_t)LEN
855 
859 #define RD_KAFKA_V_OPAQUE(opaque) \
860  _LRK_TYPECHECK(RD_KAFKA_VTYPE_OPAQUE, void *, opaque), \
861  (void *)opaque
862 
866 #define RD_KAFKA_V_MSGFLAGS(msgflags) \
867  _LRK_TYPECHECK(RD_KAFKA_VTYPE_MSGFLAGS, int, msgflags), \
868  (int)msgflags
869 
872 #define RD_KAFKA_V_TIMESTAMP(timestamp) \
873  _LRK_TYPECHECK(RD_KAFKA_VTYPE_TIMESTAMP, int64_t, timestamp), \
874  (int64_t)timestamp
875 
881 #define RD_KAFKA_V_HEADER(NAME,VALUE,LEN) \
882  _LRK_TYPECHECK3(RD_KAFKA_VTYPE_HEADER, const char *, NAME, \
883  const void *, VALUE, ssize_t, LEN), \
884  (const char *)NAME, (const void *)VALUE, (ssize_t)LEN
885 
895 #define RD_KAFKA_V_HEADERS(HDRS) \
896  _LRK_TYPECHECK(RD_KAFKA_VTYPE_HEADERS, rd_kafka_headers_t *, HDRS), \
897  (rd_kafka_headers_t *)HDRS
898 
899 
922 typedef struct rd_kafka_headers_s rd_kafka_headers_t;
923 
931 RD_EXPORT rd_kafka_headers_t *rd_kafka_headers_new (size_t initial_count);
932 
937 RD_EXPORT void rd_kafka_headers_destroy (rd_kafka_headers_t *hdrs);
938 
942 RD_EXPORT rd_kafka_headers_t *
943 rd_kafka_headers_copy (const rd_kafka_headers_t *src);
944 
961 RD_EXPORT rd_kafka_resp_err_t
962 rd_kafka_header_add (rd_kafka_headers_t *hdrs,
963  const char *name, ssize_t name_size,
964  const void *value, ssize_t value_size);
965 
973 RD_EXPORT rd_kafka_resp_err_t
974 rd_kafka_header_remove (rd_kafka_headers_t *hdrs, const char *name);
975 
976 
993 RD_EXPORT rd_kafka_resp_err_t
994 rd_kafka_header_get_last (const rd_kafka_headers_t *hdrs,
995  const char *name, const void **valuep, size_t *sizep);
996 
1010 RD_EXPORT rd_kafka_resp_err_t
1011 rd_kafka_header_get (const rd_kafka_headers_t *hdrs, size_t idx,
1012  const char *name, const void **valuep, size_t *sizep);
1013 
1014 
1022 RD_EXPORT rd_kafka_resp_err_t
1023 rd_kafka_header_get_all (const rd_kafka_headers_t *hdrs, size_t idx,
1024  const char **namep,
1025  const void **valuep, size_t *sizep);
1026 
1027 
1028 
1041 // FIXME: This doesn't show up in docs for some reason
1042 // "Compound rd_kafka_message_t is not documented."
1043 
1057 typedef struct rd_kafka_message_s {
1059  rd_kafka_topic_t *rkt;
1060  int32_t partition;
1061  void *payload;
1065  size_t len;
1068  void *key;
1070  size_t key_len;
1072  int64_t offset;
1082  void *_private;
1087 
1088 
1092 RD_EXPORT
1094 
1095 
1096 
1097 
1104 static RD_INLINE const char *
1105 RD_UNUSED
1107  if (!rkmessage->err)
1108  return NULL;
1109 
1110  if (rkmessage->payload)
1111  return (const char *)rkmessage->payload;
1112 
1113  return rd_kafka_err2str(rkmessage->err);
1114 }
1115 
1116 
1117 
1129 RD_EXPORT
1130 int64_t rd_kafka_message_timestamp (const rd_kafka_message_t *rkmessage,
1131  rd_kafka_timestamp_type_t *tstype);
1132 
1133 
1134 
1141 RD_EXPORT
1142 int64_t rd_kafka_message_latency (const rd_kafka_message_t *rkmessage);
1143 
1144 
1161 RD_EXPORT rd_kafka_resp_err_t
1163  rd_kafka_headers_t **hdrsp);
1164 
1176 RD_EXPORT rd_kafka_resp_err_t
1178  rd_kafka_headers_t **hdrsp);
1179 
1180 
1192 RD_EXPORT
1194  rd_kafka_headers_t *hdrs);
1195 
1196 
1202 RD_EXPORT size_t rd_kafka_header_cnt (const rd_kafka_headers_t *hdrs);
1203 
1204 
1220 typedef enum {
1225 
1226 
1257 RD_EXPORT
1258 rd_kafka_conf_t *rd_kafka_conf_new(void);
1259 
1260 
1264 RD_EXPORT
1265 void rd_kafka_conf_destroy(rd_kafka_conf_t *conf);
1266 
1267 
1274 RD_EXPORT
1275 rd_kafka_conf_t *rd_kafka_conf_dup(const rd_kafka_conf_t *conf);
1276 
1277 
1282 RD_EXPORT
1283 rd_kafka_conf_t *rd_kafka_conf_dup_filter (const rd_kafka_conf_t *conf,
1284  size_t filter_cnt,
1285  const char **filter);
1286 
1287 
1288 
1305 RD_EXPORT
1306 rd_kafka_conf_res_t rd_kafka_conf_set(rd_kafka_conf_t *conf,
1307  const char *name,
1308  const char *value,
1309  char *errstr, size_t errstr_size);
1310 
1311 
1317 RD_EXPORT
1318 void rd_kafka_conf_set_events(rd_kafka_conf_t *conf, int events);
1319 
1320 
1356 RD_EXPORT void
1357 rd_kafka_conf_set_background_event_cb (rd_kafka_conf_t *conf,
1358  void (*event_cb) (rd_kafka_t *rk,
1359  rd_kafka_event_t *rkev,
1360  void *opaque));
1361 
1362 
1366 RD_EXPORT
1367 void rd_kafka_conf_set_dr_cb(rd_kafka_conf_t *conf,
1368  void (*dr_cb) (rd_kafka_t *rk,
1369  void *payload, size_t len,
1370  rd_kafka_resp_err_t err,
1371  void *opaque, void *msg_opaque));
1372 
1389 RD_EXPORT
1390 void rd_kafka_conf_set_dr_msg_cb(rd_kafka_conf_t *conf,
1391  void (*dr_msg_cb) (rd_kafka_t *rk,
1392  const rd_kafka_message_t *
1393  rkmessage,
1394  void *opaque));
1395 
1396 
1401 RD_EXPORT
1402 void rd_kafka_conf_set_consume_cb (rd_kafka_conf_t *conf,
1403  void (*consume_cb) (rd_kafka_message_t *
1404  rkmessage,
1405  void *opaque));
1406 
1478 RD_EXPORT
1480  rd_kafka_conf_t *conf,
1481  void (*rebalance_cb) (rd_kafka_t *rk,
1482  rd_kafka_resp_err_t err,
1483  rd_kafka_topic_partition_list_t *partitions,
1484  void *opaque));
1485 
1486 
1487 
1502 RD_EXPORT
1504  rd_kafka_conf_t *conf,
1505  void (*offset_commit_cb) (rd_kafka_t *rk,
1506  rd_kafka_resp_err_t err,
1508  void *opaque));
1509 
1510 
1520 RD_EXPORT
1521 void rd_kafka_conf_set_error_cb(rd_kafka_conf_t *conf,
1522  void (*error_cb) (rd_kafka_t *rk, int err,
1523  const char *reason,
1524  void *opaque));
1525 
1540 RD_EXPORT
1541 void rd_kafka_conf_set_throttle_cb (rd_kafka_conf_t *conf,
1542  void (*throttle_cb) (
1543  rd_kafka_t *rk,
1544  const char *broker_name,
1545  int32_t broker_id,
1546  int throttle_time_ms,
1547  void *opaque));
1548 
1549 
1566 RD_EXPORT
1567 void rd_kafka_conf_set_log_cb(rd_kafka_conf_t *conf,
1568  void (*log_cb) (const rd_kafka_t *rk, int level,
1569  const char *fac, const char *buf));
1570 
1571 
1593 RD_EXPORT
1594 void rd_kafka_conf_set_stats_cb(rd_kafka_conf_t *conf,
1595  int (*stats_cb) (rd_kafka_t *rk,
1596  char *json,
1597  size_t json_len,
1598  void *opaque));
1599 
1600 
1601 
1616 RD_EXPORT
1617 void rd_kafka_conf_set_socket_cb(rd_kafka_conf_t *conf,
1618  int (*socket_cb) (int domain, int type,
1619  int protocol,
1620  void *opaque));
1621 
1622 
1623 
1636 RD_EXPORT void
1637 rd_kafka_conf_set_connect_cb (rd_kafka_conf_t *conf,
1638  int (*connect_cb) (int sockfd,
1639  const struct sockaddr *addr,
1640  int addrlen,
1641  const char *id,
1642  void *opaque));
1643 
1651 RD_EXPORT void
1652 rd_kafka_conf_set_closesocket_cb (rd_kafka_conf_t *conf,
1653  int (*closesocket_cb) (int sockfd,
1654  void *opaque));
1655 
1656 
1657 
1658 #ifndef _MSC_VER
1659 
1673 RD_EXPORT
1674 void rd_kafka_conf_set_open_cb (rd_kafka_conf_t *conf,
1675  int (*open_cb) (const char *pathname,
1676  int flags, mode_t mode,
1677  void *opaque));
1678 #endif
1679 
1683 RD_EXPORT
1684 void rd_kafka_conf_set_opaque(rd_kafka_conf_t *conf, void *opaque);
1685 
1689 RD_EXPORT
1690 void *rd_kafka_opaque(const rd_kafka_t *rk);
1691 
1692 
1693 
1699 RD_EXPORT
1700 void rd_kafka_conf_set_default_topic_conf (rd_kafka_conf_t *conf,
1701  rd_kafka_topic_conf_t *tconf);
1702 
1703 
1704 
1724 RD_EXPORT
1725 rd_kafka_conf_res_t rd_kafka_conf_get (const rd_kafka_conf_t *conf,
1726  const char *name,
1727  char *dest, size_t *dest_size);
1728 
1729 
1735 RD_EXPORT
1736 rd_kafka_conf_res_t rd_kafka_topic_conf_get (const rd_kafka_topic_conf_t *conf,
1737  const char *name,
1738  char *dest, size_t *dest_size);
1739 
1740 
1749 RD_EXPORT
1750 const char **rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp);
1751 
1752 
1761 RD_EXPORT
1762 const char **rd_kafka_topic_conf_dump(rd_kafka_topic_conf_t *conf,
1763  size_t *cntp);
1764 
1769 RD_EXPORT
1770 void rd_kafka_conf_dump_free(const char **arr, size_t cnt);
1771 
1776 RD_EXPORT
1777 void rd_kafka_conf_properties_show(FILE *fp);
1778 
1796 RD_EXPORT
1797 rd_kafka_topic_conf_t *rd_kafka_topic_conf_new(void);
1798 
1799 
1803 RD_EXPORT
1804 rd_kafka_topic_conf_t *rd_kafka_topic_conf_dup(const rd_kafka_topic_conf_t
1805  *conf);
1806 
1811 RD_EXPORT
1812 rd_kafka_topic_conf_t *rd_kafka_default_topic_conf_dup (rd_kafka_t *rk);
1813 
1814 
1818 RD_EXPORT
1819 void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf);
1820 
1821 
1830 RD_EXPORT
1831 rd_kafka_conf_res_t rd_kafka_topic_conf_set(rd_kafka_topic_conf_t *conf,
1832  const char *name,
1833  const char *value,
1834  char *errstr, size_t errstr_size);
1835 
1840 RD_EXPORT
1841 void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf, void *opaque);
1842 
1843 
1858 RD_EXPORT
1859 void
1860 rd_kafka_topic_conf_set_partitioner_cb (rd_kafka_topic_conf_t *topic_conf,
1861  int32_t (*partitioner) (
1862  const rd_kafka_topic_t *rkt,
1863  const void *keydata,
1864  size_t keylen,
1865  int32_t partition_cnt,
1866  void *rkt_opaque,
1867  void *msg_opaque));
1868 
1869 
1895 RD_EXPORT void
1896 rd_kafka_topic_conf_set_msg_order_cmp (rd_kafka_topic_conf_t *topic_conf,
1897  int (*msg_order_cmp) (
1898  const rd_kafka_message_t *a,
1899  const rd_kafka_message_t *b));
1900 
1901 
1909 RD_EXPORT
1910 int rd_kafka_topic_partition_available(const rd_kafka_topic_t *rkt,
1911  int32_t partition);
1912 
1913 
1914 /*******************************************************************
1915  * *
1916  * Partitioners provided by rdkafka *
1917  * *
1918  *******************************************************************/
1919 
1928 RD_EXPORT
1929 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt,
1930  const void *key, size_t keylen,
1931  int32_t partition_cnt,
1932  void *opaque, void *msg_opaque);
1933 
1942 RD_EXPORT
1943 int32_t rd_kafka_msg_partitioner_consistent (const rd_kafka_topic_t *rkt,
1944  const void *key, size_t keylen,
1945  int32_t partition_cnt,
1946  void *opaque, void *msg_opaque);
1947 
1958 RD_EXPORT
1959 int32_t rd_kafka_msg_partitioner_consistent_random (const rd_kafka_topic_t *rkt,
1960  const void *key, size_t keylen,
1961  int32_t partition_cnt,
1962  void *opaque, void *msg_opaque);
1963 
1964 
1973 RD_EXPORT
1974 int32_t rd_kafka_msg_partitioner_murmur2 (const rd_kafka_topic_t *rkt,
1975  const void *key, size_t keylen,
1976  int32_t partition_cnt,
1977  void *rkt_opaque,
1978  void *msg_opaque);
1979 
1989 RD_EXPORT
1990 int32_t rd_kafka_msg_partitioner_murmur2_random (const rd_kafka_topic_t *rkt,
1991  const void *key, size_t keylen,
1992  int32_t partition_cnt,
1993  void *rkt_opaque,
1994  void *msg_opaque);
1995 
1996 
2037 RD_EXPORT
2038 rd_kafka_t *rd_kafka_new(rd_kafka_type_t type, rd_kafka_conf_t *conf,
2039  char *errstr, size_t errstr_size);
2040 
2041 
2056 RD_EXPORT
2057 void rd_kafka_destroy(rd_kafka_t *rk);
2058 
2059 
2064 RD_EXPORT
2065 void rd_kafka_destroy_flags (rd_kafka_t *rk, int flags);
2066 
2081 #define RD_KAFKA_DESTROY_F_NO_CONSUMER_CLOSE 0x8
2082 
2083 
2084 
2088 RD_EXPORT
2089 const char *rd_kafka_name(const rd_kafka_t *rk);
2090 
2091 
2095 RD_EXPORT
2096 rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk);
2097 
2098 
2109 RD_EXPORT
2110 char *rd_kafka_memberid (const rd_kafka_t *rk);
2111 
2112 
2113 
2131 RD_EXPORT
2132 char *rd_kafka_clusterid (rd_kafka_t *rk, int timeout_ms);
2133 
2134 
2149 RD_EXPORT
2150 int32_t rd_kafka_controllerid (rd_kafka_t *rk, int timeout_ms);
2151 
2152 
2174 RD_EXPORT
2175 rd_kafka_topic_t *rd_kafka_topic_new(rd_kafka_t *rk, const char *topic,
2176  rd_kafka_topic_conf_t *conf);
2177 
2178 
2179 
2188 RD_EXPORT
2189 void rd_kafka_topic_destroy(rd_kafka_topic_t *rkt);
2190 
2191 
2195 RD_EXPORT
2196 const char *rd_kafka_topic_name(const rd_kafka_topic_t *rkt);
2197 
2198 
2202 RD_EXPORT
2203 void *rd_kafka_topic_opaque (const rd_kafka_topic_t *rkt);
2204 
2205 
2212 #define RD_KAFKA_PARTITION_UA ((int32_t)-1)
2213 
2214 
2240 RD_EXPORT
2241 int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms);
2242 
2243 
2254 RD_EXPORT
2255 void rd_kafka_yield (rd_kafka_t *rk);
2256 
2257 
2258 
2259 
2267 RD_EXPORT rd_kafka_resp_err_t
2268 rd_kafka_pause_partitions (rd_kafka_t *rk,
2269  rd_kafka_topic_partition_list_t *partitions);
2270 
2271 
2272 
2280 RD_EXPORT rd_kafka_resp_err_t
2281 rd_kafka_resume_partitions (rd_kafka_t *rk,
2282  rd_kafka_topic_partition_list_t *partitions);
2283 
2284 
2285 
2286 
2295 RD_EXPORT rd_kafka_resp_err_t
2296 rd_kafka_query_watermark_offsets (rd_kafka_t *rk,
2297  const char *topic, int32_t partition,
2298  int64_t *low, int64_t *high, int timeout_ms);
2299 
2300 
2317 RD_EXPORT rd_kafka_resp_err_t
2318 rd_kafka_get_watermark_offsets (rd_kafka_t *rk,
2319  const char *topic, int32_t partition,
2320  int64_t *low, int64_t *high);
2321 
2322 
2323 
2348 RD_EXPORT rd_kafka_resp_err_t
2349 rd_kafka_offsets_for_times (rd_kafka_t *rk,
2351  int timeout_ms);
2352 
2353 
2367 RD_EXPORT
2368 void rd_kafka_mem_free (rd_kafka_t *rk, void *ptr);
2369 
2370 
2394 RD_EXPORT
2395 rd_kafka_queue_t *rd_kafka_queue_new(rd_kafka_t *rk);
2396 
2400 RD_EXPORT
2401 void rd_kafka_queue_destroy(rd_kafka_queue_t *rkqu);
2402 
2403 
2410 RD_EXPORT
2411 rd_kafka_queue_t *rd_kafka_queue_get_main (rd_kafka_t *rk);
2412 
2413 
2423 RD_EXPORT
2424 rd_kafka_queue_t *rd_kafka_queue_get_consumer (rd_kafka_t *rk);
2425 
2436 RD_EXPORT
2437 rd_kafka_queue_t *rd_kafka_queue_get_partition (rd_kafka_t *rk,
2438  const char *topic,
2439  int32_t partition);
2440 
2463 RD_EXPORT
2464 rd_kafka_queue_t *rd_kafka_queue_get_background (rd_kafka_t *rk);
2465 
2466 
2477 RD_EXPORT
2478 void rd_kafka_queue_forward (rd_kafka_queue_t *src, rd_kafka_queue_t *dst);
2479 
2496 RD_EXPORT
2498  rd_kafka_queue_t *rkqu);
2499 
2500 
2504 RD_EXPORT
2505 size_t rd_kafka_queue_length (rd_kafka_queue_t *rkqu);
2506 
2507 
2524 RD_EXPORT
2525 void rd_kafka_queue_io_event_enable (rd_kafka_queue_t *rkqu, int fd,
2526  const void *payload, size_t size);
2527 
2542 RD_EXPORT
2543 void rd_kafka_queue_cb_event_enable (rd_kafka_queue_t *rkqu,
2544  void (*event_cb) (rd_kafka_t *rk,
2545  void *opaque),
2546  void *opaque);
2547 
2558 #define RD_KAFKA_OFFSET_BEGINNING -2
2560 #define RD_KAFKA_OFFSET_END -1
2562 #define RD_KAFKA_OFFSET_STORED -1000
2564 #define RD_KAFKA_OFFSET_INVALID -1001
2568 #define RD_KAFKA_OFFSET_TAIL_BASE -2000 /* internal: do not use */
2569 
2576 #define RD_KAFKA_OFFSET_TAIL(CNT) (RD_KAFKA_OFFSET_TAIL_BASE - (CNT))
2577 
2611 RD_EXPORT
2612 int rd_kafka_consume_start(rd_kafka_topic_t *rkt, int32_t partition,
2613  int64_t offset);
2614 
2629 RD_EXPORT
2630 int rd_kafka_consume_start_queue(rd_kafka_topic_t *rkt, int32_t partition,
2631  int64_t offset, rd_kafka_queue_t *rkqu);
2632 
2646 RD_EXPORT
2647 int rd_kafka_consume_stop(rd_kafka_topic_t *rkt, int32_t partition);
2648 
2649 
2650 
2665 RD_EXPORT
2666 rd_kafka_resp_err_t rd_kafka_seek (rd_kafka_topic_t *rkt,
2667  int32_t partition,
2668  int64_t offset,
2669  int timeout_ms);
2670 
2671 
2696 RD_EXPORT
2697 rd_kafka_message_t *rd_kafka_consume(rd_kafka_topic_t *rkt, int32_t partition,
2698  int timeout_ms);
2699 
2700 
2701 
2727 RD_EXPORT
2728 ssize_t rd_kafka_consume_batch(rd_kafka_topic_t *rkt, int32_t partition,
2729  int timeout_ms,
2730  rd_kafka_message_t **rkmessages,
2731  size_t rkmessages_size);
2732 
2733 
2734 
2758 RD_EXPORT
2759 int rd_kafka_consume_callback(rd_kafka_topic_t *rkt, int32_t partition,
2760  int timeout_ms,
2761  void (*consume_cb) (rd_kafka_message_t
2762  *rkmessage,
2763  void *opaque),
2764  void *opaque);
2765 
2766 
2783 RD_EXPORT
2784 rd_kafka_message_t *rd_kafka_consume_queue(rd_kafka_queue_t *rkqu,
2785  int timeout_ms);
2786 
2792 RD_EXPORT
2793 ssize_t rd_kafka_consume_batch_queue(rd_kafka_queue_t *rkqu,
2794  int timeout_ms,
2795  rd_kafka_message_t **rkmessages,
2796  size_t rkmessages_size);
2797 
2803 RD_EXPORT
2804 int rd_kafka_consume_callback_queue(rd_kafka_queue_t *rkqu,
2805  int timeout_ms,
2806  void (*consume_cb) (rd_kafka_message_t
2807  *rkmessage,
2808  void *opaque),
2809  void *opaque);
2810 
2811 
2837 RD_EXPORT
2838 rd_kafka_resp_err_t rd_kafka_offset_store(rd_kafka_topic_t *rkt,
2839  int32_t partition, int64_t offset);
2840 
2841 
2858 RD_EXPORT rd_kafka_resp_err_t
2859 rd_kafka_offsets_store(rd_kafka_t *rk,
2903 RD_EXPORT rd_kafka_resp_err_t
2904 rd_kafka_subscribe (rd_kafka_t *rk,
2905  const rd_kafka_topic_partition_list_t *topics);
2906 
2907 
2911 RD_EXPORT
2912 rd_kafka_resp_err_t rd_kafka_unsubscribe (rd_kafka_t *rk);
2913 
2914 
2924 RD_EXPORT rd_kafka_resp_err_t
2925 rd_kafka_subscription (rd_kafka_t *rk,
2927 
2928 
2929 
2951 RD_EXPORT
2952 rd_kafka_message_t *rd_kafka_consumer_poll (rd_kafka_t *rk, int timeout_ms);
2953 
2969 RD_EXPORT
2971 
2972 
2973 
2987 RD_EXPORT rd_kafka_resp_err_t
2988 rd_kafka_assign (rd_kafka_t *rk,
2989  const rd_kafka_topic_partition_list_t *partitions);
2990 
3000 RD_EXPORT rd_kafka_resp_err_t
3001 rd_kafka_assignment (rd_kafka_t *rk,
3002  rd_kafka_topic_partition_list_t **partitions);
3003 
3004 
3005 
3006 
3021 RD_EXPORT rd_kafka_resp_err_t
3022 rd_kafka_commit (rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets,
3023  int async);
3024 
3025 
3031 RD_EXPORT rd_kafka_resp_err_t
3032 rd_kafka_commit_message (rd_kafka_t *rk, const rd_kafka_message_t *rkmessage,
3033  int async);
3034 
3035 
3057 RD_EXPORT rd_kafka_resp_err_t
3058 rd_kafka_commit_queue (rd_kafka_t *rk,
3059  const rd_kafka_topic_partition_list_t *offsets,
3060  rd_kafka_queue_t *rkqu,
3061  void (*cb) (rd_kafka_t *rk,
3062  rd_kafka_resp_err_t err,
3064  void *opaque),
3065  void *opaque);
3066 
3067 
3080 RD_EXPORT rd_kafka_resp_err_t
3081 rd_kafka_committed (rd_kafka_t *rk,
3082  rd_kafka_topic_partition_list_t *partitions,
3083  int timeout_ms);
3084 
3085 
3086 
3103 RD_EXPORT rd_kafka_resp_err_t
3104 rd_kafka_position (rd_kafka_t *rk,
3105  rd_kafka_topic_partition_list_t *partitions);
3106 
3107 
3123 #define RD_KAFKA_MSG_F_FREE 0x1
3124 #define RD_KAFKA_MSG_F_COPY 0x2
3125 #define RD_KAFKA_MSG_F_BLOCK 0x4
3136 #define RD_KAFKA_MSG_F_PARTITION 0x8
3229 RD_EXPORT
3230 int rd_kafka_produce(rd_kafka_topic_t *rkt, int32_t partition,
3231  int msgflags,
3232  void *payload, size_t len,
3233  const void *key, size_t keylen,
3234  void *msg_opaque);
3235 
3236 
3249 RD_EXPORT
3250 rd_kafka_resp_err_t rd_kafka_producev (rd_kafka_t *rk, ...);
3251 
3252 
3277 RD_EXPORT
3278 int rd_kafka_produce_batch(rd_kafka_topic_t *rkt, int32_t partition,
3279  int msgflags,
3280  rd_kafka_message_t *rkmessages, int message_cnt);
3281 
3282 
3283 
3284 
3298 RD_EXPORT
3299 rd_kafka_resp_err_t rd_kafka_flush (rd_kafka_t *rk, int timeout_ms);
3300 
3301 
3316 typedef struct rd_kafka_metadata_broker {
3317  int32_t id;
3318  char *host;
3319  int port;
3321 
3325 typedef struct rd_kafka_metadata_partition {
3326  int32_t id;
3327  rd_kafka_resp_err_t err;
3328  int32_t leader;
3329  int replica_cnt;
3330  int32_t *replicas;
3331  int isr_cnt;
3332  int32_t *isrs;
3334 
3338 typedef struct rd_kafka_metadata_topic {
3339  char *topic;
3340  int partition_cnt;
3341  struct rd_kafka_metadata_partition *partitions;
3349 typedef struct rd_kafka_metadata {
3350  int broker_cnt;
3351  struct rd_kafka_metadata_broker *brokers;
3353  int topic_cnt;
3354  struct rd_kafka_metadata_topic *topics;
3356  int32_t orig_broker_id;
3357  char *orig_broker_name;
3359 
3360 
3377 RD_EXPORT
3379 rd_kafka_metadata (rd_kafka_t *rk, int all_topics,
3380  rd_kafka_topic_t *only_rkt,
3381  const struct rd_kafka_metadata **metadatap,
3382  int timeout_ms);
3383 
3387 RD_EXPORT
3388 void rd_kafka_metadata_destroy(const struct rd_kafka_metadata *metadata);
3389 
3390 
3411  char *member_id;
3412  char *client_id;
3413  char *client_host;
3414  void *member_metadata;
3416  int member_metadata_size;
3417  void *member_assignment;
3420 };
3421 
3426  struct rd_kafka_metadata_broker broker;
3427  char *group;
3429  char *state;
3431  char *protocol;
3434 };
3435 
3444 };
3474 RD_EXPORT
3476 rd_kafka_list_groups (rd_kafka_t *rk, const char *group,
3477  const struct rd_kafka_group_list **grplistp,
3478  int timeout_ms);
3479 
3483 RD_EXPORT
3484 void rd_kafka_group_list_destroy (const struct rd_kafka_group_list *grplist);
3485 
3486 
3527 RD_EXPORT
3528 int rd_kafka_brokers_add(rd_kafka_t *rk, const char *brokerlist);
3529 
3530 
3531 
3532 
3545 RD_EXPORT RD_DEPRECATED
3546 void rd_kafka_set_logger(rd_kafka_t *rk,
3547  void (*func) (const rd_kafka_t *rk, int level,
3548  const char *fac, const char *buf));
3549 
3550 
3558 RD_EXPORT
3559 void rd_kafka_set_log_level(rd_kafka_t *rk, int level);
3560 
3561 
3565 RD_EXPORT
3566 void rd_kafka_log_print(const rd_kafka_t *rk, int level,
3567  const char *fac, const char *buf);
3568 
3569 
3573 RD_EXPORT
3574 void rd_kafka_log_syslog(const rd_kafka_t *rk, int level,
3575  const char *fac, const char *buf);
3576 
3577 
3600 RD_EXPORT
3601 int rd_kafka_outq_len(rd_kafka_t *rk);
3602 
3603 
3604 
3611 RD_EXPORT
3612 void rd_kafka_dump(FILE *fp, rd_kafka_t *rk);
3613 
3614 
3615 
3621 RD_EXPORT
3622 int rd_kafka_thread_cnt(void);
3623 
3624 
3633 RD_EXPORT
3634 int rd_kafka_wait_destroyed(int timeout_ms);
3635 
3636 
3642 RD_EXPORT
3643 int rd_kafka_unittest (void);
3644 
3645 
3663 RD_EXPORT
3665 
3666 
3682 typedef int rd_kafka_event_type_t;
3683 #define RD_KAFKA_EVENT_NONE 0x0
3684 #define RD_KAFKA_EVENT_DR 0x1
3685 #define RD_KAFKA_EVENT_FETCH 0x2
3686 #define RD_KAFKA_EVENT_LOG 0x4
3687 #define RD_KAFKA_EVENT_ERROR 0x8
3688 #define RD_KAFKA_EVENT_REBALANCE 0x10
3689 #define RD_KAFKA_EVENT_OFFSET_COMMIT 0x20
3690 #define RD_KAFKA_EVENT_STATS 0x40
3691 #define RD_KAFKA_EVENT_CREATETOPICS_RESULT 100
3692 #define RD_KAFKA_EVENT_DELETETOPICS_RESULT 101
3693 #define RD_KAFKA_EVENT_CREATEPARTITIONS_RESULT 102
3694 #define RD_KAFKA_EVENT_ALTERCONFIGS_RESULT 103
3695 #define RD_KAFKA_EVENT_DESCRIBECONFIGS_RESULT 104
3704 RD_EXPORT
3705 rd_kafka_event_type_t rd_kafka_event_type (const rd_kafka_event_t *rkev);
3713 RD_EXPORT
3714 const char *rd_kafka_event_name (const rd_kafka_event_t *rkev);
3715 
3716 
3726 RD_EXPORT
3727 void rd_kafka_event_destroy (rd_kafka_event_t *rkev);
3728 
3729 
3745 RD_EXPORT
3746 const rd_kafka_message_t *rd_kafka_event_message_next (rd_kafka_event_t *rkev);
3747 
3748 
3762 RD_EXPORT
3763 size_t rd_kafka_event_message_array (rd_kafka_event_t *rkev,
3764  const rd_kafka_message_t **rkmessages,
3765  size_t size);
3766 
3767 
3775 RD_EXPORT
3776 size_t rd_kafka_event_message_count (rd_kafka_event_t *rkev);
3777 
3778 
3785 RD_EXPORT
3786 rd_kafka_resp_err_t rd_kafka_event_error (rd_kafka_event_t *rkev);
3787 
3788 
3797 RD_EXPORT
3798 const char *rd_kafka_event_error_string (rd_kafka_event_t *rkev);
3799 
3800 
3801 
3813 RD_EXPORT
3814 void *rd_kafka_event_opaque (rd_kafka_event_t *rkev);
3815 
3816 
3825 RD_EXPORT
3826 int rd_kafka_event_log (rd_kafka_event_t *rkev,
3827  const char **fac, const char **str, int *level);
3828 
3829 
3841 RD_EXPORT
3842 const char *rd_kafka_event_stats (rd_kafka_event_t *rkev);
3843 
3844 
3855 rd_kafka_event_topic_partition_list (rd_kafka_event_t *rkev);
3856 
3857 
3867 RD_EXPORT rd_kafka_topic_partition_t *
3868 rd_kafka_event_topic_partition (rd_kafka_event_t *rkev);
3869 
3870 
3871 
3872 typedef rd_kafka_event_t rd_kafka_CreateTopics_result_t;
3873 typedef rd_kafka_event_t rd_kafka_DeleteTopics_result_t;
3874 typedef rd_kafka_event_t rd_kafka_CreatePartitions_result_t;
3875 typedef rd_kafka_event_t rd_kafka_AlterConfigs_result_t;
3876 typedef rd_kafka_event_t rd_kafka_DescribeConfigs_result_t;
3877 
3885 RD_EXPORT const rd_kafka_CreateTopics_result_t *
3886 rd_kafka_event_CreateTopics_result (rd_kafka_event_t *rkev);
3887 
3895 RD_EXPORT const rd_kafka_DeleteTopics_result_t *
3896 rd_kafka_event_DeleteTopics_result (rd_kafka_event_t *rkev);
3897 
3905 RD_EXPORT const rd_kafka_CreatePartitions_result_t *
3906 rd_kafka_event_CreatePartitions_result (rd_kafka_event_t *rkev);
3907 
3915 RD_EXPORT const rd_kafka_AlterConfigs_result_t *
3916 rd_kafka_event_AlterConfigs_result (rd_kafka_event_t *rkev);
3917 
3925 RD_EXPORT const rd_kafka_DescribeConfigs_result_t *
3926 rd_kafka_event_DescribeConfigs_result (rd_kafka_event_t *rkev);
3927 
3928 
3929 
3930 
3940 RD_EXPORT
3941 rd_kafka_event_t *rd_kafka_queue_poll (rd_kafka_queue_t *rkqu, int timeout_ms);
3942 
3956 RD_EXPORT
3957 int rd_kafka_queue_poll_callback (rd_kafka_queue_t *rkqu, int timeout_ms);
3958 
3959 
4001 typedef rd_kafka_resp_err_t
4002 (rd_kafka_plugin_f_conf_init_t) (rd_kafka_conf_t *conf,
4003  void **plug_opaquep,
4004  char *errstr, size_t errstr_size);
4005 
4085 typedef rd_kafka_conf_res_t
4086 (rd_kafka_interceptor_f_on_conf_set_t) (rd_kafka_conf_t *conf,
4087  const char *name, const char *val,
4088  char *errstr, size_t errstr_size,
4089  void *ic_opaque);
4090 
4091 
4108 typedef rd_kafka_resp_err_t
4109 (rd_kafka_interceptor_f_on_conf_dup_t) (rd_kafka_conf_t *new_conf,
4110  const rd_kafka_conf_t *old_conf,
4111  size_t filter_cnt,
4112  const char **filter,
4113  void *ic_opaque);
4114 
4115 
4122 typedef rd_kafka_resp_err_t
4124 
4125 
4143 typedef rd_kafka_resp_err_t
4144 (rd_kafka_interceptor_f_on_new_t) (rd_kafka_t *rk, const rd_kafka_conf_t *conf,
4145  void *ic_opaque,
4146  char *errstr, size_t errstr_size);
4147 
4148 
4156 typedef rd_kafka_resp_err_t
4157 (rd_kafka_interceptor_f_on_destroy_t) (rd_kafka_t *rk, void *ic_opaque);
4159 
4160 
4161 
4182 typedef rd_kafka_resp_err_t
4183 (rd_kafka_interceptor_f_on_send_t) (rd_kafka_t *rk,
4184  rd_kafka_message_t *rkmessage,
4185  void *ic_opaque);
4186 
4209 typedef rd_kafka_resp_err_t
4211  rd_kafka_message_t *rkmessage,
4212  void *ic_opaque);
4213 
4214 
4231 typedef rd_kafka_resp_err_t
4232 (rd_kafka_interceptor_f_on_consume_t) (rd_kafka_t *rk,
4233  rd_kafka_message_t *rkmessage,
4234  void *ic_opaque);
4235 
4256 typedef rd_kafka_resp_err_t
4258  rd_kafka_t *rk,
4259  const rd_kafka_topic_partition_list_t *offsets,
4260  rd_kafka_resp_err_t err, void *ic_opaque);
4261 
4262 
4284 typedef rd_kafka_resp_err_t
4286  rd_kafka_t *rk,
4287  int sockfd,
4288  const char *brokername,
4289  int32_t brokerid,
4290  int16_t ApiKey,
4291  int16_t ApiVersion,
4292  int32_t CorrId,
4293  size_t size,
4294  void *ic_opaque);
4295 
4296 
4297 
4310 RD_EXPORT rd_kafka_resp_err_t
4312  rd_kafka_conf_t *conf, const char *ic_name,
4314  void *ic_opaque);
4315 
4316 
4329 RD_EXPORT rd_kafka_resp_err_t
4331  rd_kafka_conf_t *conf, const char *ic_name,
4333  void *ic_opaque);
4334 
4348 RD_EXPORT rd_kafka_resp_err_t
4350  rd_kafka_conf_t *conf, const char *ic_name,
4352  void *ic_opaque);
4353 
4354 
4376 RD_EXPORT rd_kafka_resp_err_t
4378  rd_kafka_conf_t *conf, const char *ic_name,
4380  void *ic_opaque);
4381 
4382 
4383 
4396 RD_EXPORT rd_kafka_resp_err_t
4398  rd_kafka_t *rk, const char *ic_name,
4400  void *ic_opaque);
4401 
4402 
4415 RD_EXPORT rd_kafka_resp_err_t
4417  rd_kafka_t *rk, const char *ic_name,
4419  void *ic_opaque);
4420 
4433 RD_EXPORT rd_kafka_resp_err_t
4435  rd_kafka_t *rk, const char *ic_name,
4436  rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement,
4437  void *ic_opaque);
4438 
4439 
4452 RD_EXPORT rd_kafka_resp_err_t
4454  rd_kafka_t *rk, const char *ic_name,
4456  void *ic_opaque);
4457 
4458 
4471 RD_EXPORT rd_kafka_resp_err_t
4473  rd_kafka_t *rk, const char *ic_name,
4475  void *ic_opaque);
4476 
4477 
4490 RD_EXPORT rd_kafka_resp_err_t
4492  rd_kafka_t *rk, const char *ic_name,
4494  void *ic_opaque);
4495 
4496 
4497 
4498 
4519 RD_EXPORT rd_kafka_resp_err_t
4520 rd_kafka_topic_result_error (const rd_kafka_topic_result_t *topicres);
4521 
4528 RD_EXPORT const char *
4529 rd_kafka_topic_result_error_string (const rd_kafka_topic_result_t *topicres);
4530 
4536 RD_EXPORT const char *
4537 rd_kafka_topic_result_name (const rd_kafka_topic_result_t *topicres);
4538 
4539 
4587 typedef enum rd_kafka_admin_op_t {
4588  RD_KAFKA_ADMIN_OP_ANY = 0,
4596 
4609 typedef struct rd_kafka_AdminOptions_s rd_kafka_AdminOptions_t;
4610 
4630 RD_EXPORT rd_kafka_AdminOptions_t *
4631 rd_kafka_AdminOptions_new (rd_kafka_t *rk, rd_kafka_admin_op_t for_api);
4632 
4633 
4637 RD_EXPORT void rd_kafka_AdminOptions_destroy (rd_kafka_AdminOptions_t *options);
4638 
4639 
4653 RD_EXPORT rd_kafka_resp_err_t
4655  int timeout_ms,
4656  char *errstr, size_t errstr_size);
4657 
4658 
4681 RD_EXPORT rd_kafka_resp_err_t
4683  int timeout_ms,
4684  char *errstr, size_t errstr_size);
4685 
4686 
4700 RD_EXPORT rd_kafka_resp_err_t
4702  int true_or_false,
4703  char *errstr, size_t errstr_size);
4704 
4705 
4726 RD_EXPORT rd_kafka_resp_err_t
4728  int32_t broker_id,
4729  char *errstr, size_t errstr_size);
4730 
4731 
4732 
4737 RD_EXPORT void
4739  void *opaque);
4740 
4741 
4742 
4743 
4744 
4745 
4753 typedef struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t;
4754 
4769 RD_EXPORT rd_kafka_NewTopic_t *
4770 rd_kafka_NewTopic_new (const char *topic, int num_partitions,
4771  int replication_factor,
4772  char *errstr, size_t errstr_size);
4773 
4778 RD_EXPORT void
4780 
4781 
4787 RD_EXPORT void
4789  size_t new_topic_cnt);
4790 
4791 
4813 RD_EXPORT rd_kafka_resp_err_t
4815  int32_t partition,
4816  int32_t *broker_ids,
4817  size_t broker_id_cnt,
4818  char *errstr, size_t errstr_size);
4819 
4832 RD_EXPORT rd_kafka_resp_err_t
4834  const char *name, const char *value);
4835 
4836 
4854 RD_EXPORT void
4855 rd_kafka_CreateTopics (rd_kafka_t *rk,
4856  rd_kafka_NewTopic_t **new_topics,
4857  size_t new_topic_cnt,
4858  const rd_kafka_AdminOptions_t *options,
4859  rd_kafka_queue_t *rkqu);
4860 
4861 
4872 RD_EXPORT const rd_kafka_topic_result_t **
4874  const rd_kafka_CreateTopics_result_t *result,
4875  size_t *cntp);
4876 
4877 
4878 
4879 
4880 
4887 typedef struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t;
4888 
4898 RD_EXPORT rd_kafka_DeleteTopic_t *
4899 rd_kafka_DeleteTopic_new (const char *topic);
4900 
4905 RD_EXPORT void
4907 
4913 RD_EXPORT void
4915  size_t del_topic_cnt);
4916 
4929 RD_EXPORT
4930 void rd_kafka_DeleteTopics (rd_kafka_t *rk,
4931  rd_kafka_DeleteTopic_t **del_topics,
4932  size_t del_topic_cnt,
4933  const rd_kafka_AdminOptions_t *options,
4934  rd_kafka_queue_t *rkqu);
4935 
4936 
4937 
4948 RD_EXPORT const rd_kafka_topic_result_t **
4950  const rd_kafka_DeleteTopics_result_t *result,
4951  size_t *cntp);
4952 
4953 
4954 
4955 
4956 
4957 
4964 typedef struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t;
4965 
4979 rd_kafka_NewPartitions_new (const char *topic, size_t new_total_cnt,
4980  char *errstr, size_t errstr_size);
4981 
4986 RD_EXPORT void
4988 
4994 RD_EXPORT void
4996  size_t new_parts_cnt);
4997 
5020 RD_EXPORT rd_kafka_resp_err_t
5022  int32_t new_partition_idx,
5023  int32_t *broker_ids,
5024  size_t broker_id_cnt,
5025  char *errstr,
5026  size_t errstr_size);
5027 
5028 
5046 RD_EXPORT void
5047 rd_kafka_CreatePartitions (rd_kafka_t *rk,
5048  rd_kafka_NewPartitions_t **new_parts,
5049  size_t new_parts_cnt,
5050  const rd_kafka_AdminOptions_t *options,
5051  rd_kafka_queue_t *rkqu);
5052 
5053 
5054 
5065 RD_EXPORT const rd_kafka_topic_result_t **
5067  const rd_kafka_CreatePartitions_result_t *result,
5068  size_t *cntp);
5069 
5070 
5071 
5072 
5073 
5086 typedef enum rd_kafka_ConfigSource_t {
5105  RD_KAFKA_CONFIG_SOURCE__CNT,
5108 
5112 RD_EXPORT const char *
5114 
5115 
5116 typedef struct rd_kafka_ConfigEntry_s rd_kafka_ConfigEntry_t;
5117 
5121 RD_EXPORT const char *
5122 rd_kafka_ConfigEntry_name (const rd_kafka_ConfigEntry_t *entry);
5123 
5128 RD_EXPORT const char *
5129 rd_kafka_ConfigEntry_value (const rd_kafka_ConfigEntry_t *entry);
5130 
5134 RD_EXPORT rd_kafka_ConfigSource_t
5135 rd_kafka_ConfigEntry_source (const rd_kafka_ConfigEntry_t *entry);
5136 
5141 RD_EXPORT int
5142 rd_kafka_ConfigEntry_is_read_only (const rd_kafka_ConfigEntry_t *entry);
5143 
5149 RD_EXPORT int
5150 rd_kafka_ConfigEntry_is_default (const rd_kafka_ConfigEntry_t *entry);
5151 
5159 RD_EXPORT int
5160 rd_kafka_ConfigEntry_is_sensitive (const rd_kafka_ConfigEntry_t *entry);
5161 
5165 RD_EXPORT int
5166 rd_kafka_ConfigEntry_is_synonym (const rd_kafka_ConfigEntry_t *entry);
5167 
5168 
5178 RD_EXPORT const rd_kafka_ConfigEntry_t **
5179 rd_kafka_ConfigEntry_synonyms (const rd_kafka_ConfigEntry_t *entry,
5180  size_t *cntp);
5181 
5182 
5183 
5184 
5188 typedef enum rd_kafka_ResourceType_t {
5190  RD_KAFKA_RESOURCE_ANY = 1,
5196 
5200 RD_EXPORT const char *
5203 typedef struct rd_kafka_ConfigResource_s rd_kafka_ConfigResource_t;
5206 RD_EXPORT rd_kafka_ConfigResource_t *
5207 rd_kafka_ConfigResource_new (rd_kafka_ResourceType_t restype,
5208  const char *resname);
5209 
5214 RD_EXPORT void
5215 rd_kafka_ConfigResource_destroy (rd_kafka_ConfigResource_t *config);
5216 
5217 
5223 RD_EXPORT void
5224 rd_kafka_ConfigResource_destroy_array (rd_kafka_ConfigResource_t **config,
5225  size_t config_cnt);
5226 
5227 
5240 RD_EXPORT rd_kafka_resp_err_t
5241 rd_kafka_ConfigResource_set_config (rd_kafka_ConfigResource_t *config,
5242  const char *name, const char *value);
5243 
5244 
5252 RD_EXPORT const rd_kafka_ConfigEntry_t **
5253 rd_kafka_ConfigResource_configs (const rd_kafka_ConfigResource_t *config,
5254  size_t *cntp);
5255 
5256 
5257 
5261 RD_EXPORT rd_kafka_ResourceType_t
5262 rd_kafka_ConfigResource_type (const rd_kafka_ConfigResource_t *config);
5263 
5267 RD_EXPORT const char *
5268 rd_kafka_ConfigResource_name (const rd_kafka_ConfigResource_t *config);
5269 
5273 RD_EXPORT rd_kafka_resp_err_t
5274 rd_kafka_ConfigResource_error (const rd_kafka_ConfigResource_t *config);
5275 
5280 RD_EXPORT const char *
5281 rd_kafka_ConfigResource_error_string (const rd_kafka_ConfigResource_t *config);
5282 
5283 
5311 RD_EXPORT
5312 void rd_kafka_AlterConfigs (rd_kafka_t *rk,
5313  rd_kafka_ConfigResource_t **configs,
5314  size_t config_cnt,
5315  const rd_kafka_AdminOptions_t *options,
5316  rd_kafka_queue_t *rkqu);
5317 
5318 
5336 RD_EXPORT const rd_kafka_ConfigResource_t **
5338  const rd_kafka_AlterConfigs_result_t *result,
5339  size_t *cntp);
5340 
5341 
5342 
5343 
5344 
5345 
5377 RD_EXPORT
5378 void rd_kafka_DescribeConfigs (rd_kafka_t *rk,
5379  rd_kafka_ConfigResource_t **configs,
5380  size_t config_cnt,
5381  const rd_kafka_AdminOptions_t *options,
5382  rd_kafka_queue_t *rkqu);
5383 
5384 
5395 RD_EXPORT const rd_kafka_ConfigResource_t **
5397  const rd_kafka_DescribeConfigs_result_t *result,
5398  size_t *cntp);
5399 
5403 #ifdef __cplusplus
5404 }
5405 #endif
5406 #endif /* _RDKAFKA_H_ */
void * _private
Definition: rdkafka.h:617
rd_kafka_resp_err_t
Error codes.
Definition: rdkafka.h:253
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:4197
rd_kafka_topic_t * rkt
Definition: rdkafka.h:1059
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:358
RD_EXPORT const char * rd_kafka_ConfigEntry_name(const rd_kafka_ConfigEntry_t *entry)
rd_kafka_ConfigSource_t
Definition: rdkafka.h:5100
RD_EXPORT int32_t rd_kafka_controllerid(rd_kafka_t *rk, int timeout_ms)
Returns the current ControllerId as reported in broker metadata.
RD_EXPORT int rd_kafka_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:315
RD_EXPORT const rd_kafka_DeleteTopics_result_t * rd_kafka_event_DeleteTopics_result(rd_kafka_event_t *rkev)
rd_kafka_resp_err_t err
Definition: rdkafka.h:1058
rd_kafka_conf_res_t
Configuration result type.
Definition: rdkafka.h:1220
int member_cnt
Definition: rdkafka.h:3447
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:635
Definition: rdkafka.h:333
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 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:637
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)
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 *opaque, void *msg_opaque)
Consistent partitioner.
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 char * rd_kafka_event_name(const rd_kafka_event_t *rkev)
Definition: rdkafka.h:808
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:5204
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:4601
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_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.
Definition: rdkafka.h:289
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:345
Definition: rdkafka.h:428
Definition: rdkafka.h:801
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:303
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:3443
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:4299
size_t key_len
Definition: rdkafka.h:1070
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:283
RD_EXPORT rd_kafka_event_type_t rd_kafka_event_type(const rd_kafka_event_t *rkev)
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:1222
int member_assignment_size
Definition: rdkafka.h:3433
RD_EXPORT rd_kafka_t * rd_kafka_new(rd_kafka_type_t type, rd_kafka_conf_t *conf, char *errstr, size_t errstr_size)
Creates a new Kafka handle and starts its operation according to the specified type (RD_KAFKA_CONSUME...
RD_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:3439
char * group
Definition: rdkafka.h:3441
Partition information.
Definition: rdkafka.h:3339
Definition: rdkafka.h:272
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:454
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:4123
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:3426
const char * name
Definition: rdkafka.h:486
Definition: rdkafka.h:279
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.
char * client_host
Definition: rdkafka.h:3427
struct rd_kafka_group_info * groups
Definition: rdkafka.h:3456
RD_EXPORT void rd_kafka_queue_cb_event_enable(rd_kafka_queue_t *rkqu, void(*event_cb)(rd_kafka_t *rk, void *opaque), void *opaque)
Enable callback event triggering for queue.
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.
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:806
RD_EXPORT void rd_kafka_topic_conf_set_opaque(rd_kafka_topic_conf_t *conf, void *opaque)
Sets the application's opaque pointer that will be passed to all topic callbacks as the rkt_opaque ar...
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:299
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:4158
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.
char * protocol_type
Definition: rdkafka.h:3444
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:4224
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:4271
Definition: rdkafka.h:307
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:4016
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 *opaque), void *opaque)
Consume multiple messages from queue with callback.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_AdminOptions_set_request_timeout(rd_kafka_AdminOptions_t *options, int timeout_ms, char *errstr, size_t errstr_size)
Sets the overall request timeout, including broker lookup, request transmission, operation time on br...
RD_EXPORT 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 int rd_kafka_outq_len(rd_kafka_t *rk)
Returns the current out queue length.
int group_cnt
Definition: rdkafka.h:3457
Definition: rdkafka.h:325
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:799
RD_EXPORT const rd_kafka_message_t * rd_kafka_event_message_next(rd_kafka_event_t *rkev)
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 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:3442
rd_kafka_ResourceType_t
Definition: rdkafka.h:5202
size_t len
Definition: rdkafka.h:1065
RD_EXPORT rd_kafka_resp_err_t rd_kafka_commit(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *offsets, int async)
Commit offsets on broker for the provided list of partitions.
RD_EXPORT void rd_kafka_conf_set_default_topic_conf(rd_kafka_conf_t *conf, rd_kafka_topic_conf_t *tconf)
Definition: rdkafka.h:406
RD_EXPORT rd_kafka_type_t rd_kafka_type(const rd_kafka_t *rk)
Returns Kafka handle type.
RD_EXPORT int rd_kafka_produce(rd_kafka_topic_t *rkt, int32_t partition, int msgflags, void *payload, size_t len, const void *key, size_t keylen, void *msg_opaque)
Produce and send a single message to broker.
rd_kafka_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:4100
Definition: rdkafka.h:293
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:408
Definition: rdkafka.h:446
RD_EXPORT rd_kafka_ConfigSource_t rd_kafka_ConfigEntry_source(const rd_kafka_ConfigEntry_t *entry)
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:3440
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)
Definition: rdkafka.h:311
RD_EXPORT rd_kafka_resp_err_t rd_kafka_ConfigResource_error(const rd_kafka_ConfigResource_t *config)
Definition: rdkafka.h:5208
RD_EXPORT rd_kafka_resp_err_t rd_kafka_interceptor_add_on_acknowledgement(rd_kafka_t *rk, const char *ic_name, rd_kafka_interceptor_f_on_acknowledgement_t *on_acknowledgement, void *ic_opaque)
Append an on_acknowledgement() interceptor.
RD_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:485
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 void rd_kafka_destroy(rd_kafka_t *rk)
Destroy Kafka handle.
struct rd_kafka_NewTopic_s rd_kafka_NewTopic_t
Definition: rdkafka.h:4767
RD_EXPORT void rd_kafka_set_log_level(rd_kafka_t *rk, int level)
Specifies the maximum logging level produced by internal kafka logging and debugging.
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 *opaque), void *opaque)
Consumes messages from topic rkt and partition, calling the provided callback for each consumed messs...
RD_EXPORT const rd_kafka_DescribeConfigs_result_t * rd_kafka_event_DescribeConfigs_result(rd_kafka_event_t *rkev)
Definition: rdkafka.h:807
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_EXPORT int rd_kafka_wait_destroyed(int timeout_ms)
Wait for all rd_kafka_t objects to be destroyed.
Definition: rdkafka.h:5116
Definition: rdkafka.h:392
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 *opaque), void *opaque)
Commit offsets on broker for the provided list of partitions.
Definition: rdkafka.h:380
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_message_headers(const rd_kafka_message_t *rkmessage, rd_kafka_headers_t **hdrsp)
Get the message header list.
int64_t offset
Definition: rdkafka.h:1072
RD_EXPORT rd_kafka_headers_t * rd_kafka_headers_new(size_t initial_count)
Create a new headers list.
Definition: rdkafka.h:384
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_DeleteTopics_result_topics(const rd_kafka_DeleteTopics_result_t *result, size_t *cntp)
DeleteTopics result type and methods.
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 RD_DEPRECATED int rd_kafka_errno(void)
Returns the thread-local system errno.
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:4604
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:4171
Definition: rdkafka.h:366
Definition: rdkafka.h:4602
Definition: rdkafka.h:268
Definition: rdkafka.h:356
Definition: rdkafka.h:4605
Definition: rdkafka.h:349
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:256
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:3424
void * key
Definition: rdkafka.h:1068
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:803
Definition: rdkafka.h:281
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:634
int rd_kafka_event_type_t
Event types.
Definition: rdkafka.h:3696
Topic information.
Definition: rdkafka.h:3352
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:270
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)
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:5205
Definition: rdkafka.h:802
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() ...
int32_t partition
Definition: rdkafka.h:611
Definition: rdkafka.h:266
Definition: rdkafka.h:295
Definition: rdkafka.h:264
void * opaque
Definition: rdkafka.h:615
const char * desc
Definition: rdkafka.h:487
struct rd_kafka_NewPartitions_s rd_kafka_NewPartitions_t
Definition: rdkafka.h:4978
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 int64_t rd_kafka_message_timestamp(const rd_kafka_message_t *rkmessage, rd_kafka_timestamp_type_t *tstype)
Returns the message timestamp for a consumed message.
RD_EXPORT rd_kafka_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:4137
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)
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:378
Definition: rdkafka.h:331
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() ...
static RD_INLINE const char *RD_UNUSED 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...
Definition: rdkafka.h:1106
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 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.
RD_EXPORT void rd_kafka_group_list_destroy(const struct rd_kafka_group_list *grplist)
Release list memory.
Definition: rdkafka.h:438
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.
Definition: rdkafka.h:204
RD_EXPORT void rd_kafka_conf_set_background_event_cb(rd_kafka_conf_t *conf, void(*event_cb)(rd_kafka_t *rk, rd_kafka_event_t *rkev, void *opaque))
Generic event callback to be used with the event API to trigger callbacks for rd_kafka_event_t object...
RD_EXPORT rd_kafka_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.
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 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...
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:426
void * member_assignment
Definition: rdkafka.h:3431
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:636
RD_EXPORT rd_kafka_resp_err_t rd_kafka_subscribe(rd_kafka_t *rk, const rd_kafka_topic_partition_list_t *topics)
Subscribe to topic set using balanced consumer groups.
RD_EXPORT 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:1223
RD_EXPORT const char ** rd_kafka_conf_dump(rd_kafka_conf_t *conf, size_t *cntp)
Dump the configuration properties and values of conf to an array with "key", "value" pairs...
RD_EXPORT rd_kafka_resp_err_t rd_kafka_event_error(rd_kafka_event_t *rkev)
Definition: rdkafka.h:323
char * topic
Definition: rdkafka.h:610
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:329
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 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:4623
Definition: rdkafka.h:800
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:4246
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:341
Definition: rdkafka.h:4608
RD_EXPORT int rd_kafka_poll(rd_kafka_t *rk, int timeout_ms)
Polls the provided kafka handle for events.
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).
Definition: rdkafka.h:191
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).
RD_EXPORT const char * rd_kafka_ConfigSource_name(rd_kafka_ConfigSource_t confsource)
Definition: rdkafka.h:442
Definition: rdkafka.h:396
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:192
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)...
Definition: rdkafka.h:203
Definition: rdkafka.h:291
Definition: rdkafka.h:5207
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:287
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:5206
Definition: rdkafka.h:360
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.
Definition: rdkafka.h:262
Metadata container.
Definition: rdkafka.h:3363
Definition: rdkafka.h:321
RD_EXPORT rd_kafka_resp_err_t rd_kafka_unsubscribe(rd_kafka_t *rk)
Unsubscribe from the current subscription set.
RD_EXPORT void rd_kafka_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_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:313
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:440
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)
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 size_t rd_kafka_event_message_count(rd_kafka_event_t *rkev)
Definition: rdkafka.h:1221
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:285
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).
Definition: rdkafka.h:202
Definition: rdkafka.h:309
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:201
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.
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)
struct rd_kafka_DeleteTopic_s rd_kafka_DeleteTopic_t
Definition: rdkafka.h:4901
RD_EXPORT const char * rd_kafka_name(const rd_kafka_t *rk)
Returns Kafka handle name.
A Kafka message as returned by the rd_kafka_consume*() family of functions as well as provided to the...
Definition: rdkafka.h:1057
Definition: rdkafka.h:297
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_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:1060
Definition: rdkafka.h:352
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:301
RD_EXPORT rd_kafka_message_t * rd_kafka_consumer_poll(rd_kafka_t *rk, int timeout_ms)
Poll the consumer for messages or events.
RD_EXPORT rd_kafka_resp_err_t rd_kafka_assignment(rd_kafka_t *rk, rd_kafka_topic_partition_list_t **partitions)
Returns the current partition assignment.
Definition: rdkafka.h:277
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:811
char * member_id
Definition: rdkafka.h:3425
List of groups.
Definition: rdkafka.h:3455
Definition: rdkafka.h:471
Definition: rdkafka.h:805
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreatePartitions_result_topics(const rd_kafka_CreatePartitions_result_t *result, size_t *cntp)
CreatePartitions result type and methods.
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)
void * metadata
Definition: rdkafka.h:613
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 *opaque), void *opaque)
Sort list using comparator cmp.
RD_EXPORT const rd_kafka_ConfigResource_t ** rd_kafka_DescribeConfigs_result_resources(const rd_kafka_DescribeConfigs_result_t *result, size_t *cntp)
DescribeConfigs result type and methods.
Definition: rdkafka.h:5103
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 increas the...
int64_t offset
Definition: rdkafka.h:612
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:260
RD_EXPORT void rd_kafka_conf_destroy(rd_kafka_conf_t *conf)
Destroys a conf object.
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:319
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...
Broker information.
Definition: rdkafka.h:3330
Error code value, name and description. Typically for use with language bindings to automatically exp...
Definition: rdkafka.h:484
Definition: rdkafka.h:422
RD_EXPORT const char * rd_kafka_ConfigEntry_value(const rd_kafka_ConfigEntry_t *entry)
Definition: rdkafka.h:402
Definition: rdkafka.h:4603
rd_kafka_resp_err_t err
Definition: rdkafka.h:616
Definition: rdkafka.h:317
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)
AlterConfigs result type and methods.
Definition: rdkafka.h:343
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() ...
Topic+Partition place holder.
Definition: rdkafka.h:609
Definition: rdkafka.h:258
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:347
struct rd_kafka_group_member_info * members
Definition: rdkafka.h:3446
RD_EXPORT void rd_kafka_AdminOptions_set_opaque(rd_kafka_AdminOptions_t *options, void *opaque)
Set application opaque value that can be extracted from the result event using rd_kafka_event_opaque(...
size_t metadata_size
Definition: rdkafka.h:614
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 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:3430
Definition: rdkafka.h:4606
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:804
Definition: rdkafka.h:5203
rd_kafka_type_t
rd_kafka_t handle type.
Definition: rdkafka.h:190
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 int32_t rd_kafka_msg_partitioner_random(const rd_kafka_topic_t *rkt, const void *key, size_t keylen, int32_t partition_cnt, void *opaque, void *msg_opaque)
Random partitioner.
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:372
RD_EXPORT rd_kafka_topic_partition_t * rd_kafka_topic_partition_list_find(rd_kafka_topic_partition_list_t *rktparlist, const char *topic, int32_t partition)
Find element by topic and partition.
RD_EXPORT void rd_kafka_topic_conf_destroy(rd_kafka_topic_conf_t *topic_conf)
Destroys a topic conf object.
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:432
void * payload
Definition: rdkafka.h:1061
Definition: rdkafka.h:4607
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:362
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:3428
RD_EXPORT rd_kafka_resp_err_t rd_kafka_consumer_close(rd_kafka_t *rk)
Close down the KafkaConsumer.
RD_EXPORT const rd_kafka_topic_result_t ** rd_kafka_CreateTopics_result_topics(const rd_kafka_CreateTopics_result_t *result, size_t *cntp)
CreateTopics result type and methods.
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:1082
RD_EXPORT int rd_kafka_ConfigEntry_is_default(const rd_kafka_ConfigEntry_t *entry)
Definition: rdkafka.h:275
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 *opaque, void *msg_opaque)
Consistent-Random partitioner.
RD_EXPORT const char * rd_kafka_topic_name(const rd_kafka_topic_t *rkt)
Returns the topic name.
char * protocol
Definition: rdkafka.h:3445
RD_EXPORT rd_kafka_message_t * rd_kafka_consume_queue(rd_kafka_queue_t *rkqu, int timeout_ms)
Consume from queue.
RD_EXPORT rd_kafka_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.
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:809