rdkafka/
lib.rs

1//! A fully asynchronous, [futures]-enabled [Apache Kafka] client
2//! library for Rust based on [librdkafka].
3//!
4//! ## The library
5//!
6//! `rust-rdkafka` provides a safe Rust interface to librdkafka. This version
7//! is compatible with librdkafka v1.9.2+.
8//!
9//! ### Documentation
10//!
11//! - [Current master branch](https://fede1024.github.io/rust-rdkafka/)
12//! - [Latest release](https://docs.rs/rdkafka/)
13//! - [Changelog](https://github.com/fede1024/rust-rdkafka/blob/master/changelog.md)
14//!
15//! ### Features
16//!
17//! The main features provided at the moment are:
18//!
19//! - Support for all Kafka versions since 0.8.x. For more information about
20//!   broker compatibility options, check the [librdkafka
21//!   documentation][broker-compat].
22//! - Consume from single or multiple topics.
23//! - Automatic consumer rebalancing.
24//! - Customizable rebalance, with pre and post rebalance callbacks.
25//! - Synchronous or asynchronous message production.
26//! - Customizable offset commit.
27//! - Create and delete topics and add and edit partitions.
28//! - Alter broker and topic configurations.
29//! - Access to cluster metadata (list of topic-partitions, replicas, active
30//!   brokers etc).
31//! - Access to group metadata (list groups, list members of groups, hostnames,
32//!   etc.).
33//! - Access to producer and consumer metrics, errors and callbacks.
34//! - Exactly-once semantics (EOS) via idempotent and transactional producers
35//!   and read-committed consumers.
36//!
37//! ### One million messages per second
38//!
39//! `rust-rdkafka` is designed to be easy and safe to use thanks to the
40//! abstraction layer written in Rust, while at the same time being extremely
41//! fast thanks to the librdkafka C library.
42//!
43//! Here are some benchmark results using the [`BaseProducer`],
44//! sending data to a single Kafka 0.11 process running in localhost (default
45//! configuration, 3 partitions). Hardware: Dell laptop, with Intel Core
46//! i7-4712HQ @ 2.30GHz.
47//!
48//! - Scenario: produce 5 million messages, 10 bytes each, wait for all of them to be acked
49//!   - 1045413 messages/s, 9.970 MB/s  (average over 5 runs)
50//!
51//! - Scenario: produce 100000 messages, 10 KB each, wait for all of them to be acked
52//!   - 24623 messages/s, 234.826 MB/s  (average over 5 runs)
53//!
54//! For more numbers, check out the [kafka-benchmark] project.
55//!
56//! ### Client types
57//!
58//! `rust-rdkafka` provides low level and high level consumers and producers.
59//!
60//! Low level:
61//!
62//! * [`BaseConsumer`]: a simple wrapper around the librdkafka consumer. It
63//!   must be periodically `poll()`ed in order to execute callbacks, rebalances
64//!   and to receive messages.
65//! * [`BaseProducer`]: a simple wrapper around the librdkafka producer. As in
66//!   the consumer case, the user must call `poll()` periodically to execute
67//!   delivery callbacks.
68//! * [`ThreadedProducer`]: a `BaseProducer` with a separate thread dedicated to
69//!   polling the producer.
70//!
71//! High level:
72//!
73//!  * [`StreamConsumer`]: a [`Stream`] of messages that takes care of
74//!    polling the consumer automatically.
75//!  * [`FutureProducer`]: a [`Future`] that will be completed once
76//!    the message is delivered to Kafka (or failed).
77//!
78//! For more information about consumers and producers, refer to their
79//! module-level documentation.
80//!
81//! *Warning*: the library is under active development and the APIs are likely
82//! to change.
83//!
84//! ### Asynchronous data processing with Tokio
85//!
86//! [Tokio] is a platform for fast processing of asynchronous events in Rust.
87//! The interfaces exposed by the [`StreamConsumer`] and the [`FutureProducer`]
88//! allow rust-rdkafka users to easily integrate Kafka consumers and producers
89//! within the Tokio platform, and write asynchronous message processing code.
90//! Note that rust-rdkafka can be used without Tokio.
91//!
92//! To see rust-rdkafka in action with Tokio, check out the
93//! [asynchronous processing example] in the examples folder.
94//!
95//! ### At-least-once delivery
96//!
97//! At-least-once delivery semantics are common in many streaming applications:
98//! every message is guaranteed to be processed at least once; in case of
99//! temporary failure, the message can be re-processed and/or re-delivered,
100//! but no message will be lost.
101//!
102//! In order to implement at-least-once delivery the stream processing
103//! application has to carefully commit the offset only once the message has
104//! been processed. Committing the offset too early, instead, might cause
105//! message loss, since upon recovery the consumer will start from the next
106//! message, skipping the one where the failure occurred.
107//!
108//! To see how to implement at-least-once delivery with `rdkafka`, check out the
109//! [at-least-once delivery example] in the examples folder. To know more about
110//! delivery semantics, check the [message delivery semantics] chapter in the
111//! Kafka documentation.
112//!
113//! ### Exactly-once semantics
114//!
115//! Exactly-once semantics (EOS) can be achieved using transactional producers,
116//! which allow produced records and consumer offsets to be committed or aborted
117//! atomically. Consumers that set their `isolation.level` to `read_committed`
118//! will only observe committed messages.
119//!
120//! EOS is useful in read-process-write scenarios that require messages to be
121//! processed exactly once.
122//!
123//! To learn more about using transactions in rust-rdkafka, see the
124//! [Transactions](producer-transactions) section of the producer documentation.
125//!
126//! ### Users
127//!
128//! Here are some of the projects using rust-rdkafka:
129//!
130//! - [timely-dataflow]: a distributed data-parallel compute engine. See also
131//!   the [blog post][timely-blog] announcing its Kafka integration.
132//! - [kafka-view]: a web interface for Kafka clusters.
133//! - [kafka-benchmark]: a high performance benchmarking tool for Kafka.
134//! - [callysto]: Stream processing framework in Rust.
135//! - [bytewax]: Python stream processing framework using Timely Dataflow.
136//!
137//! *If you are using rust-rdkafka, please let us know!*
138//!
139//! ## Installation
140//!
141//! Add this to your `Cargo.toml`:
142//!
143//! ```toml
144//! [dependencies]
145//! rdkafka = { version = "0.25", features = ["cmake-build"] }
146//! ```
147//!
148//! This crate will compile librdkafka from sources and link it statically to
149//! your executable. To compile librdkafka you'll need:
150//!
151//! * the GNU toolchain
152//! * GNU `make`
153//! * `pthreads`
154//! * `zlib`: optional, but included by default (feature: `libz`)
155//! * `cmake`: optional, *not* included by default (feature: `cmake-build`)
156//! * `libssl-dev`: optional, *not* included by default (feature: `ssl`)
157//! * `libsasl2-dev`: optional, *not* included by default (feature: `gssapi`)
158//! * `libzstd-dev`: optional, *not* included by default (feature: `zstd-pkg-config`)
159//!
160//! Note that using the CMake build system, via the `cmake-build` feature, is
161//! encouraged if you can take the dependency on CMake.
162//!
163//! By default a submodule with the librdkafka sources pinned to a specific
164//! commit will be used to compile and statically link the library. The
165//! `dynamic-linking` feature can be used to instead dynamically link rdkafka to
166//! the system's version of librdkafka. Example:
167//!
168//! ```toml
169//! [dependencies]
170//! rdkafka = { version = "0.25", features = ["dynamic-linking"] }
171//! ```
172//!
173//! For a full listing of features, consult the [rdkafka-sys crate's
174//! documentation][rdkafka-sys-features]. All of rdkafka-sys features are
175//! re-exported as rdkafka features.
176//!
177//! ### Minimum supported Rust version (MSRV)
178//!
179//! The current minimum supported Rust version (MSRV) is 1.45.0. Note that
180//! bumping the MSRV is not considered a breaking change. Any release of
181//! rust-rdkafka may bump the MSRV.
182//!
183//! ### Asynchronous runtimes
184//!
185//! Some features of the [`StreamConsumer`] and [`FutureProducer`] depend on
186//! Tokio, which can be a heavyweight dependency for users who only intend to
187//! use the low-level consumers and producers. The Tokio integration is
188//! enabled by default, but can be disabled by turning off default features:
189//!
190//! ```toml
191//! [dependencies]
192//! rdkafka = { version = "0.25", default-features = false }
193//! ```
194//!
195//! If you would like to use an asynchronous runtime besides Tokio, you can
196//! integrate it with rust-rdkafka by providing a shim that implements the
197//! [`AsyncRuntime`] trait. See the following examples for details:
198//!
199//!   * [smol][runtime-smol]
200//!   * [async-std][runtime-async-std]
201//!
202//! ## Examples
203//!
204//! You can find examples in the [`examples`] folder. To run them:
205//!
206//! ```bash
207//! cargo run --example <example_name> -- <example_args>
208//! ```
209//!
210//! ## Debugging
211//!
212//! rust-rdkafka uses the [`log`] crate to handle logging.
213//! Optionally, enable the `tracing` feature to emit [`tracing`]
214//! events as opposed to [`log`] records.
215//!
216//! In test and examples, rust-rdkafka uses the  [`env_logger`] crate
217//! to format logs. In those contexts, logging can be enabled
218//! using the `RUST_LOG` environment variable, for example:
219//!
220//! ```bash
221//! RUST_LOG="librdkafka=trace,rdkafka::client=debug" cargo test
222//! ```
223//!
224//! This will configure the logging level of librdkafka to trace, and the level
225//! of the client module of the Rust client to debug. To actually receive logs
226//! from librdkafka, you also have to set the `debug` option in the producer or
227//! consumer configuration (see librdkafka
228//! [configuration][librdkafka-config]).
229//!
230//! To enable debugging in your project, make sure you initialize the logger
231//! with `env_logger::init()`, or the equivalent for any `log`-compatible
232//! logging framework.
233//!
234//! [`AsyncRuntime`]: https://docs.rs/rdkafka/*/rdkafka/util/trait.AsyncRuntime.html
235//! [`BaseConsumer`]: https://docs.rs/rdkafka/*/rdkafka/consumer/base_consumer/struct.BaseConsumer.html
236//! [`BaseProducer`]: https://docs.rs/rdkafka/*/rdkafka/producer/base_producer/struct.BaseProducer.html
237//! [`Future`]: https://doc.rust-lang.org/stable/std/future/trait.Future.html
238//! [`FutureProducer`]: https://docs.rs/rdkafka/*/rdkafka/producer/future_producer/struct.FutureProducer.html
239//! [`Stream`]: https://docs.rs/futures/*/futures/stream/trait.Stream.html
240//! [`StreamConsumer`]: https://docs.rs/rdkafka/*/rdkafka/consumer/stream_consumer/struct.StreamConsumer.html
241//! [`ThreadedProducer`]: https://docs.rs/rdkafka/*/rdkafka/producer/base_producer/struct.ThreadedProducer.html
242//! [`log`]: https://docs.rs/log
243//! [`tracing`]: https://docs.rs/tracing
244//! [`env_logger`]: https://docs.rs/env_logger
245//! [Apache Kafka]: https://kafka.apache.org
246//! [asynchronous processing example]: https://github.com/fede1024/rust-rdkafka/blob/master/examples/asynchronous_processing.rs
247//! [at-least-once delivery example]: https://github.com/fede1024/rust-rdkafka/blob/master/examples/at_least_once.rs
248//! [runtime-smol]: https://github.com/fede1024/rust-rdkafka/blob/master/examples/runtime_smol.rs
249//! [runtime-async-std]: https://github.com/fede1024/rust-rdkafka/blob/master/examples/runtime_async_std.rs
250//! [broker-compat]: https://github.com/edenhill/librdkafka/blob/master/INTRODUCTION.md#broker-version-compatibility
251//! [bytewax]: https://github.com/bytewax/bytewax
252//! [callysto]: https://github.com/vertexclique/callysto
253//! [`examples`]: https://github.com/fede1024/rust-rdkafka/blob/master/examples/
254//! [futures]: https://github.com/rust-lang/futures-rs
255//! [kafka-benchmark]: https://github.com/fede1024/kafka-benchmark
256//! [kafka-view]: https://github.com/fede1024/kafka-view
257//! [librdkafka]: https://github.com/edenhill/librdkafka
258//! [librdkafka-config]: https://github.com/edenhill/librdkafka/blob/master/CONFIGURATION.md
259//! [message delivery semantics]: https://kafka.apache.org/0101/documentation.html#semantics
260//! [producer-transactions]: https://docs.rs/rdkafka/*/rdkafka/producer/#transactions
261//! [rdkafka-sys-features]: https://github.com/fede1024/rust-rdkafka/tree/master/rdkafka-sys/README.md#features
262//! [rdkafka-sys-known-issues]: https://github.com/fede1024/rust-rdkafka/tree/master/rdkafka-sys/README.md#known-issues
263//! [smol]: https://docs.rs/smol
264//! [timely-blog]: https://github.com/frankmcsherry/blog/blob/master/posts/2017-11-08.md
265//! [timely-dataflow]: https://github.com/frankmcsherry/timely-dataflow
266//! [Tokio]: https://tokio.rs/
267
268#![forbid(missing_docs)]
269#![deny(rust_2018_idioms)]
270#![allow(clippy::type_complexity)]
271#![cfg_attr(docsrs, feature(doc_cfg))]
272
273mod log;
274
275pub use rdkafka_sys::types;
276
277pub mod admin;
278pub mod client;
279pub mod config;
280pub mod consumer;
281pub mod error;
282pub mod groups;
283pub mod message;
284pub mod metadata;
285pub mod producer;
286pub mod statistics;
287pub mod topic_partition_list;
288pub mod util;
289
290// Re-exports.
291pub use crate::client::ClientContext;
292pub use crate::config::ClientConfig;
293pub use crate::message::{Message, Timestamp};
294pub use crate::statistics::Statistics;
295pub use crate::topic_partition_list::{Offset, TopicPartitionList};
296pub use crate::util::IntoOpaque;