[ 
https://issues.apache.org/jira/browse/KAFKA-7548?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Sean Glover updated KAFKA-7548:
-------------------------------
    Description: 
Today when we call KafkaConsumer.poll(), it will fetch data from Kafka 
asynchronously and is put in to a local buffer (completedFetches).

If now we pause some TopicPartitions and call KafkaConsumer.poll(), we might 
throw away any buffered data that we might have in the local buffer for these 
TopicPartitions. Generally, if an application is calling pause on some 
TopicPartitions, it is likely to resume those TopicPartitions in near future, 
which would require KafkaConsumer to re-issue a fetch for the same data that it 
had buffered earlier for these TopicPartitions. This is a wasted effort from 
the application's point of view.

At Linkedin, we made a hotfix to see if NOT throwing away the prefetched data 
would improve the performance for stream applications like Samza. We ran a 
benchmark to compare the "before-fix" and "after-fix" versions.

We had a consumer subscribed to 10 partitions of a high volume topic and paused 
predefined number partitions for every poll call. The partitions to pause were 
chosen randomly for each poll() call.
 * Time to run Benchmark = 60 seconds.
 * MaxPollRecords = 1
 * Number of TopicPartition subscribed  = 10 

||Number Of Partitions Paused||Number of Records consumed (Before fix)||Number 
of Records consumed (After fix)||
|9|2087|4884693|

 

_Updated June 24, 2019_

I followed up with [~mgharat] on the status of this work since the current 
patch PR is stale.  This work would also be beneficial to the Alpakka Kafka 
connector, which frequently pauses partitions as a means of back-pressure from 
upstream Akka Streams graph stages.  I've reviewed the PR and

  was:
Today when we call KafkaConsumer.poll(), it will fetch data from Kafka 
asynchronously and is put in to a local buffer (completedFetches).

If now we pause some TopicPartitions and call KafkaConsumer.poll(), we might 
throw away any buffered data that we might have in the local buffer for these 
TopicPartitions. Generally, if an application is calling pause on some 
TopicPartitions, it is likely to resume those TopicPartitions in near future, 
which would require KafkaConsumer to re-issue a fetch for the same data that it 
had buffered earlier for these TopicPartitions. This is a wasted effort from 
the application's point of view.

At Linkedin, we made a hotfix to see if NOT throwing away the prefetched data 
would improve the performance for stream applications like Samza. We ran a 
benchmark to compare the "before-fix" and "after-fix" versions.

We had a consumer subscribed to 10 partitions of a high volume topic and paused 
predefined number partitions for every poll call. The partitions to pause were 
chosen randomly for each poll() call.
 * Time to run Benchmark = 60 seconds.
 * MaxPollRecords = 1
 * Number of TopicPartition subscribed  = 10 

||Number Of Partitions Paused||Number of Records consumed (Before fix)||Number 
of Records consumed (After fix)||
|9|2087|4884693|

 


> KafkaConsumer should not throw away already fetched data for paused 
> partitions.
> -------------------------------------------------------------------------------
>
>                 Key: KAFKA-7548
>                 URL: https://issues.apache.org/jira/browse/KAFKA-7548
>             Project: Kafka
>          Issue Type: Improvement
>          Components: clients
>            Reporter: Mayuresh Gharat
>            Assignee: Mayuresh Gharat
>            Priority: Major
>
> Today when we call KafkaConsumer.poll(), it will fetch data from Kafka 
> asynchronously and is put in to a local buffer (completedFetches).
> If now we pause some TopicPartitions and call KafkaConsumer.poll(), we might 
> throw away any buffered data that we might have in the local buffer for these 
> TopicPartitions. Generally, if an application is calling pause on some 
> TopicPartitions, it is likely to resume those TopicPartitions in near future, 
> which would require KafkaConsumer to re-issue a fetch for the same data that 
> it had buffered earlier for these TopicPartitions. This is a wasted effort 
> from the application's point of view.
> At Linkedin, we made a hotfix to see if NOT throwing away the prefetched data 
> would improve the performance for stream applications like Samza. We ran a 
> benchmark to compare the "before-fix" and "after-fix" versions.
> We had a consumer subscribed to 10 partitions of a high volume topic and 
> paused predefined number partitions for every poll call. The partitions to 
> pause were chosen randomly for each poll() call.
>  * Time to run Benchmark = 60 seconds.
>  * MaxPollRecords = 1
>  * Number of TopicPartition subscribed  = 10 
> ||Number Of Partitions Paused||Number of Records consumed (Before 
> fix)||Number of Records consumed (After fix)||
> |9|2087|4884693|
>  
> _Updated June 24, 2019_
> I followed up with [~mgharat] on the status of this work since the current 
> patch PR is stale.  This work would also be beneficial to the Alpakka Kafka 
> connector, which frequently pauses partitions as a means of back-pressure 
> from upstream Akka Streams graph stages.  I've reviewed the PR and



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to