Offsets committed by multiple consumer group sessions can be re-ordered by the time they arrive at Kafka, potentially resulting in duplicate message delivery, should the offset be used to resume consumption.
If a single sarama.ConsumerGroup is consuming from more than one partition, then reordering of offset commit requests can occur.
This is because of a race condition between the goroutines used to run the ConsumerClaim method for different topic partitions.
When ConsumerGroupSession.Commit() is called, this ends up calling into offsetManager.flushToBroker().
This in turn:
Builds a offset commit request, using any so-far uncommitted offsets across all of the partitions belonging to the group
Finds the group's coordinator (if not already known)
Sends the offset commit request to the broker that is the group coordinator.
However, there is no locking to prevent the interleaving of two (or more) go routines between the building an offset commit request, and sending the request to Kafka.
For example, consider the following interactions between two goroutines P0 and P1 - each belonging to the same consumer group, but consuming from different partitions of a topic:
P0 marks offset 10 and calls commit (the uncommitted offsets tracked by the session are: {P0 -> 11})
P0 builds the offset commit request (containing {P0 -> 11}).
P0 goroutine yields, and P1 goroutine is scheduled
P1 marks offset 20 and calls commit (the uncommitted offsets tracked by the session are: {P0 -> 11, P1 -> 21})
P1 goroutine yields, and P0 goroutine is scheduled
P0 sends the offset commit ({P0 -> 11}) request, gets the response, and resumes running code in the ConsumerClaim method
P0 marks offset 11 and calls commit (the uncommitted offsets tracked by the session are {0->12, 1->21})
P0 builds the offset commit request (containing {0->12, 1->21}).
P0 sends its offset commit request (containing {0->12, 1->21})
P0 goroutine yields, and P1 goroutine is scheduled
Description
Offsets committed by multiple consumer group sessions can be re-ordered by the time they arrive at Kafka, potentially resulting in duplicate message delivery, should the offset be used to resume consumption.
If a single sarama.ConsumerGroup is consuming from more than one partition, then reordering of offset commit requests can occur. This is because of a race condition between the goroutines used to run the ConsumerClaim method for different topic partitions.
When ConsumerGroupSession.Commit() is called, this ends up calling into offsetManager.flushToBroker(). This in turn:
However, there is no locking to prevent the interleaving of two (or more) go routines between the building an offset commit request, and sending the request to Kafka.
For example, consider the following interactions between two goroutines P0 and P1 - each belonging to the same consumer group, but consuming from different partitions of a topic:
From the Kafka broker's perspective, it receives:
This causes the offset committed for P0 to move "backwards" from 12 to 11.
Due to the nature of this race condition, it is more likely to occur if:
Versions
Configuration
Logs
Not applicable. Sarama does not emit any logging relating to this problem.
Additional Context