Re: Consumer Client - How to simulate heartbeats ?
Yes, you're right. No need to seek the offsets in onPartitionsAssigned method internally KafkaConsumer handles it. Thanks for sharing this with me. I'll update it. --Kamal On Mon, Apr 18, 2016 at 7:29 PM, Florian Hussonnoiswrote: > Yes, but the ConsumerRebalanceListener is optional and by the default > KafkaConsumer uses a NoOpConsumerRebalanceListener if no one is provided. > > I think the seek() is already done internally when a consumer joins or > quits the group. I'm not sure this line is actually needed. > > 2016-04-18 15:31 GMT+02:00 Kamal C : > > > When a new consumer joins to the group, it should start to read data > > from where the other consumer left. > > > > --Kamal > > > > On Mon, Apr 18, 2016 at 6:58 PM, Florian Hussonnois < > fhussonn...@gmail.com > > > > > wrote: > > > > > Thank you very much, the example is really helpful. > > > > > > My last question is : Why is it necessay to seek the consumer offsets > > into > > > the onPartitionsAssigned method ? > > > > > > > > > > > > https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java#L120 > > > > > > 2016-04-15 15:06 GMT+02:00 Kamal C : > > > > > > > Hi Florian, > > > > > > > > This may be helpful > > > > > > > > > > > > > > https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java > > > > > > > > --Kamal > > > > > > > > On Fri, Apr 15, 2016 at 2:57 AM, Jason Gustafson > > > > > wrote: > > > > > > > > > Hi Florian, > > > > > > > > > > It's actually OK if processing takes longer than the heartbeat > > > interval, > > > > > but it does need to finish before the session timeout expires or > the > > > > > consumer will be kicked out of the group (which typically is > revealed > > > by > > > > > commit failures). If the problem is just that the consumer is > > handling > > > > too > > > > > many messages at once, then Kafka 0.10 has an option to tune the > > number > > > > of > > > > > messages returned from poll() (max.poll.records), which may be > > helpful. > > > > We > > > > > also have a pause/resume API which allows you to call poll() > without > > > > > consuming any data. That's the best option at the moment for 0.9 > > > > consumers. > > > > > > > > > > For what it's worth, we've considered several times adding a > > > heartbeat() > > > > > API, but the challenge is figuring out how to handle rebalancing. > > > > > Underneath the covers, we use heartbeats to find out when the group > > is > > > > > rebalancing, so a heartbeat() option would probably have to return > a > > > flag > > > > > indicating whether a rebalance was needed. If the group has begun > > > > > rebalancing, then you would need to call poll() before the > expiration > > > of > > > > > the session timeout so that the consumer can join the rebalance. > > > > > Alternatively, we could let heartbeat() complete the rebalance > > itself, > > > > but > > > > > then you'd have to be prepared to abort processing from the > rebalance > > > > > callback. That's not really different from calling poll() after > > pausing > > > > > partitions though. The main problem in any case is that once a > > > rebalance > > > > > begins, you have the duration of the session timeout to stop > > processing > > > > and > > > > > join the rebalance. We're seeing this problem pop up pretty much > > > > everywhere > > > > > that the consumer is used, so we're trying to think of some better > > > > options > > > > > to handle it. > > > > > > > > > > Thanks, > > > > > Jason > > > > > > > > > > > > > > > On Thu, Apr 14, 2016 at 12:32 PM, Florian Hussonnois < > > > > > fhussonn...@gmail.com> > > > > > wrote: > > > > > > > > > > > Hi everyone, > > > > > > > > > > > > I have a use case where a message can take longer than ' > > > > > > heartbeat.interval.ms' to be processed by my application. As I > > > > > understand > > > > > > the heartbeats of consumer are done while the poll method is > > invoked. > > > > > > > > > > > > I would like to instantiate a worker thread to process the > messages > > > > but I > > > > > > need to wait for the messages completion before polling again. > > > > > > > > > > > > Is there a way to force the consumer to make an heartbeat without > > > > polling > > > > > > new messages ? > > > > > > > > > > > > Thanks, > > > > > > > > > > > > -- > > > > > > Florian HUSSONNOIS > > > > > > > > > > > > > > > > > > > > > > > > > > > -- > > > Florian HUSSONNOIS > > > > > > > > > -- > Florian HUSSONNOIS >
Re: Consumer Client - How to simulate heartbeats ?
Yes, but the ConsumerRebalanceListener is optional and by the default KafkaConsumer uses a NoOpConsumerRebalanceListener if no one is provided. I think the seek() is already done internally when a consumer joins or quits the group. I'm not sure this line is actually needed. 2016-04-18 15:31 GMT+02:00 Kamal C: > When a new consumer joins to the group, it should start to read data > from where the other consumer left. > > --Kamal > > On Mon, Apr 18, 2016 at 6:58 PM, Florian Hussonnois > > wrote: > > > Thank you very much, the example is really helpful. > > > > My last question is : Why is it necessay to seek the consumer offsets > into > > the onPartitionsAssigned method ? > > > > > > > https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java#L120 > > > > 2016-04-15 15:06 GMT+02:00 Kamal C : > > > > > Hi Florian, > > > > > > This may be helpful > > > > > > > > > https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java > > > > > > --Kamal > > > > > > On Fri, Apr 15, 2016 at 2:57 AM, Jason Gustafson > > > wrote: > > > > > > > Hi Florian, > > > > > > > > It's actually OK if processing takes longer than the heartbeat > > interval, > > > > but it does need to finish before the session timeout expires or the > > > > consumer will be kicked out of the group (which typically is revealed > > by > > > > commit failures). If the problem is just that the consumer is > handling > > > too > > > > many messages at once, then Kafka 0.10 has an option to tune the > number > > > of > > > > messages returned from poll() (max.poll.records), which may be > helpful. > > > We > > > > also have a pause/resume API which allows you to call poll() without > > > > consuming any data. That's the best option at the moment for 0.9 > > > consumers. > > > > > > > > For what it's worth, we've considered several times adding a > > heartbeat() > > > > API, but the challenge is figuring out how to handle rebalancing. > > > > Underneath the covers, we use heartbeats to find out when the group > is > > > > rebalancing, so a heartbeat() option would probably have to return a > > flag > > > > indicating whether a rebalance was needed. If the group has begun > > > > rebalancing, then you would need to call poll() before the expiration > > of > > > > the session timeout so that the consumer can join the rebalance. > > > > Alternatively, we could let heartbeat() complete the rebalance > itself, > > > but > > > > then you'd have to be prepared to abort processing from the rebalance > > > > callback. That's not really different from calling poll() after > pausing > > > > partitions though. The main problem in any case is that once a > > rebalance > > > > begins, you have the duration of the session timeout to stop > processing > > > and > > > > join the rebalance. We're seeing this problem pop up pretty much > > > everywhere > > > > that the consumer is used, so we're trying to think of some better > > > options > > > > to handle it. > > > > > > > > Thanks, > > > > Jason > > > > > > > > > > > > On Thu, Apr 14, 2016 at 12:32 PM, Florian Hussonnois < > > > > fhussonn...@gmail.com> > > > > wrote: > > > > > > > > > Hi everyone, > > > > > > > > > > I have a use case where a message can take longer than ' > > > > > heartbeat.interval.ms' to be processed by my application. As I > > > > understand > > > > > the heartbeats of consumer are done while the poll method is > invoked. > > > > > > > > > > I would like to instantiate a worker thread to process the messages > > > but I > > > > > need to wait for the messages completion before polling again. > > > > > > > > > > Is there a way to force the consumer to make an heartbeat without > > > polling > > > > > new messages ? > > > > > > > > > > Thanks, > > > > > > > > > > -- > > > > > Florian HUSSONNOIS > > > > > > > > > > > > > > > > > > > > -- > > Florian HUSSONNOIS > > > -- Florian HUSSONNOIS
Re: Consumer Client - How to simulate heartbeats ?
When a new consumer joins to the group, it should start to read data from where the other consumer left. --Kamal On Mon, Apr 18, 2016 at 6:58 PM, Florian Hussonnoiswrote: > Thank you very much, the example is really helpful. > > My last question is : Why is it necessay to seek the consumer offsets into > the onPartitionsAssigned method ? > > > https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java#L120 > > 2016-04-15 15:06 GMT+02:00 Kamal C : > > > Hi Florian, > > > > This may be helpful > > > > > https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java > > > > --Kamal > > > > On Fri, Apr 15, 2016 at 2:57 AM, Jason Gustafson > > wrote: > > > > > Hi Florian, > > > > > > It's actually OK if processing takes longer than the heartbeat > interval, > > > but it does need to finish before the session timeout expires or the > > > consumer will be kicked out of the group (which typically is revealed > by > > > commit failures). If the problem is just that the consumer is handling > > too > > > many messages at once, then Kafka 0.10 has an option to tune the number > > of > > > messages returned from poll() (max.poll.records), which may be helpful. > > We > > > also have a pause/resume API which allows you to call poll() without > > > consuming any data. That's the best option at the moment for 0.9 > > consumers. > > > > > > For what it's worth, we've considered several times adding a > heartbeat() > > > API, but the challenge is figuring out how to handle rebalancing. > > > Underneath the covers, we use heartbeats to find out when the group is > > > rebalancing, so a heartbeat() option would probably have to return a > flag > > > indicating whether a rebalance was needed. If the group has begun > > > rebalancing, then you would need to call poll() before the expiration > of > > > the session timeout so that the consumer can join the rebalance. > > > Alternatively, we could let heartbeat() complete the rebalance itself, > > but > > > then you'd have to be prepared to abort processing from the rebalance > > > callback. That's not really different from calling poll() after pausing > > > partitions though. The main problem in any case is that once a > rebalance > > > begins, you have the duration of the session timeout to stop processing > > and > > > join the rebalance. We're seeing this problem pop up pretty much > > everywhere > > > that the consumer is used, so we're trying to think of some better > > options > > > to handle it. > > > > > > Thanks, > > > Jason > > > > > > > > > On Thu, Apr 14, 2016 at 12:32 PM, Florian Hussonnois < > > > fhussonn...@gmail.com> > > > wrote: > > > > > > > Hi everyone, > > > > > > > > I have a use case where a message can take longer than ' > > > > heartbeat.interval.ms' to be processed by my application. As I > > > understand > > > > the heartbeats of consumer are done while the poll method is invoked. > > > > > > > > I would like to instantiate a worker thread to process the messages > > but I > > > > need to wait for the messages completion before polling again. > > > > > > > > Is there a way to force the consumer to make an heartbeat without > > polling > > > > new messages ? > > > > > > > > Thanks, > > > > > > > > -- > > > > Florian HUSSONNOIS > > > > > > > > > > > > > -- > Florian HUSSONNOIS >
Re: Consumer Client - How to simulate heartbeats ?
Thank you very much, the example is really helpful. My last question is : Why is it necessay to seek the consumer offsets into the onPartitionsAssigned method ? https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java#L120 2016-04-15 15:06 GMT+02:00 Kamal C: > Hi Florian, > > This may be helpful > > https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java > > --Kamal > > On Fri, Apr 15, 2016 at 2:57 AM, Jason Gustafson > wrote: > > > Hi Florian, > > > > It's actually OK if processing takes longer than the heartbeat interval, > > but it does need to finish before the session timeout expires or the > > consumer will be kicked out of the group (which typically is revealed by > > commit failures). If the problem is just that the consumer is handling > too > > many messages at once, then Kafka 0.10 has an option to tune the number > of > > messages returned from poll() (max.poll.records), which may be helpful. > We > > also have a pause/resume API which allows you to call poll() without > > consuming any data. That's the best option at the moment for 0.9 > consumers. > > > > For what it's worth, we've considered several times adding a heartbeat() > > API, but the challenge is figuring out how to handle rebalancing. > > Underneath the covers, we use heartbeats to find out when the group is > > rebalancing, so a heartbeat() option would probably have to return a flag > > indicating whether a rebalance was needed. If the group has begun > > rebalancing, then you would need to call poll() before the expiration of > > the session timeout so that the consumer can join the rebalance. > > Alternatively, we could let heartbeat() complete the rebalance itself, > but > > then you'd have to be prepared to abort processing from the rebalance > > callback. That's not really different from calling poll() after pausing > > partitions though. The main problem in any case is that once a rebalance > > begins, you have the duration of the session timeout to stop processing > and > > join the rebalance. We're seeing this problem pop up pretty much > everywhere > > that the consumer is used, so we're trying to think of some better > options > > to handle it. > > > > Thanks, > > Jason > > > > > > On Thu, Apr 14, 2016 at 12:32 PM, Florian Hussonnois < > > fhussonn...@gmail.com> > > wrote: > > > > > Hi everyone, > > > > > > I have a use case where a message can take longer than ' > > > heartbeat.interval.ms' to be processed by my application. As I > > understand > > > the heartbeats of consumer are done while the poll method is invoked. > > > > > > I would like to instantiate a worker thread to process the messages > but I > > > need to wait for the messages completion before polling again. > > > > > > Is there a way to force the consumer to make an heartbeat without > polling > > > new messages ? > > > > > > Thanks, > > > > > > -- > > > Florian HUSSONNOIS > > > > > > -- Florian HUSSONNOIS
Re: Consumer Client - How to simulate heartbeats ?
Hi Florian, This may be helpful https://github.com/omkreddy/kafka-examples/blob/master/consumer/src/main/java/kafka/examples/consumer/advanced/AdvancedConsumer.java --Kamal On Fri, Apr 15, 2016 at 2:57 AM, Jason Gustafsonwrote: > Hi Florian, > > It's actually OK if processing takes longer than the heartbeat interval, > but it does need to finish before the session timeout expires or the > consumer will be kicked out of the group (which typically is revealed by > commit failures). If the problem is just that the consumer is handling too > many messages at once, then Kafka 0.10 has an option to tune the number of > messages returned from poll() (max.poll.records), which may be helpful. We > also have a pause/resume API which allows you to call poll() without > consuming any data. That's the best option at the moment for 0.9 consumers. > > For what it's worth, we've considered several times adding a heartbeat() > API, but the challenge is figuring out how to handle rebalancing. > Underneath the covers, we use heartbeats to find out when the group is > rebalancing, so a heartbeat() option would probably have to return a flag > indicating whether a rebalance was needed. If the group has begun > rebalancing, then you would need to call poll() before the expiration of > the session timeout so that the consumer can join the rebalance. > Alternatively, we could let heartbeat() complete the rebalance itself, but > then you'd have to be prepared to abort processing from the rebalance > callback. That's not really different from calling poll() after pausing > partitions though. The main problem in any case is that once a rebalance > begins, you have the duration of the session timeout to stop processing and > join the rebalance. We're seeing this problem pop up pretty much everywhere > that the consumer is used, so we're trying to think of some better options > to handle it. > > Thanks, > Jason > > > On Thu, Apr 14, 2016 at 12:32 PM, Florian Hussonnois < > fhussonn...@gmail.com> > wrote: > > > Hi everyone, > > > > I have a use case where a message can take longer than ' > > heartbeat.interval.ms' to be processed by my application. As I > understand > > the heartbeats of consumer are done while the poll method is invoked. > > > > I would like to instantiate a worker thread to process the messages but I > > need to wait for the messages completion before polling again. > > > > Is there a way to force the consumer to make an heartbeat without polling > > new messages ? > > > > Thanks, > > > > -- > > Florian HUSSONNOIS > > >
Re: Consumer Client - How to simulate heartbeats ?
Hi Florian, It's actually OK if processing takes longer than the heartbeat interval, but it does need to finish before the session timeout expires or the consumer will be kicked out of the group (which typically is revealed by commit failures). If the problem is just that the consumer is handling too many messages at once, then Kafka 0.10 has an option to tune the number of messages returned from poll() (max.poll.records), which may be helpful. We also have a pause/resume API which allows you to call poll() without consuming any data. That's the best option at the moment for 0.9 consumers. For what it's worth, we've considered several times adding a heartbeat() API, but the challenge is figuring out how to handle rebalancing. Underneath the covers, we use heartbeats to find out when the group is rebalancing, so a heartbeat() option would probably have to return a flag indicating whether a rebalance was needed. If the group has begun rebalancing, then you would need to call poll() before the expiration of the session timeout so that the consumer can join the rebalance. Alternatively, we could let heartbeat() complete the rebalance itself, but then you'd have to be prepared to abort processing from the rebalance callback. That's not really different from calling poll() after pausing partitions though. The main problem in any case is that once a rebalance begins, you have the duration of the session timeout to stop processing and join the rebalance. We're seeing this problem pop up pretty much everywhere that the consumer is used, so we're trying to think of some better options to handle it. Thanks, Jason On Thu, Apr 14, 2016 at 12:32 PM, Florian Hussonnoiswrote: > Hi everyone, > > I have a use case where a message can take longer than ' > heartbeat.interval.ms' to be processed by my application. As I understand > the heartbeats of consumer are done while the poll method is invoked. > > I would like to instantiate a worker thread to process the messages but I > need to wait for the messages completion before polling again. > > Is there a way to force the consumer to make an heartbeat without polling > new messages ? > > Thanks, > > -- > Florian HUSSONNOIS >
Consumer Client - How to simulate heartbeats ?
Hi everyone, I have a use case where a message can take longer than ' heartbeat.interval.ms' to be processed by my application. As I understand the heartbeats of consumer are done while the poll method is invoked. I would like to instantiate a worker thread to process the messages but I need to wait for the messages completion before polling again. Is there a way to force the consumer to make an heartbeat without polling new messages ? Thanks, -- Florian HUSSONNOIS