You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by andrewor14 <gi...@git.apache.org> on 2016/01/12 04:09:38 UTC

[GitHub] spark pull request: [SPARK-10620][WIP] Migrate TaskMetrics to accu...

GitHub user andrewor14 opened a pull request:

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

    [SPARK-10620][WIP] Migrate TaskMetrics to accumulators

    There exist two mechanisms to pass metrics from executors to drivers: accumulators and `TaskMetrics`. Currently we send both things to the driver using two separate code paths. This is an unnecessary maintenance burden and makes the code more difficult to follow.
    
    This patch proposes that we send only accumulators to the driver. Additionally, it reimplements `TaskMetrics` using accumulators such that the new `TaskMetrics` serves mainly as a syntactic sugar to increment and access the values of the underlying accumulators. It migrates the rest of the metrics to adopt the code path already used by the existing `PEAK_EXECUTION_MEMORY`.
    
    While an effort has been made to preserve as much of the public API as possible, there were a few known breaking `@DeveloperApi` changes that would be very awkward to maintain. These are:
    - `TaskMetrics#hostname` field was removed; there were no consumers except the event log
    - `ExceptionFailure#taskMetrics` field was replaced with `accumUpdates`
    - `SparkListenerExecutorMetricsUpdate#taskMetrics` field was replaced with `accumUpdates`
    
    This is WIP because I would like to add tests for some of the intricate cases that I ran into while implementing this.

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

    $ git pull https://github.com/andrewor14/spark task-metrics-to-accums

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

    https://github.com/apache/spark/pull/10717.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 #10717
    
----
commit 8d5765751c500de5fb699100d5b10eebbcab4e5e
Author: Andrew Or <an...@databricks.com>
Date:   2015-12-30T02:27:27Z

    Remove unused methods or replace them
    
    There are a bunch of decrement X methods that were not used.
    Also, there are a few set X methods that could have easily just
    been increment X. The latter change is more in line with
    accumulators.

commit 1ad286813b758fda992c3d85406a7a628c39d1f7
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-04T21:34:42Z

    Implement initial framework to migrate metrics to accums
    
    This commit uses the existing PEAK_EXECUTION_MEMORY mechanism
    to bring a few other fields in TaskMetrics to use accumulators.

commit a4ca6b2de82dac5f255cd40ace83c77b508ac43d
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-04T21:54:05Z

    Migrate a few more easy metrics

commit 373898e7bf7f887fc008638625072ed1b68d8359
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-04T23:39:37Z

    ShuffleReadMetrics + namespacing accumulators
    
    This commit ports ShuffleReadMetrics to use accumulators,
    preserving as much of the existing semantics as possible.
    It also introduces a nicer way to organize all the internal
    accumulators by namespacing them.

commit e74632c3e0ed6b20f52f112a0cfdad27e036a827
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-05T21:01:08Z

    General code cleanup

commit 7e74bf38ac13e6eb148b15dd95e4241de1f7c118
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-05T21:47:24Z

    ShuffleWriteMetrics

commit 396088d1fff77dd17d280ba3297c4b18d142b99f
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-05T22:17:51Z

    OutputMetrics

commit 0404e3e8f291f6cb8b007fb09d252ee2bb513308
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-05T23:52:31Z

    InputMetrics
    
    This commit was a little tricky because it ripped the bytes read
    callback from TaskMetrics and related classes. It does change
    behavior in the sense that now we periodically update the number
    of bytes read (every 1000 records) instead of doing it every time
    we send an executor heartbeat. The advantage here is code
    simplicity.

commit 17becb05aa347586fe07fd452e288479c7649e82
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T00:06:31Z

    Fix JsonProtocol + JsonProtocolSuite

commit 809a93ac3b4961e35bfb180fa1e46bc36296153c
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T00:25:22Z

    Fix tests where TaskMetrics had no accumulators
    
    Tests are still failing as of this commit. E.g. SortShuffleSuite.

commit 32ba9e3b78f679261e36b4ea095733a14b30fd7b
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T00:29:18Z

    Rename a few shuffle write metrics for consistency

commit 78fb33e3b7714c3ffff932434000780d17835c24
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T02:06:10Z

    Fix metrics in local mode (tests)
    
    Tests were previously failing because we end up double counting
    metrics in local mode. This is because each TaskContext shares
    the same list of accumulators, so they end up updating the metrics
    on top of each other. The fix is to ensure TaskContext clears any
    existing values on the accumulators before passing them on.

commit 8117898d585d62e536c7c87f56bc545118cc97b1
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T02:42:25Z

    Fix "harmless" exception in peak memory tests
    
    The exception was harmless because it didn't actually fail the
    test. However, the test harness was actually badly written. We
    used to always assume that the first job will have an ID of 0,
    but there could very well be other tests sharing the same
    SparkContext. This is now fixed and we no longer see the exception.
    
    As of this commit, all known test failures have been fixed.
    I'm sure there will be more...

commit 6bd9c0a1caaac38be75f4c8269225a53c75d6509
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T19:13:07Z

    Simplify internal accumulator update mechanism
    
    Instead of passing in a callback, we can just return the
    accumulator values directly, which we have. "We" here refers to
    TaskMetrics.

commit ed293282118c414e6d5c405a9c2222616ce9d3bd
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T21:43:24Z

    Fix tests

commit 20119126a26420b30c4bba417d2c04aaffac305a
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T23:19:56Z

    Clean up
    
    This commit addresses outstanding TODO's and makes the deprecated
    APIs DeveloperApi instead. This allows us to deal with how to
    do the deprecation properly later. This commit also reverts a few
    unnecessary changes to reduce the size of the diff.

commit c3de4f0c8e5dcfbfe4dbddad293c4dc2a8fddaf4
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-06T23:33:40Z

    Merge branch 'master' of github.com:apache/spark into task-metrics-to-accums
    
    Conflicts:
    	core/src/main/scala/org/apache/spark/TaskContextImpl.scala
    	sql/core/src/test/scala/org/apache/spark/sql/execution/ui/SQLListenerSuite.scala

commit 9222f11d2b838b3bd61d62396719e44cb728bf66
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-07T03:55:50Z

    Do not send TaskMetrics from executors to driver
    
    Instead, send only accumulator updates. As of this commit
    TaskMetrics is only used as a syntactic sugar on the executor
    side to modify accumulator values by names. Now we no longer
    send the same thing in two different codepaths.
    
    Now that we never send TaskMetrics from executors to the driver,
    we also never send accumulators that way. Then we can revert some
    of the accumulator changes.

commit 2add53f8f83eeec9dd9338a18c32121ddded8afc
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-07T04:08:14Z

    Restore accumulator serialization semantics
    
    In the previous commit, we made accumulator communication one-way
    again, which is the same as before this patch, so we restored all
    the semantics involved in serializing accumulators as before.
    
    Note: tests are still failing because of a duplicate accumulator
    name in some SQL things. Run `DataFrameCallbackSuite` for more
    detail.

commit afe957c5f28a1c24cd49194e60fb309e301aa427
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-07T21:06:12Z

    Fix semantics of accumulators when tasks fail
    
    Currently we still get values for tasks that fail. We should
    keep this semantics in the new accumulator updates as well.

commit c7240f3dd4490e81905131841596eee72be72898
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-07T21:38:55Z

    Fix a few more tests
    
    There are a few places where we passed in empty internal
    accumulators to TaskContextImpl, so the TaskMetrics creation
    would fail. These are now fixed.

commit fa086c36647520f6e04622003a8e795bbffd98ad
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-08T01:24:35Z

    Fix SQL UI
    
    Before this commit the SQL UI would not display any accumulators.
    This is because it is powered by the SQLListener, which reads
    accumulators from TaskMetrics. However, we did not update the
    accumulator values before posting the TaskMetrics, so the UI
    never saw the updates from the tasks.
    
    This commit also fixes a few related test failures.

commit 361442e30ae5938a23e93a276feae1112ce816a0
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-08T02:52:28Z

    Clean up: lift odd unique name requirement
    
    Now internal accumulators no longer need to have unique names.
    This was an unnecessary hack for the SQL accumulators that can
    be reverted through some clean ups.

commit 5aa6aa1bb60d4dead372b307efb1b3b725b4e409
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-08T18:59:59Z

    Move smaller metrics classes to their own files
    
    for readability.

commit 7118be5c16e98bcae7b5effa1ef462dc8f780655
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-08T19:11:07Z

    Fix SQLQuerySuite
    
    A few bugs:
    
    (1) In Executor.scala, we updated TaskMetrics after collecting the
    accumulator values. We should do it the other order.
    
    (2) The test utility method of verifying whether peak execution
    memory is set imposed this requirement on every single job run
    in the test body. This does not apply for SQL's external sort,
    however, because one of the jobs does a sample and so does not
    update peak execution memory.
    
    (3) We were getting accumulators from executors that were not
    registered on the driver. Not exactly sure what the cause is but
    it could very well have to do with GC on the driver since we use
    weak references there. We shouldn't crash the scheduler if this
    happens.

commit 176e91d0abfaed3d8741821787a3732b0b9f3688
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-08T21:22:21Z

    Remove unused hostname from TaskMetrics

commit 7939e1c74d9dd62e383c71c72d9b353c3f6d8dfc
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-08T22:18:17Z

    Reinitialize OutputMetrics et al during reconstruction
    
    Such that downstream listeners can access their values.
    This commit also generalizes the internal accumulator type from
    Long to anything, since we need to store the read and write
    methods of InputMetrics and OutputMetrics respectively.

commit c029f62be0628af904fe1a1ad795e8208f55e51d
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-08T23:52:02Z

    Fix *ShuffleSuite
    
    This fixes a bug where when we reconstruct TaskMetrics we just
    pass in mutable accumulators, such that when new tasks come in
    they change the values of the old tasks. A more subtle bug here
    is that we were passing in the accumulated values instead of the
    local task values. Both are now fixed.
    
    TODO: write a test for all of these please.

commit b3c51dd3930f8c304f0fc67fc476d95d066f9a1e
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-09T01:45:48Z

    Fix DAGSchedulerSuite
    
    The fake accumulator values should no longer all be Longs. Ugh.

commit d531f3fff759dbfca6f24eb41ec37428ea055785
Author: Andrew Or <an...@databricks.com>
Date:   2016-01-09T06:51:17Z

    Simplify accumulator update code a little

----


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172610449
  
    **[Test build #49601 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49601/consoleFull)** for PR 10717 at commit [`ed9de9c`](https://github.com/apache/spark/commit/ed9de9c084b0a8df41b24768404c63cb331c8a25).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172088814
  
    retest this please


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172681157
  
    Note to self: DO NOT DELETE THIS BRANCH!!


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171868462
  
    **[Test build #49435 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49435/consoleFull)** for PR 10717 at commit [`5ca949f`](https://github.com/apache/spark/commit/5ca949f23170ca8b1763800ca106024ff29c85cf).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172643047
  
    **[Test build #49601 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49601/consoleFull)** for PR 10717 at commit [`ed9de9c`](https://github.com/apache/spark/commit/ed9de9c084b0a8df41b24768404c63cb331c8a25).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171868548
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172107100
  
    Yeah it's still WIP but once it's in a more ready state I'll try to break it up.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171832844
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171817787
  
    **[Test build #49424 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49424/consoleFull)** for PR 10717 at commit [`b31862a`](https://github.com/apache/spark/commit/b31862a089b20e516c77b558917c4210cdb31cfd).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171035550
  
    **[Test build #49255 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49255/consoleFull)** for PR 10717 at commit [`67153e6`](https://github.com/apache/spark/commit/67153e68cb455c5e99bd0374a9a6d828f0e53c66).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171019767
  
    Build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171839929
  
    **[Test build #49431 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49431/consoleFull)** for PR 10717 at commit [`fe451e5`](https://github.com/apache/spark/commit/fe451e5d516413347ed7bd7ebd94eb25088de60e).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171894853
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171830720
  
    **[Test build #49430 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49430/consoleFull)** for PR 10717 at commit [`3dcc4e1`](https://github.com/apache/spark/commit/3dcc4e16acd0c2ceaa264ad23abcf067935d6a8f).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172098855
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171117384
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171843479
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172100380
  
    **[Test build #49485 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49485/consoleFull)** for PR 10717 at commit [`b58f2e6`](https://github.com/apache/spark/commit/b58f2e64e05e3e7f89ff8da4823007d810fc98b4).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172142232
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171032731
  
    retest this please


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-170772475
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172144616
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171530961
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172652408
  
    OK, I broke this down into smaller PRs so far:
    
    #10810 - move classes to their own files for readability
    #10811 - rename 3 fields in ShuffleWriteMetrics
    #10815 - refactor TaskMetrics to make it not use public var's
    
    More coming soon, I'll keep this comment updated.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172091026
  
    **[Test build #49480 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49480/consoleFull)** for PR 10717 at commit [`23af334`](https://github.com/apache/spark/commit/23af334f2ab7c7fbd54ff9ca2b0bc547f5187937).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171818153
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171527492
  
    **[Test build #49371 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49371/consoleFull)** for PR 10717 at commit [`8b99071`](https://github.com/apache/spark/commit/8b99071f1591abce3cb61651413747188fb47ec5).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172124993
  
    **[Test build #49485 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49485/consoleFull)** for PR 10717 at commit [`b58f2e6`](https://github.com/apache/spark/commit/b58f2e64e05e3e7f89ff8da4823007d810fc98b4).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172681125
  
    By the end of all the smaller patches there will be too much conflict to resolve, so I'm closing this.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172125140
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172050783
  
    **[Test build #49469 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49469/consoleFull)** for PR 10717 at commit [`ce10ea1`](https://github.com/apache/spark/commit/ce10ea169cb37611f0e7e36c5303f3ab454fcc7a).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-170772298
  
    **[Test build #49207 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49207/consoleFull)** for PR 10717 at commit [`61fc9e1`](https://github.com/apache/spark/commit/61fc9e12bb46983a086f63c126c70120841631ab).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171106558
  
    **[Test build #49267 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49267/consoleFull)** for PR 10717 at commit [`b271205`](https://github.com/apache/spark/commit/b271205c6c2b4170d5abd2ebb52f48fa8463112d).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171512042
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171106734
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171040100
  
    **[Test build #49255 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49255/consoleFull)** for PR 10717 at commit [`67153e6`](https://github.com/apache/spark/commit/67153e68cb455c5e99bd0374a9a6d828f0e53c66).
     * This patch **fails MiMa tests**.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `class PageView(val url: String, val status: Int, val zipCode: Int, val userID: Int)`
      * `class FlumeEventServer(receiver: FlumeReceiver) extends AvroSourceProtocol `
      * `class UnsafeCartesianRDD(left: RDD[UnsafeRow], right: RDD[UnsafeRow], numFieldsOfRight: Int)`
      * `case class JdbcType(databaseTypeDefinition: String, jdbcNullType: Int)`
      * `class ObjectInputStreamWithLoader(_inputStream: InputStream, loader: ClassLoader)`
      * `class ConstantInputDStream[T: ClassTag](_ssc: StreamingContext, rdd: RDD[T])`
      * `abstract class InputDStream[T: ClassTag] (_ssc: StreamingContext)`
      * `abstract class ReceiverInputDStream[T: ClassTag](_ssc: StreamingContext)`


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-170772471
  
    **[Test build #49207 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49207/consoleFull)** for PR 10717 at commit [`61fc9e1`](https://github.com/apache/spark/commit/61fc9e12bb46983a086f63c126c70120841631ab).
     * This patch **fails Scala style tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171064401
  
    **[Test build #49261 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49261/consoleFull)** for PR 10717 at commit [`c94420b`](https://github.com/apache/spark/commit/c94420b3552fd15fe786ecc02e6f8d70c8d2f316).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#discussion_r49795953
  
    --- Diff: project/MimaExcludes.scala ---
    @@ -147,6 +147,16 @@ object MimaExcludes {
             ProblemFilters.exclude[MissingMethodProblem]("org.apache.spark.streaming.flume.sink.Logging.org$apache$spark$streaming$flume$sink$Logging$$_log_="),
             ProblemFilters.exclude[MissingMethodProblem]("org.apache.spark.streaming.flume.sink.TransactionProcessor.org$apache$spark$streaming$flume$sink$Logging$$log_"),
             ProblemFilters.exclude[MissingMethodProblem]("org.apache.spark.streaming.flume.sink.TransactionProcessor.org$apache$spark$streaming$flume$sink$Logging$$log__=")
    +      ) ++ Seq(
    +        // SPARK-10620 Migrate TaskMetrics to accumulators
    +        ProblemFilters.exclude[MissingMethodProblem](
    +          "org.apache.spark.TaskContext.internalMetricsToAccumulators"),
    +        ProblemFilters.exclude[MissingMethodProblem](
    +          "org.apache.spark.TaskContext.collectInternalAccumulators"),
    +        ProblemFilters.exclude[MissingMethodProblem](
    +          "org.apache.spark.TaskContext.collectAccumulators"),
    +        ProblemFilters.exclude[MissingMethodProblem]("org.apache.spark.Accumulable.this"),
    --- End diff --
    
    Do you remember which change triggered this error?


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172643339
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171517738
  
    **[Test build #49372 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49372/consoleFull)** for PR 10717 at commit [`dd6d8ab`](https://github.com/apache/spark/commit/dd6d8ab72b3ddb025728365bb1e8116a7da67199).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171832818
  
    **[Test build #49430 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49430/consoleFull)** for PR 10717 at commit [`3dcc4e1`](https://github.com/apache/spark/commit/3dcc4e16acd0c2ceaa264ad23abcf067935d6a8f).
     * This patch **fails MiMa tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172139646
  
    retest this please


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171116959
  
    **[Test build #49270 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49270/consoleFull)** for PR 10717 at commit [`8065b25`](https://github.com/apache/spark/commit/8065b250fffed9566fc93286a43e5e4a47dbb948).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-170832360
  
    **[Test build #49225 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49225/consoleFull)** for PR 10717 at commit [`4d681fa`](https://github.com/apache/spark/commit/4d681fa04f2c4d8ea5d22b03a15724e7394cbc0e).
     * This patch **fails MiMa tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171021289
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171512515
  
    **[Test build #49371 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49371/consoleFull)** for PR 10717 at commit [`8b99071`](https://github.com/apache/spark/commit/8b99071f1591abce3cb61651413747188fb47ec5).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172047878
  
    retest this please


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171101749
  
    **[Test build #49261 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49261/consoleFull)** for PR 10717 at commit [`c94420b`](https://github.com/apache/spark/commit/c94420b3552fd15fe786ecc02e6f8d70c8d2f316).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172098587
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171075771
  
    **[Test build #49267 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49267/consoleFull)** for PR 10717 at commit [`b271205`](https://github.com/apache/spark/commit/b271205c6c2b4170d5abd2ebb52f48fa8463112d).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172090487
  
    **[Test build #49482 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49482/consoleFull)** for PR 10717 at commit [`23af334`](https://github.com/apache/spark/commit/23af334f2ab7c7fbd54ff9ca2b0bc547f5187937).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171847379
  
    **[Test build #49435 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49435/consoleFull)** for PR 10717 at commit [`5ca949f`](https://github.com/apache/spark/commit/5ca949f23170ca8b1763800ca106024ff29c85cf).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172144614
  
    **[Test build #49509 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49509/consoleFull)** for PR 10717 at commit [`00a12a4`](https://github.com/apache/spark/commit/00a12a468e1f83f6fce61b88ce494059e512df21).
     * This patch **fails Scala style tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171530613
  
    **[Test build #49372 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49372/consoleFull)** for PR 10717 at commit [`dd6d8ab`](https://github.com/apache/spark/commit/dd6d8ab72b3ddb025728365bb1e8116a7da67199).
     * This patch **fails Spark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171040172
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172051923
  
    **[Test build #49469 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49469/consoleFull)** for PR 10717 at commit [`ce10ea1`](https://github.com/apache/spark/commit/ce10ea169cb37611f0e7e36c5303f3ab454fcc7a).
     * This patch **fails Scala style tests**.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `trait ObjectOperator extends LogicalPlan `
      * `case class MapPartitions(`
      * `case class AppendColumns(`
      * `case class MapGroups(`
      * `case class CoGroup(`
      * `trait ObjectOperator extends SparkPlan `
      * `case class MapPartitions(`
      * `case class AppendColumns(`
      * `case class MapGroups(`
      * `case class CoGroup(`


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171083042
  
    **[Test build #49270 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49270/consoleFull)** for PR 10717 at commit [`8065b25`](https://github.com/apache/spark/commit/8065b250fffed9566fc93286a43e5e4a47dbb948).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-170832397
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171101966
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172144528
  
    **[Test build #49509 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49509/consoleFull)** for PR 10717 at commit [`00a12a4`](https://github.com/apache/spark/commit/00a12a468e1f83f6fce61b88ce494059e512df21).


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172104593
  
    This patch is really difficult to review since it combines simple refactoring and more subtle logic. Is it possible to break this up? If there were patches that did simple clean up, those can get reviewed and merged very quickly.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172051936
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171818148
  
    **[Test build #49424 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49424/consoleFull)** for PR 10717 at commit [`b31862a`](https://github.com/apache/spark/commit/b31862a089b20e516c77b558917c4210cdb31cfd).
     * This patch **fails Scala style tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172070731
  
    Merged build finished. Test FAILed.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

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


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-171843472
  
    **[Test build #49431 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49431/consoleFull)** for PR 10717 at commit [`fe451e5`](https://github.com/apache/spark/commit/fe451e5d516413347ed7bd7ebd94eb25088de60e).
     * This patch **fails Scala style tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-172098937
  
    retest this please


---
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-10620][WIP] Migrate TaskMetrics to accu...

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

    https://github.com/apache/spark/pull/10717#issuecomment-170829529
  
    **[Test build #49225 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/49225/consoleFull)** for PR 10717 at commit [`4d681fa`](https://github.com/apache/spark/commit/4d681fa04f2c4d8ea5d22b03a15724e7394cbc0e).


---
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