hehuiyuan opened a new pull request #18995:
URL: https://github.com/apache/flink/pull/18995


   
   ## What is the purpose of the change
    if table.exec.emit.allow-lateness is set,  cleanup time is 
window.maxtimestap + table.exec.emit.allow-lateness,
   the window result is not emit when cleanup timer call onEventTime .
   
   
   For example:
   the sql of job :
   ```
   CREATE TABLE tableSource(
       name string,
       age int not null,
       sex string,
       dt TIMESTAMP(3),
       WATERMARK FOR dt AS dt - INTERVAL '0' SECOND
   ) WITH (
   
   );
   
   
   
   
   CREATE TABLE tableSink(
       windowstart timestamp(3),
       windowend timestamp(3),
       name string,
       age int,
       cou bigint
   )
   WITH (
   
   );
   INSERT INTO tablesink
     SELECT
       TUMBLE_START(dt, INTERVAL '1' HOUR),
       TUMBLE_END(dt, INTERVAL '1' HOUR),
       name,
       age,
       count(sex)
   FROM tableSource
   GROUP BY TUMBLE(dt, INTERVAL '1' HOUR), name,age 
   ```
   
   
   
   and table config:
   ```
   table.exec.emit.allow-lateness = 1 hour 
   table.exec.emit.late-fire.delay = 1 min
   table.exec.emit.early-fire.delay = 1min
   ```
   
   
   
   The data:
   ```
   >hehuiyuan1,22,woman,2022-03-05 00:30:22.000
   >hehuiyuan1,22,woman,2022-03-05 00:40:22.000
    //pause ,wait for the window trigger for earlyTrigger 1 min
   >hehuiyuan1,22,woman,2022-03-05 00:50:22.000
   >hehuiyuan1,22,woman,2022-03-05 00:56:22.000
   //pause ,wait for the window trigger for earlyTrigger 1 min 
   >hehuiyuan1,22,woman,2022-03-05 01:00:00.000
   //pause ,wait for the window trigger for earlyTrigger 1 min 
   >hehuiyuan1,22,woman,2022-03-05 00:59:20.000 --latency data
   //pause ,wait for the window trigger for earlyTrigger 1 min 
   >hehuiyuan1,22,woman,2022-03-05 00:59:20.100 --latency data 
   >hehuiyuan1,22,woman,2022-03-05 02:00:00.000 -- window state clean for 
[0:00:00 1:00:00]
   >hehuiyuan1,22,woman,2022-03-05 02:10:00.000 
   ```
   
   
   
   The result:
   ```
   > +I(+I[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 2])
   
   > -U(-U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 2]) 
   > +U(+U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 4]) 
   
   > +I(+I[2022-03-05T01:00, 2022-03-05T02:00, hehuiyuan1, 22, 1])
   
   > -U(-U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 4]) 
   > +U(+U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 5])
    
    
   > +I(+I[2022-03-05T02:00, 2022-03-05T03:00, hehuiyuan1, 22, 2]) 
   ```
   
   
   
   The window result  is lost when `hehuiyuan1,22,woman,2022-03-05 
00:59:20.100`  arrived, the lateTrigger is not trigger and the window[0:00:00 
,1:00:00] is cleaned when the data `hehuiyuan1,22,woman,2022-03-05 
02:00:00.000` arrived that updated watermark.
   
   
   
   
   The window[0:00:00 ,1:00:00]   has 6 pieces of data, but we got 5.
   
   The trigger is AfterEndOfWindowEarlyAndLate .
   
   So WindowOpearator may need to emit reuslt when the window cleanupTimer call 
onEventTime.
   
   
   
   
   I think the correct result is as follows:
   ```
   > +I(+I[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 2])
   
   > -U(-U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 2])
   > +U(+U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 4])
   
   > +I(+I[2022-03-05T01:00, 2022-03-05T02:00, hehuiyuan1, 22, 1])
   
   > -U(-U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 4])
   > +U(+U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 5])
   
   > -U(-U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 5])
   > +U(+U[2022-03-05T00:00, 2022-03-05T01:00, hehuiyuan1, 22, 6])
   > +I(+I[2022-03-05T02:00, 2022-03-05T03:00, hehuiyuan1, 22, 2]) 
   ```
   
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


Reply via email to