Re: No output when using event time with multiple Kafka partitions

2016-08-15 Thread Yassine Marzougui
Hi Aljoscha, Sorry for the late response, I was busy and couldn't make time to work on this again again until now. Indeed, it turns out only one of the partitions is not receiving elements. The reason is that the producer will stick to a partition for topic.metadata.refresh.interval.ms (defaults t

Re: No output when using event time with multiple Kafka partitions

2016-07-29 Thread Aljoscha Krettek
Hi, when running in local mode the default parallelism is always the number of (possibly virtual) CPU cores. The parallelism of the sink is set before it gets a chance to find out how many Kafka partitions there are. I think the reason for the behavior you're observing is that only one of your two

Re: No output when using event time with multiple Kafka partitions

2016-07-27 Thread Yassin Marzouki
I just tried playing with the source paralleism setting, and I got a very strange result: If specify the source parallism using env.addSource(kafka).setParallelism(N), results are printed correctly for any number N except for N=4. I guess that's related to the number of task slots since I have a 4

Re: No output when using event time with multiple Kafka partitions

2016-07-27 Thread Yassin Marzouki
Hi Kostas, When I remove the window and the apply() and put print() after assignTimestampsAndWatermarks, the messages are printed correctly: 2> Request{ts=2015-01-01, 06:15:34:000} 2> Request{ts=2015-01-02, 16:38:10:000} 2> Request{ts=2015-01-02, 18:58:41:000} 2> Request{ts=2015-01-02, 19:10:00:0

Re: No output when using event time with multiple Kafka partitions

2016-07-27 Thread Kostas Kloudas
Hi Yassine, Could you just remove the window and the apply, and just put a print() after the: > .assignTimestampsAndWatermarks(new AscendingTimestampExtractor() { > @Override > public long extractAscendingTimestamp(Request req) { > return req.ts; > } > }) This at least will

No output when using event time with multiple Kafka partitions

2016-07-25 Thread Yassin Marzouki
Hi everyone, I am reading messages from a Kafka topic with 2 partitions and using event time. This is my code: .assignTimestampsAndWatermarks(new AscendingTimestampExtractor() { @Override public long extractAscendingTimestamp(Request req) { return req.ts; } }) .windowAll(Tumbl