You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by larryxiao <gi...@git.apache.org> on 2014/08/12 10:41:41 UTC

[GitHub] spark pull request: [SPARK-2062][GraphX] VertexRDD.apply does not ...

GitHub user larryxiao opened a pull request:

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

    [SPARK-2062][GraphX] VertexRDD.apply does not use the mergeFunc

    create verticesDeduplicate with reduceByKey, using mergeFunc
    then proceed with verticesDedup
    
    But this is not tested and I want to add a test on VertexRDD.apply, 
    because it need Edges, should I place it in VertexRDDSuite or else?

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

    $ git pull https://github.com/larryxiao/spark 2062

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

    https://github.com/apache/spark/pull/1903.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 #1903
    
----
commit 20d80a38349d3fad0800763cb90d07b224927ad4
Author: Larry Xiao <xi...@sjtu.edu.cn>
Date:   2014-08-12T08:35:00Z

    [SPARK-2062][GraphX] VertexRDD.apply does not use the mergeFunc
    
    create verticesDeduplicate with reduceByKey, using mergeFunc
    then proceed with verticesDedup

----


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55725153
  
      [QA tests have finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20378/consoleFull) for   PR 1903 at commit [`dfdb3c9`](https://github.com/apache/spark/commit/dfdb3c91e791f66e0e3a58f32bb72c1f4855397b).
     * This patch **passes** 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-52584319
  
    It is ok now?
    
    About testing: how do I test it? I think it should be added in GraphSuite. Is it necessary?
    
    Thank you!


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-54243879
  
      [QA tests have started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/19623/consoleFull) for   PR 1903 at commit [`1c70366`](https://github.com/apache/spark/commit/1c70366d2fdb8354cbb58eb465b137be1ccb562f).
     * This patch merges cleanly.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-54242231
  
      [QA tests have started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/19618/consoleFull) for   PR 1903 at commit [`e4ca697`](https://github.com/apache/spark/commit/e4ca697a4a5b4a6cad5cf6dfd19dd793fb9d41c5).
     * This patch merges cleanly.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-56083212
  
      [QA tests have started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20546/consoleFull) for   PR 1903 at commit [`625aa9d`](https://github.com/apache/spark/commit/625aa9d2c2fd434040416b863cfe72c9208d7b26).
     * This patch merges cleanly.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-54694479
  
    Can one of the admins verify this patch?


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#discussion_r17028039
  
    --- Diff: graphx/src/main/scala/org/apache/spark/graphx/impl/ShippableVertexPartition.scala ---
    @@ -36,17 +36,27 @@ private[graphx]
     object ShippableVertexPartition {
       /** Construct a `ShippableVertexPartition` from the given vertices without any routing table. */
       def apply[VD: ClassTag](iter: Iterator[(VertexId, VD)]): ShippableVertexPartition[VD] =
    -    apply(iter, RoutingTablePartition.empty, null.asInstanceOf[VD])
    +    apply(iter, RoutingTablePartition.empty, null.asInstanceOf[VD], (a, b) => a)
     
       /**
        * Construct a `ShippableVertexPartition` from the given vertices with the specified routing
        * table, filling in missing vertices mentioned in the routing table using `defaultVal`.
        */
       def apply[VD: ClassTag](
    -      iter: Iterator[(VertexId, VD)], routingTable: RoutingTablePartition, defaultVal: VD)
    +    iter: Iterator[(VertexId, VD)], routingTable: RoutingTablePartition, defaultVal: VD)
    +    : ShippableVertexPartition[VD] = 
    +    apply(iter, routingTable, defaultVal, (a, b) => a)
    +
    +  /**
    +   * Construct a `ShippableVertexPartition` from the given vertices with the specified routing
    +   * table, filling in missing vertices mentioned in the routing table using `defaultVal`,
    +   * and merging duplicate vertex atrribute with mergeFunc.
    +   */
    +  def apply[VD: ClassTag](
    +    iter: Iterator[(VertexId, VD)], routingTable: RoutingTablePartition, defaultVal: VD, mergeFunc: (VD, VD) => VD)
    --- End diff --
    
    Looks like this line is too long - it would be great if you could wrap it. Also, I think the Spark style is for parameter lists to be indented 4 spaces instead of 2.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55937556
  
    Your doc update made me realize that now that we're taking a mergeFunc in ShippableVertexPartition.initFrom, it's not ideal to use the iterator concatenation approach for setting the default values anymore, because the mergeFunc will get run on the default value, which might surprise users. I submitted a PR (larryxiao/spark#1) to avoid this by doing the merge first, *then* populating the default values.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-56004578
  
    Jenkins, test 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-51887950
  
    Can one of the admins verify this patch?


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55831449
  
      [QA tests have started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20434/consoleFull) for   PR 1903 at commit [`614059f`](https://github.com/apache/spark/commit/614059fd711a5cd0351c133fc98a288f8c712ad1).
     * This patch merges cleanly.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-54241932
  
    ok to test


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-56009542
  
      [QA tests have finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20532/consoleFull) for   PR 1903 at commit [`625aa9d`](https://github.com/apache/spark/commit/625aa9d2c2fd434040416b863cfe72c9208d7b26).
     * This patch **fails** 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-54242289
  
      [QA tests have finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/19618/consoleFull) for   PR 1903 at commit [`e4ca697`](https://github.com/apache/spark/commit/e4ca697a4a5b4a6cad5cf6dfd19dd793fb9d41c5).
     * This patch **fails** 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55718541
  
      [QA tests have started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20378/consoleFull) for   PR 1903 at commit [`dfdb3c9`](https://github.com/apache/spark/commit/dfdb3c91e791f66e0e3a58f32bb72c1f4855397b).
     * This patch merges cleanly.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55718245
  
    Yeah, a note about that default would be great.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-52378594
  
    This isn't quite the right approach, since the call to reduceByKey will result in two rounds of communication and hash aggregations (in reduceByKey and copartitionWithVertices) when only one is necessary. It would be better to add a ShippableVertexPartition constructor that takes a mergeFunc, then just pass the mergeFunc from here into that constructor.
    
    Also, the capitalization in `VD1` and `VD2` suggests that they are type parameters when they are actually function parameters -- they should probably just be `a` and `b`.
    
    I can make these changes this weekend if you like.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55725506
  
    This looks good! I'll merge it pending the doc update.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

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


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-56091645
  
      [QA tests have finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20546/consoleFull) for   PR 1903 at commit [`625aa9d`](https://github.com/apache/spark/commit/625aa9d2c2fd434040416b863cfe72c9208d7b26).
     * This patch **passes** 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55985270
  
    Thanks Ankur!
    I learn something :)


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-56082379
  
    Unrelated failure in Streaming. Jenkins, 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-56004870
  
      [QA tests have started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20532/consoleFull) for   PR 1903 at commit [`625aa9d`](https://github.com/apache/spark/commit/625aa9d2c2fd434040416b863cfe72c9208d7b26).
     * This patch merges cleanly.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55718184
  
    ok to test


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-54247107
  
      [QA tests have finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/19623/consoleFull) for   PR 1903 at commit [`1c70366`](https://github.com/apache/spark/commit/1c70366d2fdb8354cbb58eb465b137be1ccb562f).
     * This patch **fails** 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-56140430
  
    Thanks! Merged into master and branch-1.1.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55714900
  
    As described in commit message:
    
        a copy of vertices with defaultVal is created before, and it's b in
        (a, b) => b
    
        see in VertexPartition.scala
        val fullIter = iter ++ routingTable.iterator.map(vid => (vid, defaultVal))
    
    So there's hidden rule that default mergeFunc should be (a, b) => a.
    Should I write a comment to let user know about 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-55836255
  
      [QA tests have finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/20434/consoleFull) for   PR 1903 at commit [`614059f`](https://github.com/apache/spark/commit/614059fd711a5cd0351c133fc98a288f8c712ad1).
     * This patch **passes** 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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-52587969
  
    This looks good! A test would be good too. VertexRDDSuite seems like the right place, since nothing else actually calls this variant of VertexRDD.apply. It should be OK to create an empty EdgeRDD in VertexRDDSuite for testing purposes.
    
    Here's a simple test:
    
        val verts = sc.parallelize(List((0L, 1), (0L, 2), (1L, 3)))
        val edges = EdgeRDD.fromEdges(sc.parallelize(List.empty[Edge[Int]]))
        val rdd = VertexRDD(verts, edges, 0, (a: Int, b: Int) => a + b)
        assert(rdd.collect.toSet == Set((0L, 3), (1L, 3)))


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-52590257
  
    Thank you Ankur!
    
    I'll add test to it.


---
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-2062][GraphX] VertexRDD.apply does not ...

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

    https://github.com/apache/spark/pull/1903#issuecomment-52381062
  
    Thanks
    I  can do it


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