Re: Understanding org.apache.kafka.streams.errors.TopologyBuilderException

2016-10-11 Thread Guozhang Wang
Sachin, Just a side note in addition to what Matthias mentioned: in the coming 0.10.1.0 release Kafka Streams has added the feature to do auto-repartitioning by detecting if the message key are joinable or not. To give a few examples: --- stream1 = builder.stream("topic1"); stream2 = builder

Re: Understanding org.apache.kafka.streams.errors.TopologyBuilderException

2016-10-09 Thread Sachin Mittal
Thanks for pointing this out. I am doing exactly like this now and it is working fine. Sachin On Sun, Oct 9, 2016 at 12:32 PM, Matthias J. Sax wrote: > You must ensure, that both streams are co-partitioned (ie, same number > of partitions and using the join key). > > (see "Note" box: > http://

Re: Understanding org.apache.kafka.streams.errors.TopologyBuilderException

2016-10-09 Thread Matthias J. Sax
You must ensure, that both streams are co-partitioned (ie, same number of partitions and using the join key). (see "Note" box: http://docs.confluent.io/3.0.1/streams/developer-guide.html#joining-streams) You can enforce co-partitioning by introducing a call to .through() before doing the join (on

Re: Understanding org.apache.kafka.streams.errors.TopologyBuilderException

2016-10-08 Thread Sachin Mittal
I don't think that is the issue. The join api says: public KTable join(KTable other, ValueJoiner joiner) In my case V is Map> V1 is List R is Map> K is String Note the final V and V1 are arrived after doing transformation on original streams So there are intermediate steps like stream.map(new K

RE: Understanding org.apache.kafka.streams.errors.TopologyBuilderException

2016-10-08 Thread Martin Gainty
> From: sjmit...@gmail.com > Date: Sat, 8 Oct 2016 15:30:33 +0530 > Subject: Understanding > org.apache.kafka.streams.errors.TopologyBuilderException > To: users@kafka.apache.org

Understanding org.apache.kafka.streams.errors.TopologyBuilderException

2016-10-08 Thread Sachin Mittal
Hi, I am getting this exception Exception in thread "main" org.apache.kafka.streams.errors.TopologyBuilderException: Invalid topology building: KTABLE-MAPVALUES-07 and KSTREAM-AGGREGATE-09 are not joinable What I am trying to do is I aggregate a KStream into a KTable of type KTabl