Re: FlinkKafkaProducerXX

2017-12-04 Thread Mikhail Pryakhin
Exactly, at least it's worth mentioning the partitioner used by default in case none was specified in the javadoc, because the default behavior might not seem obvious. Kind Regards, Mike Pryakhin > On 3 Dec 2017, at 22:08, Stephan Ewen wrote: > > Sounds like adding a round robin partitioner

Re: FlinkKafkaProducerXX

2017-12-03 Thread Stephan Ewen
Sounds like adding a round robin partitioner to the set of readily available partitioners would make sense. On Fri, Dec 1, 2017 at 5:16 AM, Tzu-Li (Gordon) Tai wrote: > Hi Mike, > > The rationale behind implementing the FlinkFixedPartitioner as the default > is so that each Flink sink partition

Re: FlinkKafkaProducerXX

2017-11-30 Thread Tzu-Li (Gordon) Tai
Hi Mike, The rationale behind implementing the FlinkFixedPartitioner as the default is so that each Flink sink partition (i.e. one sink parallel subtask) maps to a single Kafka partition. One other thing to clarify: By setting the partitioner to null, the partitioning is based on a hash of the re

FlinkKafkaProducerXX

2017-11-29 Thread Mikhail Pryakhin
Hi all, I've just come across a FlinkKafkaProducer misconfiguration issue especially when a FlinkKafkaProducer is created without specifying a kafka partitioner then a FlinkFixedPartitioner instance is used, and all messages end up in a single kafka partition (in case I have a single task manage