[
https://issues.apache.org/jira/browse/FLINK-4679?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15676958#comment-15676958
]
Fabian Hueske commented on FLINK-4679:
--------------------------------------
Hi [~jark], I think this approach could work for row windows which are
count-based. Event-time count-windows would in addition require to sort the
stream on event time before the window operation is applied.
For event-time time-window row-windows, we would need something more complex. I
was thinking of a custom operator that collects records in a priority queue
ordered by timestamp. Once a watermark is received for the upper bound of a
window (can be define with preceding and following time, but initially, we
should start with preceding only, IMO), the priority queue is used to evaluate
the window function and to purge too old records.
An event-time count-window row-window could use the same infrastructure (the
prio-queue would take care of the sorting) and use different logic to evaluate
windows and drop records (based on count rather than time).
For processing-time window, we could use a simple queue and evaluate every time
a new record is added. A processing count-window could be implemented similar.
So it might make sense to have everything implemented as custom operators, as
is seems that we could reuse some code parts.
What do you think [~jark], [~twalthr]?
> Add TumbleRow row-windows for streaming tables
> ----------------------------------------------
>
> Key: FLINK-4679
> URL: https://issues.apache.org/jira/browse/FLINK-4679
> Project: Flink
> Issue Type: Sub-task
> Components: Table API & SQL
> Affects Versions: 1.2.0
> Reporter: Fabian Hueske
> Assignee: Jark Wu
>
> Add TumbleRow row-windows for streaming tables as described in
> [FLIP-11|https://cwiki.apache.org/confluence/display/FLINK/FLIP-11%3A+Table+API+Stream+Aggregations].
>
> This task requires to implement a custom stream operator and integrate it
> with checkpointing and timestamp / watermark logic.
--
This message was sent by Atlassian JIRA
(v6.3.4#6332)