[ 
https://issues.apache.org/jira/browse/FLINK-22198?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17401669#comment-17401669
 ] 

Dong Lin commented on FLINK-22198:
----------------------------------


Here are my findings regarding why the timestamp of the produced record is more 
than 7 days earlier than the test time:

1) KafkaWriter will explicitly assign the timestamp to the produced record, 
rather than using the system current time for the produced request [1].
2) The timestamp could be from the StreamRecord that fed to the 
Input::processElement(..) method, which in turn is determined by the caller of 
those method (e.g. SQL related logic).
3) The code used to produce records in the 
testSourceSinkWithKeyAndPartialValue() explicitly assigns some timestamp with 
date such as 2020-03-08. This seems to be the only place where a 
non-current-system-time timestamp could be provided.

My hypothesis is this:
- The timestamp used in the test (e.g. 2020-03-08) has been used as the 
timestamp of the records sent to Kafka.
- The test will fail if and only if Kafka Server happens to trigger log 
retention logic (which is triggered at regularly time interval) after the 
records are produced but before the records have been consumed. This explains 
why the test is flaky.

I would suggest to involve the developer who knows the expected behavior of the 
SQL/Table API tested by this test to validate the hypothesis described above.

[~xtsong] [~renqs] Do you know who can help answer the SQL related questions?

[1] 
https://github.com/apache/flink/blob/6c9818323b41a84137c52822d2993df788dbc9bb/flink-connectors/flink-connector-kafka/src/main/java/org/apache/flink/connector/kafka/sink/KafkaWriter.java#L138


> KafkaTableITCase hang.
> ----------------------
>
>                 Key: FLINK-22198
>                 URL: https://issues.apache.org/jira/browse/FLINK-22198
>             Project: Flink
>          Issue Type: Bug
>          Components: Connectors / Kafka
>    Affects Versions: 1.14.0, 1.12.4
>            Reporter: Guowei Ma
>            Assignee: Qingsheng Ren
>            Priority: Blocker
>              Labels: pull-request-available, stale-blocker, test-stability
>             Fix For: 1.14.0
>
>
> https://dev.azure.com/apache-flink/apache-flink/_build/results?buildId=16287&view=logs&j=c5f0071e-1851-543e-9a45-9ac140befc32&t=1fb1a56f-e8b5-5a82-00a0-a2db7757b4f5&l=6625
> There is no any artifacts.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)

Reply via email to