You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by zsxwing <gi...@git.apache.org> on 2015/10/01 17:00:30 UTC

[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

GitHub user zsxwing opened a pull request:

    https://github.com/apache/spark/pull/8958

    [SPARK-10900][Streaming]Add output operation events to StreamingListener

    Add output operation events to StreamingListener so as to implement the following UI features:
    
    1. Progress bar of a batch in the batch list.
    2. Be able to display output operation `description` and `duration` when there is no spark job in a Streaming job.

You can merge this pull request into a Git repository by running:

    $ git pull https://github.com/zsxwing/spark output-operation-events

Alternatively you can review and apply these changes as the patch at:

    https://github.com/apache/spark/pull/8958.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

    This closes #8958
    
----
commit 1ffae9302bc699bba750693ba3d08327e0b62f57
Author: zsxwing <zs...@gmail.com>
Date:   2015-10-01T14:42:58Z

    Add output operation events to StreamingListener

----


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145717661
  
    Merged build started.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145723281
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43266/
    Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by tdas <gi...@git.apache.org>.
Github user tdas commented on a diff in the pull request:

    https://github.com/apache/spark/pull/8958#discussion_r41075408
  
    --- Diff: streaming/src/main/scala/org/apache/spark/streaming/scheduler/StreamingListener.scala ---
    @@ -39,6 +39,17 @@ case class StreamingListenerBatchCompleted(batchInfo: BatchInfo) extends Streami
     case class StreamingListenerBatchStarted(batchInfo: BatchInfo) extends StreamingListenerEvent
     
     @DeveloperApi
    +case class StreamingListenerOutputOperationStarted(
    +    outputOperationId: Int,
    +    outputOperationCallSite: String,
    +    startTime: Long) extends StreamingListenerEvent
    +
    +@DeveloperApi
    +case class StreamingListenerOutputOperationCompleted(
    --- End diff --
    
    I think its better for future API compatibility to have another class called `OutputOperationInfo` (similar to `BatchInfo`). We can add more field in `OutputOperationInfo` later, without breaking binary compatibility.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-144756156
  
      [Test build #43151 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43151/consoleFull) for   PR 8958 at commit [`ba8f9b8`](https://github.com/apache/spark/commit/ba8f9b8a9aa53a88a87592cc833180ee4aaf6ee8).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by zsxwing <gi...@git.apache.org>.
Github user zsxwing commented on a diff in the pull request:

    https://github.com/apache/spark/pull/8958#discussion_r41128455
  
    --- Diff: streaming/src/test/scala/org/apache/spark/streaming/StreamingListenerSuite.scala ---
    @@ -140,6 +140,27 @@ class StreamingListenerSuite extends TestSuiteBase with Matchers {
         }
       }
     
    +  test("output operation reporting") {
    +    ssc = new StreamingContext("local[2]", "test", Milliseconds(1000))
    +    val inputStream = ssc.receiverStream(new StreamingListenerSuiteReceiver)
    +    inputStream.foreachRDD(_.count())
    +    inputStream.foreachRDD(_.collect())
    +    inputStream.foreachRDD(_.count())
    +
    +    val collector = new OutputOperationInfoCollector
    +    ssc.addStreamingListener(collector)
    +
    +    ssc.start()
    +    try {
    +      eventually(timeout(30 seconds), interval(20 millis)) {
    --- End diff --
    
    `batch info reporting` only has an output operation. So I created this new test so as to test multiple output operations.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-144771352
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43151/
    Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-144754459
  
     Merged build triggered.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145488113
  
     Merged build triggered.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-144771351
  
    Merged build finished. Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145717651
  
     Merged build triggered.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145497725
  
      [Test build #43237 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43237/console) for   PR 8958 at commit [`0fc508c`](https://github.com/apache/spark/commit/0fc508cb3cc4981d7d164a1021a43eca2fc71a23).
     * This patch **passes all tests**.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `case class OutputOperationInfo(`
      * `case class StreamingListenerOutputOperationStarted(outputOperationInfo: OutputOperationInfo)`
      * `case class StreamingListenerOutputOperationCompleted(outputOperationInfo: OutputOperationInfo)`



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by zsxwing <gi...@git.apache.org>.
Github user zsxwing commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-144754444
  
    /cc @tdas


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by asfgit <gi...@git.apache.org>.
Github user asfgit closed the pull request at:

    https://github.com/apache/spark/pull/8958


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145717879
  
      [Test build #43266 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43266/consoleFull) for   PR 8958 at commit [`320fb33`](https://github.com/apache/spark/commit/320fb3396a050655f643c0e4b6019d73248691ea).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145488143
  
    Merged build started.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-144754489
  
    Merged build started.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145723279
  
    Merged build finished. Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145497821
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43237/
    Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145497818
  
    Merged build finished. Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by tdas <gi...@git.apache.org>.
Github user tdas commented on a diff in the pull request:

    https://github.com/apache/spark/pull/8958#discussion_r41178301
  
    --- Diff: streaming/src/test/scala/org/apache/spark/streaming/StreamingListenerSuite.scala ---
    @@ -140,6 +140,27 @@ class StreamingListenerSuite extends TestSuiteBase with Matchers {
         }
       }
     
    +  test("output operation reporting") {
    +    ssc = new StreamingContext("local[2]", "test", Milliseconds(1000))
    +    val inputStream = ssc.receiverStream(new StreamingListenerSuiteReceiver)
    +    inputStream.foreachRDD(_.count())
    +    inputStream.foreachRDD(_.collect())
    +    inputStream.foreachRDD(_.count())
    +
    +    val collector = new OutputOperationInfoCollector
    +    ssc.addStreamingListener(collector)
    +
    +    ssc.start()
    +    try {
    +      eventually(timeout(30 seconds), interval(20 millis)) {
    --- End diff --
    
    makes sense especially now that there is a output operation info


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145723206
  
      [Test build #43266 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43266/console) for   PR 8958 at commit [`320fb33`](https://github.com/apache/spark/commit/320fb3396a050655f643c0e4b6019d73248691ea).
     * This patch **passes all tests**.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `case class HyperLogLogPlusPlus(child: Expression, relativeSD: Double = 0.05)`
      * `case class JsonTuple(children: Seq[Expression])`
      * `case class OutputOperationInfo(`
      * `case class StreamingListenerOutputOperationStarted(outputOperationInfo: OutputOperationInfo)`
      * `case class StreamingListenerOutputOperationCompleted(outputOperationInfo: OutputOperationInfo)`



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by tdas <gi...@git.apache.org>.
Github user tdas commented on a diff in the pull request:

    https://github.com/apache/spark/pull/8958#discussion_r41178108
  
    --- Diff: streaming/src/main/scala/org/apache/spark/streaming/scheduler/OutputOperationInfo.scala ---
    @@ -0,0 +1,41 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.streaming.scheduler
    +
    +import org.apache.spark.annotation.DeveloperApi
    +
    +/**
    + * :: DeveloperApi ::
    + * Class having information on output operations.
    + * @param id Id of this output operation. Different output operations have different ids in a batch.
    + * @param description the description of this output operation.
    + * @param startTime Clock time of when the output operation started processing
    + * @param endTime Clock time of when the output operation started processing
    + */
    +@DeveloperApi
    +case class OutputOperationInfo(
    +    id: Int,
    --- End diff --
    
    This probably should have some reference to which batch this output operation belongs to. Batch time probably.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by tdas <gi...@git.apache.org>.
Github user tdas commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145719296
  
    LGTM. Will merge to master when tests pass.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by tdas <gi...@git.apache.org>.
Github user tdas commented on a diff in the pull request:

    https://github.com/apache/spark/pull/8958#discussion_r41067198
  
    --- Diff: streaming/src/test/scala/org/apache/spark/streaming/StreamingListenerSuite.scala ---
    @@ -140,6 +140,27 @@ class StreamingListenerSuite extends TestSuiteBase with Matchers {
         }
       }
     
    +  test("output operation reporting") {
    +    ssc = new StreamingContext("local[2]", "test", Milliseconds(1000))
    +    val inputStream = ssc.receiverStream(new StreamingListenerSuiteReceiver)
    +    inputStream.foreachRDD(_.count())
    +    inputStream.foreachRDD(_.collect())
    +    inputStream.foreachRDD(_.count())
    +
    +    val collector = new OutputOperationInfoCollector
    +    ssc.addStreamingListener(collector)
    +
    +    ssc.start()
    +    try {
    +      eventually(timeout(30 seconds), interval(20 millis)) {
    --- End diff --
    
    Cant this test be merged with the "batch info reporting" test? Looking back it would be weird -- why is the testing of these 2 method done separately from the other batch-level events.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-144771166
  
      [Test build #43151 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43151/console) for   PR 8958 at commit [`ba8f9b8`](https://github.com/apache/spark/commit/ba8f9b8a9aa53a88a87592cc833180ee4aaf6ee8).
     * This patch **passes all tests**.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `case class StreamingListenerOutputOperationStarted(`
      * `case class StreamingListenerOutputOperationCompleted(`



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] spark pull request: [SPARK-10900][Streaming]Add output operation e...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/8958#issuecomment-145488634
  
      [Test build #43237 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/43237/consoleFull) for   PR 8958 at commit [`0fc508c`](https://github.com/apache/spark/commit/0fc508cb3cc4981d7d164a1021a43eca2fc71a23).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org