Kafka Engine: the Story Continues

If you love ClickHouse®, Altinity can simplify your experience with our fully managed service in your cloud (BYOC) or our cloud and 24/7 expert support. Learn more.
The Kafka table engine in ClickHouse® is a user-friendly way to read Kafka data into ClickHouse tables that does not require any extra components. It was contributed to ClickHouse by Cloudflare back in 2017. Since then it has been battle tested and used by thousands of ClickHouse installations. Below we give an overview of the Kafka engine and propose a list of improvements that should make it even better.
By the way, this is not a guide to using the Kafka table engine. If you want help on Kafka operation look at the Kafka Engine Tutorial and Kafka Engine FAQ. We plan to issue an update soon that covers the latest guidance on these topics.
The Kafka engine recap
The Kafka engine in ClickHouse is an interface to Kafka brokers. It is directly integrated into ClickHouse, and therefore available in any open source ClickHouse installation as well as Altinity.Cloud. As a consumer, Kafka engine handles communication with the broker, pulls messages, and sends them to ClickHouse in batches. The batch of messages appears as an insert block, and can be later processed by Materialized View in a natural ClickHouse-way:

Kafka engine can handle multiple input formats, e.g. JSON or Protobuf, and allows flexible configuration. Multiple Kafka engine tables can be used in one ClickHouse server process, allowing to consume from multiple topics and write to multiple tables. It scales well across multiple cores and multiple ClickHouse nodes. We could see ClickHouse servers consuming millions of records per second from Kafka using the Kafka engine.
Before going to the proposed improvements it is important to remember the history of the Kafka engine development.
History of the Kafka table engine
The Kafka engine was added into ClickHouse in late 2017 by Marek Vavruša (@vavrusa) from CloudFlare. It became a part of CloudFlare analytics architecture described in the groundbreaking article “HTTP Analytics for 6M requests per second using ClickHouse” in early 2018. Throughout 2018, Marek Vavruša worked on enhancing the stability of the Kafka engine. Eventually, CloudFlare shifted focus to other topics, leaving the support of the Kafka engine to the ClickHouse community.
The initial CloudFlare implementation was minimalistic and trivial. However, it showed great potential. The development continued by the Yandex team. By the end of 2019 the Kafka engine had pretty much all the main features that make it popular today. Integration with Materialized Views, atomic inserts, virtual columns, and many stability fixes made it ready for generic applications. The year after focus switched to performance improvements and multi-threading, support for various input formats, Kerberos authentication, and improving deduplication logic.
At this time Altinity started to play a very active role in maintaining the Kafka engine, doing a lot of low level work – analyzing bug reports, making small fixes, ensuring at-least-once semantics and so on. For example, in 2020 Mikhail Filimonov from Altinity alone submitted 30 pull requests to ClickHouse, some of which covered multiple Kafka engine issues and improvements. We also published the Kafka Engine Tutorial, articles, and webinars to educate users how to use the Kafka engine efficiently. This paid back with increased popularity of the Kafka engine especially in cloud deployments of ClickHouse where both ClickHouse and Kafka can be used as managed services, thereby reducing operational overheads.
In the 2021 the Kafka engine entered a maintenance phase There were small bug fixes, added support for Parquet and Arrow, and improved configuration options. The most important usability change of that time was adding the kafka_handle_error_mode feature, which allowed users to capture malformed messages in a separate table. In 2022 and 2023, the maintenance work was mostly performed by our friends from SemRush, but Altinity also added the system.kafka_consumers table and kafka events for better observability.
Making the Kafka Engine Better
Now it is 2024. So we stepped back, analyzed our experience helping hundreds of the Kafka engine users at Altinity.Cloud and on prem, and asked ourselves: what are the typical issues users are facing and how can we fix them in ClickHouse? The result is the list below that is a part of our “Building a Better ClickHouse” roadmap. In more detail it is also available as a GitHub issue: https://github.com/ClickHouse/ClickHouse/issues/65297
Error handling
Error handling in the Kafka engine is inconvenient and requires a lot of care. One malformed message may block consumers. The following improvements can make it better:
- Add exponential backoff for retries. One failing consumer can currently hurt the whole cluster, because it will be leaving/entering consumer group all the time, which is a ‘stop the world’ operation.
- Add two more error handling modes ( kafka_handle_error_mode):
- ‘ignore’ – ignore all errors completely, just increase error counters
- ‘dead_letter_queue`. This mode would redirect the erroneous messages to a dedicated table
Usability
The following improvements can improve usability of the Kafka engine in general:
- Round-robin consumption of several partitions. Currently in the case of a big lag ClickHouse will consume partitions one after another, i.e. the first partition till the end, then the second and so on. Because of that the maximum lag stays high all the time, and the lag in time (not in messages) starts decreasing only after the last partition starts to be consumed. See
- Schema inference for Kafka. One possible option is to add a kafka() table function kafka for schema inference: DESCRIBE kafka(…). But it could be others as well.
See the Kafka Schema Inference article in Altinity Knowledge Base. - Allow to manage consumer group offsets in SQL (e.g. to skip something, retry something, rewind back etc.). It can be implemented via SYSTEM commands or ALTER TABLE MODIFY SETTING.
Exactly-once message delivery (EOS)
The current implementation of the Kafka engine guarantees at-least-once semantics. That means a message is guaranteed to be inserted into ClickHouse, but in rare cases it can be inserted twice.
This happens as follows in the current implementation. When the Kafka engine consumes messages, there are two commits performed into two different distributed systems: first commit is data flush into ClickHouse, second – commit offset into Kafka. That makes it impossible to guarantee exactly once semantics: if Kafka becomes unavailable while we are flushing data into ClickHouse we will not be able to commit and can get duplicates. (It’s a variant of the ‘Two Generals’ Problem). In order to guarantee exactly once message delivery, there should be a single commit, and offsets should be committed together with data.
We have a plan for how to solve this problem with incremental changes, but the details are outside of the scope of this article.
Housekeeping
There are a number of small things that can be done in order to make implementation cleaner.
- Clean startup/shutdown. The initialization sequence of different table types in ClickHouse is not deterministic. Therefore it is possible that the Kafka engine starts consuming before the destination table is ready. Same is true for stopping the server. That leads to unnecessary errors in the log and potential consistency issues.
- Have a separate auto-expandable pool for consumers. The current implementation re-uses one of the existing pools, which adds unneeded dependency and confusion.
- Improve documentation
- Upgrade to the latest version of librdkafka – the C++ library that encapsulates consumer logic.
Research
There are some areas that require additional research before committing any work.
- Benchmark consumer performance for different formats and investigate if it can be improved. For example, there are several reports that consuming Protobuf messages is too slow.
- Research / document sticky consumers, so the partition will never be moved between shards
- Backup Kafka offsets, that would allow accurate restart when cluster is restored from a backup
- Kafka engine as a producer. Currently the Kafka engine as a producer can be used only for simple cases like signaling some anomalies, i.e. sending a few messages sometimes, not a big stream
Kafka Engine vs Kafka Connect
The Kafka engine is not the only way ClickHouse can integrate with Kafka. One can write custom consumers that insert into ClickHouse. There is also ClickHouse Kafka Connect Sink. While it addresses the same use case – ingest data from Kafka to ClickHouse – it uses a totally different approach compared to the Kafka engine.
The main difference is that ClickHouse Kafka Connect is a separate tool. It is developed using the Kafka Connect framework from Confluent and distributed as a ‘jar’ file that needs to be deployed and maintained separately. As such – it is not a part of ClickHouse and has its own release cycle and maintenance process.
One of the main features that ClickHouse Kafka Connect offers is exactly-once-delivery. It may store offsets in Keeper, that makes it more reliable. A similar solution for the Kafka engine is currently in development (see #57625).
The main tradeoff of ClickHouse Kafka Connect is that it adds an extra component to the ingest pipeline that must be separately configured and maintained. It requires the Kafka Connect framework to run, which adds additional infrastructure to pipelines. It also requires support from Kafka service providers. The Kafka Engine on the other hand is embedded in ClickHouse, so the full ingest pipeline can be managed entirely in ClickHouse using SQL.Users can decide whether which of these tradeoffs meet their needs.
Final words
The proposed improvements are published in more detail in the ClickHouse GitHub repository https://github.com/ClickHouse/ClickHouse/issues/65297. We welcome the ClickHouse community to work together on the Kafka engine improvements as well as other features that make ClickHouse a better database for users.
In case you missed it, check out Building a Better ClickHouse article that introduces ClickHouse community roadmap and enumerates important ClickHouse features that need to be improved or (re-)implemented. We are excited to work on the best open source analytical database on the planet, and welcome everybody to share this joy!
ClickHouse® is a registered trademark of ClickHouse, Inc.; Altinity is not affiliated with or associated with ClickHouse, Inc.