[
https://issues.apache.org/jira/browse/FLINK-7195?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16087227#comment-16087227
]
ASF GitHub Bot commented on FLINK-7195:
---------------------------------------
GitHub user tzulitai opened a pull request:
https://github.com/apache/flink/pull/4344
(release-1.3) [FLINK-7195] [kafka] Remove partition list querying when
restoring state in FlinkKafkaConsumer
This issue is a re-appearance of FLINK-6006. On restore, we should not
respect any fetched partitions list from Kafka and perform any filtering of the
restored partition states. There are corner cases where, due to Kafka broker
downtime, some partitions may be missing in the fetched partition list.
Therefore, we should not respect the fetched partitions list on restore time to
manipulate the restored state, which may lead to broken state. To be more
precise, we actually should not require fetching partitions on restore.
We've stepped on our own foot again and reintroduced this bug in
ed68fedbe90db03823d75a020510ad3c344fa73e. This PR adds proper unit tests for
this that does not rely on the internal implementations and test only on public
abstractions of `FlinkKafkaConsumerBase`.
You can merge this pull request into a Git repository by running:
$ git pull https://github.com/tzulitai/flink FLINK-7195
Alternatively you can review and apply these changes as the patch at:
https://github.com/apache/flink/pull/4344.patch
To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:
This closes #4344
----
commit 12af5d8b0e43b62935dc619258fb8f957b11d0bc
Author: Tzu-Li (Gordon) Tai <[email protected]>
Date: 2017-07-14T11:51:03Z
[FLINK-7195] [kafka] Remove partition list querying when restoring state in
FlinkKafkaConsumer
----
> FlinkKafkaConsumer should not respect fetched partitions to filter restored
> partition states
> --------------------------------------------------------------------------------------------
>
> Key: FLINK-7195
> URL: https://issues.apache.org/jira/browse/FLINK-7195
> Project: Flink
> Issue Type: Bug
> Components: Kafka Connector
> Affects Versions: 1.3.1
> Reporter: Tzu-Li (Gordon) Tai
> Assignee: Tzu-Li (Gordon) Tai
> Priority: Blocker
> Fix For: 1.3.2
>
>
> This issue is a re-appearance of FLINK-6006. On restore, we should not
> respect any fetched partitions list from Kafka and perform any filtering of
> the restored partition states. There are corner cases where, due to Kafka
> broker downtime, some partitions may be missing in the fetched partition
> list. To be more precise, we actually should not require fetching partitions
> on restore.
> We've stepped on our own foot again and reintroduced this bug in
> https://github.com/apache/flink/pull/3378/commits/ed68fedbe90db03823d75a020510ad3c344fa73e.
> The previous test for this behavior was too implementation specific, and
> therefore the leak in catching this on different internal implementations.
> We should have a proper unit test for this that does not rely on the internal
> implementations and test only on public abstractions of
> {{FlinkKafkaConsumerBase}}.
--
This message was sent by Atlassian JIRA
(v6.4.14#64029)