Re: Kafka Streams - Join synchronization issue

2017-04-18 Thread Marco Abitabile
o you see any issue here?Thanks a lot. Marco 2017-04-18 13:14 GMT+02:00 Eno Thereska : > Hi Marco, > > I noticed your window is 1 second width, not 1 minute width. Is that > intentional? > > Thanks > Eno > > On 17 Apr 2017, at 19:41, Marco Abitabile > wrote: > >

Re: Kafka Streams - Join synchronization issue

2017-04-17 Thread Marco Abitabile
45 GMT+02:00 Eno Thereska : > Hi Marco, > > Could you share a bit of your code, or at a minimum provide some info on: > - is userActivitiesStream and geoDataStream a KStream of KTable? > - what is the length of "timewindow"? > > Thanks > Eno > > > On 16 A

Kafka Streams - Join synchronization issue

2017-04-16 Thread Marco Abitabile
Hi All! I need a little hint to understand how join works, in regards of stream synchronization. This mail is a bit long, I need to explain the issue I'm facing. *TL-TR: * it seems that join synchonization between stream is not respected as explained here: https://cwiki.apache.org/confluence/dis

Kafka Streams - Join synchronization issue

2017-04-16 Thread Marco Abitabile
Hi All! I need a little hint to understand how join works, in regards of stream synchronization. This mail is a bit long, I need to explain the issue I'm facing. *TL-TR: * it seems that join synchonization between stream is not respected as explained here: https://cwiki.apache.org/confluence/dis

[Kafka Streams] - problem joining 2 different streams

2017-03-27 Thread Marco Abitabile
Hi all, I'm struggling with an apparently simple problem. I'm joining 2 different streams: Stream1. User activity data, with key, value --> Stream2. User location data (such as the city name) with key, value --> Keys are homogeneous in content and represents the id of the user's device. The

Re: Strange behaviour in Session Windows

2017-03-07 Thread Marco Abitabile
ur call to aggregate, pass in the sessionStore created above, > i.e., > > aggregate( > MySession::new, > MySession::aggregateSessions, > MySession::mergeSessions, > SessionWindows > .with(WINDOW_INACTIVITY_GAPS_MS), > mySessionSerde, > sessionStore

Re: Strange behaviour in Session Windows

2017-03-06 Thread Marco Abitabile
s is completely driven by the data and specifically the time > extracted from the data. I'm not sure if you can provide a sample of the > data going through the system? It might be helpful in trying to debug the > issue. (I'm not seeing anything obvious in the code). >

Strange behaviour in Session Windows

2017-03-06 Thread Marco Abitabile
Hello, I'm playing around with the brand new SessionWindows. I have a simple topology such as: KStream sess = builder.stream(stringSerde, jsonSerde, SOURCE_TOPIC); sess .map(MySession::enhanceWithUserId_And_PutUserIdAsKey) .groupByKey(stringSerde, jsonSerde) .aggregate( MySes

[KIP-94] SessionWindows - IndexOutOfBoundsException in simple use case

2017-02-22 Thread Marco Abitabile
Hello, I apologies with Matthias since I posted yesterday this issue on the wrong place on github :( I'm trying a simple use case of session windowing. TimeWindows works perfectly, however as I replace with SessionWindows, this exception is thrown: Exception in thread "StreamThread-1" org.apache