[
https://issues.apache.org/jira/browse/KAFKA-6906?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16492827#comment-16492827
]
Jagadesh Adireddi edited comment on KAFKA-6906 at 5/28/18 5:06 PM:
-------------------------------------------------------------------
Hi [~mjsax],
Sorry for the late response. I am on vacation. I want to re-iterate the
solution provided from above comments.
1) When EOS is turned on, we need to call StreamTask#commitOffsets for both
`PipeInput` and `advanceWallClockTime` cases.
2) We need to delete `commitOffsetNeeded` field in `StreamTask` class.
3) For each topic partition in StreamTask#{{consumedOffsets, we need to filter
out topic partitions that have changes from last commit. And commit
transaction.}}
{{4) And partitions with no changes from last commit Map, need to sent to
`}}{{commitSync}}{{` method, once KIP-211 is merged.}}
{{Can you please clarify whether point 3 & 4 applicable irrespective of EOS is
turned-ON/OFF.And}}{{ }}*consumedOffsets* Map or *stateMgr* Map in `StreamTask`
gives the information of last commit?
{{ }}
`
`
was (Author: [email protected]):
Hi [~mjsax],
Sorry for the late response. I am on vacation. I want to re-iterate the
solution provided from above comments.
1) When EOS is turned on, we need to call StreamTask#commitOffsets for both
`PipeInput` and `advanceWallClockTime` cases.
2) We need to delete `commitOffsetNeeded` field in `StreamTask` class.
3) For each topic partition in StreamTask#{{consumedOffsets, we need to filter
out topic partitions that have changes from last commit. And commit
transaction.}}
{{4) And partitions with no changes from last commit Map, need to sent to
`}}{{commitSync}}{{` method, once KIP-211 is merged.}}
{{Can you please clarify whether point 3 & 4 applicable in both cases EOS is
turned-ON/OFF.And}}{{ }}*consumedOffsets* Map or *stateMgr* Map in `StreamTask`
gives the information of last commit?
{{ }}
`
`
> Kafka Streams does not commit transactions if data is produced via wall-clock
> punctuation
> -----------------------------------------------------------------------------------------
>
> Key: KAFKA-6906
> URL: https://issues.apache.org/jira/browse/KAFKA-6906
> Project: Kafka
> Issue Type: Bug
> Components: streams
> Affects Versions: 1.1.0
> Reporter: Matthias J. Sax
> Assignee: Jagadesh Adireddi
> Priority: Major
>
> Committing in Kafka Streams happens in regular intervals. However, committing
> only happens if new input records got processed since the last commit (via
> setting flag `commitOffsetNeeded` within `StreamTask#process()`)
> However, data could also be emitted via wall-clock based punctuation calls.
> Especially if EOS is enabled, this is an issue (maybe also for non-EOS)
> because the current running transaction is not committed and thus might time
> out leading to a fatal error.
--
This message was sent by Atlassian JIRA
(v7.6.3#76005)