Re: kafka 0.10.1 ProcessorTopologyTestDriver issue with .map and .groupByKey.count

2016-12-01 Thread Hamidreza Afzali
I have added an example for KStreamDriver to the GitHub Gist and updated the JIRA issue. https://issues.apache.org/jira/browse/KAFKA-4461 https://gist.github.com/hrafzali/c2f50e7b957030dab13693eec1e49c13 Hamid

Re: kafka 0.10.1 ProcessorTopologyTestDriver issue with .map and .groupByKey.count

2016-11-29 Thread Hamidreza Afzali
I have created a JIRA issue: https://issues.apache.org/jira/browse/KAFKA-4461 Hamid

Re: kafka 0.10.1 ProcessorTopologyTestDriver issue with .map and .groupByKey.count

2016-11-24 Thread Hamidreza Afzali
Hi Damian, It processes correctly when using KStreamTestDriver. Best, Hamid

Re: kafka 0.10.1 ProcessorTopologyTestDriver issue with .map and .groupByKey.count

2016-11-24 Thread Hamidreza Afzali
The map() returns non-null keys and values and produces the following stream: [KSTREAM-MAP-01]: A , 1 [KSTREAM-MAP-01]: A , 2 [KSTREAM-MAP-01]: B , 3 The issue arises when the combination of map() and groupByKey().count() is used with ProcessorTopologyTestDriver. I have

Re: kafka 0.10.1 ProcessorTopologyTestDriver issue with .map and .groupByKey.count

2016-11-23 Thread Hamidreza Afzali
Thanks Matthias. Disabling the cache didn't solve the issue. Here's a sample code: https://gist.github.com/hrafzali/c2f50e7b957030dab13693eec1e49c13 The topology doesn't produce any result but it works when commenting out .map(...) in line 21. Thanks, Hamid

kafka 0.10.1 ProcessorTopologyTestDriver issue with .map and .groupByKey.count

2016-11-22 Thread Hamidreza Afzali
Hi, When using ProcessorTopologyTestDriver in the latest Kafka 0.10.1, the combination of .map(...) and .groupByKey(...).count(...) does not produce any result. The topology looks like this: builder.stream(Serdes.String, Serdes.Integer, inputTopic) .map((k, v) => new KeyValue(fn(k), v)) .gro

Re: Kafka 0.10.1 ProcessorTopologyTestDriver and WindowedStreamPartitioner issue

2016-10-04 Thread Hamidreza Afzali
Thanks Guozhang. I can confirm the issue is resolved. Hamid

Re: Kafka 0.10.1 ProcessorTopologyTestDriver and WindowedStreamPartitioner issue

2016-10-03 Thread Hamidreza Afzali
Thanks Guozhang. We use ProcessorTopologyTestDriver for unit tests. Hamid > On 28 Sep 2016, at 11:48 AM, Hamidreza Afzali > wrote: > > Hi, > > We are using the latest Kafka 0.10.1 branch. The combination of > ProcessorTopologyTestDriver and WindowedStreamPartitio

Kafka 0.10.1 ProcessorTopologyTestDriver and WindowedStreamPartitioner issue

2016-09-28 Thread Hamidreza Afzali
Hi, We are using the latest Kafka 0.10.1 branch. The combination of ProcessorTopologyTestDriver and WindowedStreamPartitioner is resulting in a division by 0 exception because of the empty list of partitions: https://github.com/apache/kafka/blob/0.10.1/streams/src/test/java/org/apache/kafka/tes