Can you please check what the input watermark of your operations is? There is a metric called "currentLowWatermark" for this.
Best, Aljoscha > On 3. Jan 2018, at 15:54, shashank agarwal <shashank...@gmail.com> wrote: > > Actually, In Kafka there are other topics also (around 5-6 topics) I am > consuming particular topic 'x' which only contains events. Other topics have > different data. > > I am using two consumers in my program for 2 different topics. in first topic > x i am extracting the timestamp from origintimestamp variable in other one i > am using system current millis. > > > > > > > On Wed, Jan 3, 2018 at 8:06 PM, Aljoscha Krettek <aljos...@apache.org > <mailto:aljos...@apache.org>> wrote: > Ok, but will there be events in all Kafka partitions/topics? > > >> On 3. Jan 2018, at 15:33, shashank agarwal <shashank...@gmail.com >> <mailto:shashank...@gmail.com>> wrote: >> >> Hi, >> >> Yes, Events will always carry a variable OriginTimestamp which I am using in >> the extractor. I have used fallback also in case of data missing will put >> System current millis. >> >> Still, it's not printing results. >> >> Best, >> Shashank >> >> >> >> >> >> >> >> On Wed, Jan 3, 2018 at 7:40 PM, Aljoscha Krettek <aljos...@apache.org >> <mailto:aljos...@apache.org>> wrote: >> Hi, >> >> Are all the partitions always carrying data that has advancing timestamps? >> When using Event-time the Kafka source (and Flink in general) needs to have >> steady progress in all partitions, otherwise the watermark does not advance, >> which in turn means that processing will be stalled downstream. >> >> Best, >> Aljoscha >> >> >>> On 3. Jan 2018, at 14:29, shashank agarwal <shashank...@gmail.com >>> <mailto:shashank...@gmail.com>> wrote: >>> >>> Hello, >>> >>> I have some patterns in my program. For an example, >>> >>> A followedBy B. >>> >>> As I am using kafka source and my event API's using load balancers so >>> sometimes B comes before A. So my CEP doesn't generate any result for those >>> events. >>> >>> I have then tried event time and applied >>> "BoundedOutOfOrdernessTimestampExtractor" on kafkasource with extract time >>> from an origin time variable which I have in the event. I am using >>> watermark lateness of 10 seconds in that. >>> >>> Now CEP stopped generating results. It's not even generating results where >>> Event B comes after A. I have tried within (10 seconds) in CEP also still >>> not generating results. >>> >>> Am I doing anything wrong? >>> >>> I have to cover the case where B can come after A from Kafka. >>> >>> -- >>> Thanks Regards >>> >>> SHASHANK AGARWAL >>> --- Trying to mobilize the things.... >>> >>> >>> >>> >>> >> >> >> >> >> -- >> Thanks Regards >> >> SHASHANK AGARWAL >> --- Trying to mobilize the things.... > > > > > -- > Thanks Regards > > SHASHANK AGARWAL > --- Trying to mobilize the things.... >