Skip to content

Conversation

@zsxwing
Copy link
Member

@zsxwing zsxwing commented Apr 8, 2015

This PR includes:

  1. Send StreamingListenerBatchSubmitted when JobSet is submitted
  2. Fix StreamingListenerBatchStarted.batchInfo.processingStartTime
  3. Fix a type: completedaBatchInfos -> completedBatchInfos

…reamingListenerBatchStarted.batchInfo.processingStartTime; fix a typo
@zsxwing
Copy link
Member Author

zsxwing commented Apr 8, 2015

cc @tdas

@SparkQA
Copy link

SparkQA commented Apr 8, 2015

Test build #29849 has started for PR 5414 at commit 493f978.

@jerryshao
Copy link
Contributor

Good catch, LGTM :).

@SparkQA
Copy link

SparkQA commented Apr 8, 2015

Test build #29849 has finished for PR 5414 at commit 493f978.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.
  • This patch does not change any dependencies.

@AmplabJenkins
Copy link

Test PASSed.
Refer to this link for build results (access rights to CI server needed):
https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/29849/
Test PASSed.

Copy link
Contributor

Choose a reason for hiding this comment

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

Good catch. OMG!

@tdas
Copy link
Contributor

tdas commented Apr 9, 2015

This looks good to me at first glance, but I am concerned that such a thing went unnoticed for so long despite unit tests in StreamingListenerSuite. Let me think whether we can improve the tests. If so, we definitely should add tests as part of this PR so that such a thing cannot happen again.

Copy link
Contributor

Choose a reason for hiding this comment

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

I think this can be refactored to dedup lines 139 and 144.

Copy link
Contributor

Choose a reason for hiding this comment

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

I think it is hard to dedup cause of ordering call.

Copy link
Member Author

Choose a reason for hiding this comment

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

Right, handleJobStart will set jobSet.hasStarted to true.

Copy link
Contributor

Choose a reason for hiding this comment

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

How about

val isFirstJobOfJobSet = !jobSet.hasStarted
jobSet.handleJobStart(job)
if (isFirstJobOfJobSet) {
    listenerBus.post(StreamingListenerBatchStarted(jobSet.toBatchInfo))
}

Copy link
Member Author

Choose a reason for hiding this comment

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

Good idea. Updated.

@tdas
Copy link
Contributor

tdas commented Apr 9, 2015

Overall, I realized that there is not tests for StreamingJobProgressListener, which is my bad. :( We really need to add some tests for the StreamingJobProgressListener which tests some basic aspects like this. Could you add a StreamingJobProgressListenerSuite ?

@zsxwing
Copy link
Member Author

zsxwing commented Apr 9, 2015

Could you add a StreamingJobProgressListenerSuite ?

OK. I will add some basic tests for StreamingJobProgressListener

@zsxwing
Copy link
Member Author

zsxwing commented Apr 9, 2015

Added some tests to StreamingListenerSuite for this issue, and added StreamingJobProgressListenerSuite for some basic tests of StreamingJobProgressListener.

@SparkQA
Copy link

SparkQA commented Apr 9, 2015

Test build #29935 has started for PR 5414 at commit 79b4fed.

Copy link
Contributor

Choose a reason for hiding this comment

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

How long does these two units tests take? I wonder whether it is best (in the interest of minimizing test run times) to collapse all these two unit tests into the earlier "batch info reporting".

Copy link
Member Author

Choose a reason for hiding this comment

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

[info] StreamingListenerSuite:
[info] - batch info reporting (4 seconds, 976 milliseconds)
[info] - receiver info reporting (847 milliseconds)
[info] - SPARK-6766: batch info should be submitted (667 milliseconds)
[info] - SPARK-6766: processingStartTime of batch info should not be None when starting (691 milliseconds)
[info] ScalaTest
[info] Run completed in 8 seconds, 996 milliseconds.
[info] Total number of tests run: 4
[info] Suites: completed 1, aborted 0
[info] Tests: succeeded 4, failed 0, canceled 0, ignored 0, pending 0
[info] All tests passed.

Copy link
Member Author

Choose a reason for hiding this comment

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

This was my last test result.

Copy link
Contributor

Choose a reason for hiding this comment

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

Hmm, each of the two new unit tests is taking .7 seconds. Not good. Best to merge them into the first unit test.
The reason its taking time is that its starting a new SparkContext every time, and running batches after that.

@SparkQA
Copy link

SparkQA commented Apr 9, 2015

Test build #29935 timed out for PR 5414 at commit 79b4fed after a configured wait of 120m.

@AmplabJenkins
Copy link

Test FAILed.
Refer to this link for build results (access rights to CI server needed):
https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/29935/
Test FAILed.

@zsxwing
Copy link
Member Author

zsxwing commented Apr 9, 2015

retest this please

@SparkQA
Copy link

SparkQA commented Apr 9, 2015

Test build #29937 has started for PR 5414 at commit 79b4fed.

@SparkQA
Copy link

SparkQA commented Apr 9, 2015

Test build #29937 has finished for PR 5414 at commit 79b4fed.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.
  • This patch does not change any dependencies.

@AmplabJenkins
Copy link

Test PASSed.
Refer to this link for build results (access rights to CI server needed):
https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/29937/
Test PASSed.

@SparkQA
Copy link

SparkQA commented Apr 9, 2015

Test build #29985 has started for PR 5414 at commit ca0955b.

@zsxwing
Copy link
Member Author

zsxwing commented Apr 9, 2015

Collapsed the new unit tests together.

@SparkQA
Copy link

SparkQA commented Apr 10, 2015

Test build #29985 has finished for PR 5414 at commit ca0955b.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.
  • This patch does not change any dependencies.

@AmplabJenkins
Copy link

Test PASSed.
Refer to this link for build results (access rights to CI server needed):
https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/29985/
Test PASSed.

Copy link
Contributor

Choose a reason for hiding this comment

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

Why this line? Run streams should have already stopped the ssc, isnt it?

Copy link
Member Author

Choose a reason for hiding this comment

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

Sorry that I misread runStreams. This line is not necessary.

@tdas
Copy link
Contributor

tdas commented Apr 10, 2015

Just a few minor comments, otherwise LGTM. Can merge as soon as you update these.

@SparkQA
Copy link

SparkQA commented Apr 10, 2015

Test build #30019 has started for PR 5414 at commit 2f85060.

@tdas
Copy link
Contributor

tdas commented Apr 10, 2015

The concerned testsuites have passed, I am merging this to master and 1.3 branch. Thanks!

@asfgit asfgit closed this in 18ca089 Apr 10, 2015
@tdas
Copy link
Contributor

tdas commented Apr 10, 2015

I merged it in master, but could not merge it to 1.3 because of conflicts. Could you open another PR to branch 1.3? Would be good to have these bug fixes in 1.3.x

@SparkQA
Copy link

SparkQA commented Apr 10, 2015

Test build #30019 has finished for PR 5414 at commit 2f85060.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.
  • This patch does not change any dependencies.

@AmplabJenkins
Copy link

Test PASSed.
Refer to this link for build results (access rights to CI server needed):
https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/30019/
Test PASSed.

@zsxwing zsxwing deleted the SPARK-6766 branch April 10, 2015 09:28
asfgit pushed a commit that referenced this pull request Apr 14, 2015
…ed and StreamingListenerBatchStarted (backport to branch 1.3)

Backport SPARK-6766 #5414 to branch 1.3

Conflicts:

	streaming/src/main/scala/org/apache/spark/streaming/ui/StreamingJobProgressListener.scala

Author: zsxwing <zsxwing@gmail.com>

Closes #5452 from zsxwing/SPARK-6766-branch-1.3 and squashes the following commits:

cb87e44 [zsxwing] [SPARK-6766][Streaming] Fix issue about StreamingListenerBatchSubmitted and StreamingListenerBatchStarted (backport to branch 1.3)
asfgit pushed a commit that referenced this pull request Apr 14, 2015
…tches" lists to StreamingPage

This PR adds two lists, `Active Batches` and `Completed Batches`. Here is the screenshot:

![batch_list](https://cloud.githubusercontent.com/assets/1000778/7060458/d8898572-deb3-11e4-938b-6f8602c71a9f.png)

Due to [SPARK-6766](https://issues.apache.org/jira/browse/SPARK-6766), I need to merge #5414 in my local machine to get the above screenshot.

Author: zsxwing <zsxwing@gmail.com>

Closes #5434 from zsxwing/SPARK-6796 and squashes the following commits:

be50fc6 [zsxwing] Fix the code style
51b792e [zsxwing] Fix the unit test
6f3078e [zsxwing] Make 'startTime' readable
f40e0a9 [zsxwing] Merge branch 'master' into SPARK-6796
2525336 [zsxwing] Rename 'Processed batches' and 'Waiting batches' and also add links
a69c091 [zsxwing] Show the number of total completed batches too
a12ad7b [zsxwing] Change 'records' to 'events' in the UI
86b5e7f [zsxwing] Make BatchTableBase abstract
b248787 [zsxwing] Add tests to verify the new tables
d18ab7d [zsxwing] Fix the code style
6ceffb3 [zsxwing] Add "Active Batches" and "Completed Batches" lists to StreamingPage
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

None yet

Projects

None yet

Development

Successfully merging this pull request may close these issues.

5 participants