Re: Aggregation using event timestamp than clock window

2018-01-28 Thread Rohan Thimmappa
Hi Gary, Thanks.I do have some of the events coming in after one pauses and i am able to see watermarked being advanced event being triggered. Rohan On Mon, Jan 15, 2018 at 5:40 AM, Gary Yao wrote: > Hi Rohan, > > In your example, are you saying that after 5:40 you will not receive any > ev

Re: Aggregation using event timestamp than clock window

2018-01-15 Thread Gary Yao
Hi Rohan, In your example, are you saying that after 5:40 you will not receive any events at all which could advance the watermark? I am asking because if you are receiving events for other keys/ids from your KafkaSource after 5:40, the watermark will still be advanced and fire the tumbling windo

Re: Aggregation using event timestamp than clock window

2018-01-15 Thread Rohan Thimmappa
No. My question is slightly different. say i get report from 5.10-5.40. the device went offline and never comes back. i will not get any report after 5.40. So 5-6 window never gets closed as we will not get any report after 5.40. in this case 5.00-5.40 data is still in flink memory that will never

Re: Aggregation using event timestamp than clock window

2018-01-14 Thread Gary Yao
Hi Rohan, I am not sure if I fully understand your problem. For example, if you receive an event with a start time of 4:50 and an end time of 5:30, do you want the "usage" from 4:50 - 5:00 to be included in the 4:00 - 5:00 window? What if the event had an end time of 5:31? Do you then want to igno

Re: Aggregation using event timestamp than clock window

2018-01-12 Thread Rohan Thimmappa
Hi Gary, This is perfect. I am able to get the window working on message timestamp then clock window also stream the data that are late. I also having one edge case. for eg i get my last report at 4.57 and i never get 5.00+ hour report *ever*. i would like to wait for sometime. My reporting int

Re: Aggregation using event timestamp than clock window

2018-01-12 Thread Gary Yao
Hi Rohan, Your ReportTimestampExtractor assigns timestamps to the stream records correctly but uses the wall clock to emit Watermarks (System.currentTimeMillis). In Flink Watermarks are the mechanism to advance the event time. Hence, you should emit Watermarks according to the time that you extrac

Aggregation using event timestamp than clock window

2018-01-11 Thread Rohan Thimmappa
Hi All, I have following requirement 1. i have avro json message containing {eventid, usage, starttime, endtime} 2. i am reading this from kafka source 3. if there is overlapping hour in a record split the record by rounding off to hourly bounderies 4.My objective is a) read the message b) aggr