[
https://issues.apache.org/jira/browse/STORM-822?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15302339#comment-15302339
]
ASF GitHub Bot commented on STORM-822:
--------------------------------------
Github user jianbzhou commented on the pull request:
https://github.com/apache/storm/pull/1131#issuecomment-221922582
@hmcl, currently if user give firstPollOffsetStrategy=UNCOMMITTED_LATEST or
LATEST, the spout will not work, because if a kafka consumer re-balance
happened, the offset will be seeked to the end, and there will be lots of
messages not consumed/emitted/acked&failed, so will never find the next
continuous offset to commit, so the log will keep showing that "Non continuous
offset found"......
I have a questions here - if a spout read and emit one message, I assume
storm will ensure the message will be acked or failed without exception, right?
because if it is possible that one emitted message failed to get acked or
failed message under some strange situations, it means we cannot find the
continuous message to commit, which will directly break the spout. Could you
please help confirm if my assumption is correct?
If my assumption is not correct - which means one emitted message may not
be able to get acked or failed message back, then we must change the spout(need
a timeout setting if failed to find next continuous message to commit) -
currently the spout will always find the next continuous message to commit, it
will try forever...
due to the spout will always find the next continuous message to commit, we
need to be cautious for below method:
private boolean poll() {
return !waitingToEmit() && emitted.size() <
kafkaSpoutConfig.getMaxUncommittedOffsets();
}
if the MaxUncommittedOffsets is too small, the spout will frequently stop
polling from kafka, if a rebalance happened and seek back to the failed
message, at this moment if the spout stop polling, will also cause the spout
failed to find the next committed message. Currently we set this value to
200000 and seems working fine for now.
Looking forward to hearing from you! thanks!
> Kafka Spout New Consumer API
> ----------------------------
>
> Key: STORM-822
> URL: https://issues.apache.org/jira/browse/STORM-822
> Project: Apache Storm
> Issue Type: Story
> Components: storm-kafka
> Reporter: Thomas Becker
> Assignee: Hugo Louro
> Fix For: 1.0.0, 2.0.0
>
>
--
This message was sent by Atlassian JIRA
(v6.3.4#6332)