||
You were right that it is an issue with the watermarks; outside of the when the job was stopped they were never emitted downstream, so no timer was ever triggered.

It appears that you need to set the setAutoWatermarkInterval in the ExecutionConfig via

env.getConfig().setAutoWatermarkInterval(Duration.ofMillis(500).toMillis());


to have them periodically emitted. Alternatively you could override BoundedOutOfOrdernessWatermarks#onEvent to also emit a watermark for event (for example, by calling #onPeriodicEmit).

Put another way, if you use any of the built-in WatermarkGenerators and use event-time, then it appears that you *must* set this interval.

This behavior is...less than ideal I must admit, and it does not appear to be properly documented.

On 1/27/2021 1:56 PM, Chesnay Schepler wrote:

Based on your description you aren't doing anything obviously wrong.

Would it be possible for you to share the code with us?

On 1/27/2021 1:02 PM, Pilgrim Beart wrote:
A newbie question:

I've created a basic Flink DataStream job for an IoT use-case, with file source and sink for testing. I key by device ID, then in a ProcessFunction set an EventTime Timer to fire if a device falls silent, i.e. a timeout, which I cancel if another message arrives from that device within the timeout.

My test source generates 3 devices, one of which falls silent for more than the timeout period during the stream, then resumes again. So I expect the Timer to fire for that device during the stream, and then for all the Timers to fire after the end of the stream.

The timers do indeed fire at the end of the stream (e.g. with a timeout of 1000, the timers all fire 1000 after the end of the stream, which is correct). But no timer fires for the device which falls silent during the stream (even though other devices are still talking, advancing event time). I've verified that I am keying correctly by ID.

I suspect this is something to do with Watermarks. I'm using forBoundedOutOfOrderness watermarking with a duration of 0.

All suggestions welcome, thanks.

-Pilgrim
--
Learn more at https://devicepilot.com <https://devicepilot.com> @devicepilot <https://t.sidekickopen70.com/s2t/c/5/f18dQhb0S7kv8cpgQZVc6VPt59hl3kW7_k2842PjkFxW2R1KhZ7v4vclW2Rxbb82bzNKzf7GYHvr01?te=W3R5hFj4cm2zwW4fQ47l4fGCmnW3Fbt5S3H4THtF3F6jFSWsSg1&si=5987503666495488&pi=6c5d2342-2e03-4926-a6bd-49f564169759>  +44 7961 125282 See our latest features <https://t.sidekickopen70.com/s2t/c/5/f18dQhb0S7kv8cpgQZVc6VPt59hl3kW7_k2842PjkFxW2R1KhZ7v4vclW2Rxbb82bzNKzf7GYHvr01?te=W3R5hFj4cm2zwW3Q_1QY1JxXxvW3SYLMH3T0vWRW1JxwY51LDhHjW3K1M0S1GFyF-3b732&si=5987503666495488&pi=6c5d2342-2e03-4926-a6bd-49f564169759> and book me <https://t.sidekickopen70.com/s2t/c/5/f18dQhb0S7kv8cpgQZVc6VPt59hl3kW7_k2842PjkFxW2R1KhZ7v4vclW2Rxbb82bzNKzf7GYHvr01?te=W3R5hFj4cm2zwW3T1k3k1JxXxvW3SYLMH3T0vWRW1JxwY51LBcj1W4fJfX_4cgB3QW3ZW3jf3_qrT_4Wt5h1&si=5987503666495488&pi=6c5d2342-2e03-4926-a6bd-49f564169759> for a video call.



Reply via email to