[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-23 Thread tzulitai
Github user tzulitai commented on the issue: https://github.com/apache/flink/pull/4301 This was merged via #4357. Closing .. --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-17 Thread aljoscha
Github user aljoscha commented on the issue: https://github.com/apache/flink/pull/4301 Oye, this is more complicated than I thought. On `release-1.3` the assignment actually works if the Kafka brokers always return the partitions in the same order. The reason is that the assignment

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-17 Thread aljoscha
Github user aljoscha commented on the issue: https://github.com/apache/flink/pull/4301 Note, that this doesn't normally occur because the strategy for assigning Kafka partitions and for assigning operator state is the same (right now). However, this means that you will have active

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-17 Thread aljoscha
Github user aljoscha commented on the issue: https://github.com/apache/flink/pull/4301 Yes, I don't think we can get around this when restoring from "old" state. I also have another suspicion: I don't think that

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-14 Thread tzulitai
Github user tzulitai commented on the issue: https://github.com/apache/flink/pull/4301 @aljoscha on some second thinking, I don't think we can easily deal with the fact that, when restoring from 1.3.1 / 1.3.0 savepoints in 1.3.2, users will not benefit from this bug fix.

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-12 Thread tzulitai
Github user tzulitai commented on the issue: https://github.com/apache/flink/pull/4301 @StephanEwen Regarding no-rediscover on restore test: yes, could say that it is covered in `KafkaConsumerTestBase.runMultipleSourcesOnePartitionExactlyOnceTest()`. It's an end-to-end

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-12 Thread StephanEwen
Github user StephanEwen commented on the issue: https://github.com/apache/flink/pull/4301 Do we have a test for the case where there are fewer partitions than sources so that some sources do not get partitions on restore? To make sure they do not accidentally re-discover? --- If

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-12 Thread StephanEwen
Github user StephanEwen commented on the issue: https://github.com/apache/flink/pull/4301 Just to double-check: I see that the state of the partitions is in a `ListState`. That means after recovery, they can be differently distributed than before. Does that not conflict with the

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-12 Thread tzulitai
Github user tzulitai commented on the issue: https://github.com/apache/flink/pull/4301 This would then mean we discourage restoring from a 1.3.x savepoint, because the state is potentially incorrect. I wonder if we then actually want to always fetch partitions on startup (fresh

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-12 Thread tzulitai
Github user tzulitai commented on the issue: https://github.com/apache/flink/pull/4301 Yes, that is true. This assignment logic is only applied on fresh starts. --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-12 Thread aljoscha
Github user aljoscha commented on the issue: https://github.com/apache/flink/pull/4301 Quick comment for my own clarification: when restoring from a 1.3.x savepoint, the new assignment logic will not be used, right? In Flink 1.3.x there is no dynamic partition discovery and so when

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-12 Thread tzulitai
Github user tzulitai commented on the issue: https://github.com/apache/flink/pull/4301 @StephanEwen thanks for the review. Your suggestion makes a lot of sense. I've fixed this up as the following: - Have a new method `KafkaTopicAssigner.assign(KafkaTopicPartition

[GitHub] flink issue #4301: (release-1.3) [FLINK-7143] [kafka] Fix indeterminate part...

2017-07-11 Thread StephanEwen
Github user StephanEwen commented on the issue: https://github.com/apache/flink/pull/4301 I think that would fix the bug. There are two things I would like to improve, though: 1. Relying on `hashCode()` makes very implicit assumptions about the behavior of the hash code