Re: Partitioning behavior of Kafka Streams without explicit StreamPartitioner

2017-02-10 Thread Matthias J. Sax
What Mathieu say makes a lot of sense. Just some background info: A KStream is always created from a Kafka topic. > KStream stream = builder.stream("topicName"); A #through() is just syntactic sugar to stream.to(XXX) followed by builder.stream(XXX). So StreamPartitioner problem solved :) Abou

Re: Partitioning behavior of Kafka Streams without explicit StreamPartitioner

2017-02-10 Thread Mathieu Fenniak
Well, I think what you're doing is unusual for sure. The Streams API is really about transforming streams of data from input to output... so therefore the API doesn't have an injection point like you're looking for. I'd say it's intentional (I'm just a user though). If I were in your shoes, I'd p

Re: Partitioning behavior of Kafka Streams without explicit StreamPartitioner

2017-02-10 Thread Steven Schlansker
> On Feb 10, 2017, at 1:09 PM, Mathieu Fenniak > wrote: > > Hey Steven, > > If you have one KStream, and you want to produce to a topic that is read by > another KStream, you'd use the ".through" method of the first KStream. > ".through" both outputs to a topic and returns a KStream that reads

Re: Partitioning behavior of Kafka Streams without explicit StreamPartitioner

2017-02-10 Thread Mathieu Fenniak
Hey Steven, If you have one KStream, and you want to produce to a topic that is read by another KStream, you'd use the ".through" method of the first KStream. ".through" both outputs to a topic and returns a KStream that reads from that topic. (".to" just outputs to a topic) If you want to prod

Re: Partitioning behavior of Kafka Streams without explicit StreamPartitioner

2017-02-10 Thread Steven Schlansker
So then I guess my problem really is that I am operating at two different levels of abstraction. How do I produce to a KStream? I could imagine a method: public void KStream.put(K, V, Callback?); but I don't see anything like that. Nor do the "QueryableStoreTypes" really seem like what I want

Re: Partitioning behavior of Kafka Streams without explicit StreamPartitioner

2017-02-09 Thread Matthias J. Sax
It's by design. The reason it, that Streams uses a single producer to write to different output topic. As different output topics might have different key and/or value types, the producer is instantiated with byte[] as key and value type, and Streams serialized the data before handing it to the pr

Partitioning behavior of Kafka Streams without explicit StreamPartitioner

2017-02-09 Thread Steven Schlansker
Hi, I discovered what I consider to be really confusing behavior -- wondering if this is by design or a bug. The Kafka Partitioner interface: public int partition(String topic, Object key, byte[] keyBytes, Object value, byte[] valueBytes, Cluster cluster); has both "Object value" and "byte[] va