Skip to content

Conversation

@uncleGen
Copy link
Contributor

What changes were proposed in this pull request?

Structured Streaming progress reporter will always report an empty progress when there is no new data. As design, we should provide progress updates every 10s (default) when there is no new data.

Before PR:

20200428175008
20200428175037
20200428175102

After PR:

image

Why are the changes needed?

Fixes a bug around incorrect progress report

Does this PR introduce any user-facing change?

Fixes a bug around incorrect progress report

How was this patch tested?

existing ut and manual test

if (hasExecuted) {
// Reset noDataEventTimestamp if we processed any data
lastNoExecutionProgressEventTime = Long.MinValue
lastNoExecutionProgressEventTime = triggerClock.getTimeMillis()
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

When reset "lastNoExecutionProgressEventTime =Long.MinValue", it will make "now - noDataProgressEventInterval >= lastNoExecutionProgressEventTime" always true when there is no new data. Then progress reporter will report an empty progress.

@uncleGen
Copy link
Contributor Author

cc @xuanyuanking

@SparkQA
Copy link

SparkQA commented Apr 28, 2020

Test build #121989 has finished for PR 28391 at commit 4201180.

  • This patch fails Spark unit tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

@HeartSaVioR
Copy link
Contributor

This was actually proposed in #28040 but indicated UT failure hence abandoned. Happy to see it addressed, but looks like you'll also need to deal with UT failure. (UT failure is relevant.)

@xuanyuanking
Copy link
Member

I think the change here is reasonable since the condition changed from hasNewData to hasExecuted in SPARK-31278. But yes, as @HeartSaVioR mentioned the UT is relevant. Maybe we also need to add new UT in StreamingQueryListenerSuite.scala?

@uncleGen
Copy link
Contributor Author

@HeartSaVioR @xuanyuanking Thanks for reviewing, let me add ut.

@SparkQA
Copy link

SparkQA commented May 9, 2020

Test build #122460 has finished for PR 28391 at commit 4a55b25.

  • This patch fails Scala style tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented May 9, 2020

Test build #122461 has finished for PR 28391 at commit 914188c.

  • This patch fails due to an unknown error code, -9.
  • This patch merges cleanly.
  • This patch adds no public classes.

@xuanyuanking
Copy link
Member

retest this please

@SparkQA
Copy link

SparkQA commented May 9, 2020

Test build #122462 has finished for PR 28391 at commit 914188c.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented May 9, 2020

Test build #122464 has finished for PR 28391 at commit c3db6cf.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

Copy link
Member

@xuanyuanking xuanyuanking left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

LGTM, just some nits of comments.

noDataBatchEnableKey -> flag.toString,
// set `STREAMING_NO_DATA_PROGRESS_EVENT_INTERVAL` a small value to
// report an `empty` progress when no data come.
noDataProgressIntervalKey -> "1")
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

The way for fixing the failed UTs seems a little flaky. As your approach deletes empty progress generally, so here you set STREAMING_NO_DATA_PROGRESS_EVENT_INTERVAL to a small enough value to generate an empty one as expected, I think we can do better in either way:

  • Add comments to explain how small STREAMING_NO_DATA_PROGRESS_EVENT_INTERVAL needs to be set here is enough, seems less than 10 seconds is OK?
  • Delete the empty progress checking with explanation in comments.

Copy link
Contributor

@HeartSaVioR HeartSaVioR May 31, 2020

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Let's either use the manual clock or fix the UT like the way it doesn't depend on STREAMING_NO_DATA_PROGRESS_EVENT_INTERVAL.

For latter like we can set noDataProgressIntervalKey to 1000000 and remove the last assertion, which isn't actually testing the behavior of deduplication. Even better, you can still keep the last assertion for only when flag == true, which has a meaning of verification that state cleanup in empty input batch doesn't produce new outputs.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

SPARK-31928 is filed to address the flakiness of this test on the master branch - that said, if we can make sure this test become no longer flaky, that should be nice.

@xuanyuanking
Copy link
Member

cc @zsxwing @brkyvz @tdas

@HyukjinKwon
Copy link
Member

retest this please

@SparkQA
Copy link

SparkQA commented May 28, 2020

Test build #123227 has finished for PR 28391 at commit c3db6cf.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

StartStream(
Trigger.ProcessingTime(100),
triggerClock = clock,
Map(noDataProgressIntervalKey -> "100")),
Copy link
Contributor Author

@uncleGen uncleGen Jun 3, 2020

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

set "noDataProgressIntervalKey" to 100. After advance manual clock at batch 3 (with no data), it will report an empty progress event.

spark.streams.addListener(listener)

def checkProgressEvent(count: Int): StreamAction = {
AssertOnQuery { _ =>
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

add an inner function to simplify test.

@SparkQA
Copy link

SparkQA commented Jun 3, 2020

Test build #123471 has finished for PR 28391 at commit 8479341.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

Copy link
Contributor

@HeartSaVioR HeartSaVioR left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

LGTM

@uncleGen
Copy link
Contributor Author

uncleGen commented Jun 4, 2020

@HeartSaVioR @xuanyuanking Thanks for reviewing.

@HeartSaVioR
Copy link
Contributor

cc. @tdas @zsxwing @jose-torres

Copy link
Contributor

@gaborgsomogyi gaborgsomogyi left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

LGTM

@HyukjinKwon
Copy link
Member

I am going to merge this in few days if there's no more comments given that multiple LGTMs from the community. Let me know if there's any concern @tdas @zsxwing @jose-torres.

@HyukjinKwon
Copy link
Member

retest this please

@uncleGen
Copy link
Contributor Author

retest this please.

@SparkQA
Copy link

SparkQA commented Jun 12, 2020

Test build #123888 has finished for PR 28391 at commit 8479341.

  • This patch fails due to an unknown error code, -9.
  • This patch merges cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented Jun 12, 2020

Test build #123893 has finished for PR 28391 at commit 8479341.

  • This patch fails due to an unknown error code, -9.
  • This patch merges cleanly.
  • This patch adds no public classes.

@gaborgsomogyi
Copy link
Contributor

retest this please

@SparkQA
Copy link

SparkQA commented Jun 12, 2020

Test build #123901 has finished for PR 28391 at commit 8479341.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

@HyukjinKwon
Copy link
Member

Merged to master and branch-3.0.

HyukjinKwon pushed a commit that referenced this pull request Jun 14, 2020
### What changes were proposed in this pull request?

Structured Streaming progress reporter will always report an `empty` progress when there is no new data. As design, we should provide progress updates every 10s (default) when there is no new data.

Before PR:

![20200428175008](https://user-images.githubusercontent.com/7402327/80474832-88a8ca00-897a-11ea-820b-d4be6127d2fe.jpg)
![20200428175037](https://user-images.githubusercontent.com/7402327/80474844-8ba3ba80-897a-11ea-873c-b7137bd4a804.jpg)
![20200428175102](https://user-images.githubusercontent.com/7402327/80474848-8e061480-897a-11ea-806e-28c6bbf1fe03.jpg)

After PR:

![image](https://user-images.githubusercontent.com/7402327/80475099-f35a0580-897a-11ea-8fb3-53f343df2c3f.png)

### Why are the changes needed?

Fixes a bug around incorrect progress report

### Does this PR introduce any user-facing change?

Fixes a bug around incorrect progress report

### How was this patch tested?

existing ut and manual test

Closes #28391 from uncleGen/SPARK-31593.

Authored-by: uncleGen <[email protected]>
Signed-off-by: HyukjinKwon <[email protected]>
(cherry picked from commit 1e40bcc)
Signed-off-by: HyukjinKwon <[email protected]>
holdenk pushed a commit to holdenk/spark that referenced this pull request Jun 25, 2020
### What changes were proposed in this pull request?

Structured Streaming progress reporter will always report an `empty` progress when there is no new data. As design, we should provide progress updates every 10s (default) when there is no new data.

Before PR:

![20200428175008](https://user-images.githubusercontent.com/7402327/80474832-88a8ca00-897a-11ea-820b-d4be6127d2fe.jpg)
![20200428175037](https://user-images.githubusercontent.com/7402327/80474844-8ba3ba80-897a-11ea-873c-b7137bd4a804.jpg)
![20200428175102](https://user-images.githubusercontent.com/7402327/80474848-8e061480-897a-11ea-806e-28c6bbf1fe03.jpg)

After PR:

![image](https://user-images.githubusercontent.com/7402327/80475099-f35a0580-897a-11ea-8fb3-53f343df2c3f.png)

### Why are the changes needed?

Fixes a bug around incorrect progress report

### Does this PR introduce any user-facing change?

Fixes a bug around incorrect progress report

### How was this patch tested?

existing ut and manual test

Closes apache#28391 from uncleGen/SPARK-31593.

Authored-by: uncleGen <[email protected]>
Signed-off-by: HyukjinKwon <[email protected]>
(cherry picked from commit 1e40bcc)
Signed-off-by: HyukjinKwon <[email protected]>
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Projects

None yet

Development

Successfully merging this pull request may close these issues.

6 participants