Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Fully utilize the max.in.flight.requests.per.connection parameter on the idempotent producer #4989

Open
wants to merge 1 commit into
base: master
Choose a base branch
from

Conversation

marcin-krystianc
Copy link

Currently, the librdkafka implementation sends only one produce request at a time, regardless of the configured value for the max.in.flight.requests.per.connection parameter. This limitation prevents the producer from fully leveraging the potential of concurrent requests, which can impact throughput and performance.

With the proposed changes, the max.in.flight.requests.per.connection parameter will be respected, allowing the number of concurrent requests to be aligned with the specified value. This enhancement aims to optimize the idempotent producer's performance by enabling multiple produce requests to be sent simultaneously, thereby improving overall latency.

To validate my findings, I conducted tests on a network with a simulated delay of 1000 ms, configured using the tc command (tc qdisc add dev eth0 root netem delay 1000ms). I have also attached pcap files for your review.

Relevant producer parameters:

--config enable.idempotence=true
--config max.in.flight.requests.per.connection=5
--config linger.ms=200
--config socket.nagle.disable=true
418	13:01:07.506672	172.18.0.4	172.18.0.3	59686,19092	Kafka Produce v10 Request
432	13:01:08.508093	172.18.0.3	172.18.0.4	19092,59686	Kafka Produce v10 Response
454	13:01:08.508619	172.18.0.4	172.18.0.3	59686,19092	Kafka Produce v10 Request
459	13:01:09.509790	172.18.0.3	172.18.0.4	19092,59686	Kafka Produce v10 Response
467	13:01:09.510013	172.18.0.4	172.18.0.3	59686,19092	Kafka Produce v10 Request
474	13:01:10.511038	172.18.0.3	172.18.0.4	19092,59686	Kafka Produce v10 Response
482	13:01:10.511223	172.18.0.4	172.18.0.3	59686,19092	Kafka Produce v10 Request
487	13:01:11.512288	172.18.0.3	172.18.0.4	19092,59686	Kafka Produce v10 Response
495	13:01:11.512462	172.18.0.4	172.18.0.3	59686,19092	Kafka Produce v10 Request
500	13:01:12.513293	172.18.0.3	172.18.0.4	19092,59686	Kafka Produce v10 Response
508	13:01:12.513462	172.18.0.4	172.18.0.3	59686,19092	Kafka Produce v10 Request
237	12:58:32.288766	172.18.0.4	172.18.0.3	60190,19092	Kafka Produce v10 Request
239	12:58:32.288771	172.18.0.4	172.18.0.3	60190,19092	Kafka Produce v10 Request
246	12:58:32.329058	172.18.0.4	172.18.0.3	60190,19092	Kafka Produce v10 Request
247	12:58:32.329061	172.18.0.4	172.18.0.3	60190,19092	Kafka Produce v10 Request
250	12:58:32.788187	172.18.0.4	172.18.0.3	60190,19092	Kafka Produce v10 Request
260	12:58:33.301184	172.18.0.3	172.18.0.4	19092,60190	Kafka Produce v10 Response
271	12:58:33.301766	172.18.0.4	172.18.0.3	60190,19092	Kafka Produce v10 Request
272	12:58:33.302767	172.18.0.3	172.18.0.4	19092,60190	Kafka Produce v10 Response
275	12:58:33.330077	172.18.0.3	172.18.0.4	19092,60190	Kafka Produce v10 Response
277	12:58:33.330703	172.18.0.3	172.18.0.4	19092,60190	Kafka Produce v10 Response

@marcin-krystianc marcin-krystianc requested a review from a team as a code owner March 11, 2025 09:09
@confluent-cla-assistant
Copy link

🎉 All Contributor License Agreements have been signed. Ready to merge.
✅ marcin-krystianc
Please push an empty commit if you would like to re-run the checks to verify CLA status for all contributors.

@@ -4100,8 +4100,7 @@ static int rd_kafka_toppar_producer_serve(rd_kafka_broker_t *rkb,

/* Limit the number of in-flight requests (per partition)
* to the broker's sequence de-duplication window. */
max_requests = RD_MIN(max_requests,
RD_KAFKA_IDEMP_MAX_INFLIGHT - inflight);
Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

The inflight variable holds the number of inflight messages (rd_atomic32_get(&rktp->rktp_msgs_inflight)) instead of the number of inflight requests.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks @marcin-krystianc for spotting this! I'll check if other changes are needed but it doesn't seem at the moment.

Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment
Labels
Projects
None yet
Development

Successfully merging this pull request may close these issues.

2 participants