You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by marmbrus <gi...@git.apache.org> on 2014/04/15 07:46:31 UTC

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

GitHub user marmbrus opened a pull request:

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

    SPARK-1456 Remove view bounds on Ordered in favor of a context bound on Ordering.

    This doesn't require creating new Ordering objects per row.  Additionally, [view bounds are going to be deprecated](https://issues.scala-lang.org/browse/SI-7629), so we should get rid of them while APIs are still flexible.

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

    $ git pull https://github.com/marmbrus/spark viewBounds

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

    https://github.com/apache/spark/pull/410.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 #410
    
----
commit 35798a8accb829ccb0b637fad54f208d77582b60
Author: Michael Armbrust <mi...@databricks.com>
Date:   2014-04-15T03:14:37Z

    Remove view bounds on Ordered in favor of a context bound on Ordering.

----


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40655122
  
     Merged build triggered. 


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40834077
  
    All automated tests passed.
    Refer to this link for build results: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/14240/


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#discussion_r11692970
  
    --- Diff: core/src/main/scala/org/apache/spark/rdd/OrderedRDDFunctions.scala ---
    @@ -27,12 +27,14 @@ import org.apache.spark.{Logging, RangePartitioner}
      * use these functions. They will work with any key type that has a `scala.math.Ordered`
      * implementation.
      */
    -class OrderedRDDFunctions[K <% Ordered[K]: ClassTag,
    +class OrderedRDDFunctions[K : Ordering : ClassTag,
    --- End diff --
    
    @marmbrus Do you mind updating the scaladoc above since now users can pass their own orderings in addition to an `Ordered` type? It might be nice to just give a one-line example of how they can define a custom ordering as well.


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40835109
  
    lgtm


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

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


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40830816
  
     Merged build triggered. 


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40447921
  
     Merged build triggered. 


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40835201
  
    merged.



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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40658327
  
    All automated tests passed.
    Refer to this link for build results: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/14188/


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40447926
  
    Merged build started. 


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#discussion_r11620835
  
    --- Diff: core/src/main/scala/org/apache/spark/Partitioner.scala ---
    @@ -89,12 +89,14 @@ class HashPartitioner(partitions: Int) extends Partitioner {
      * A [[org.apache.spark.Partitioner]] that partitions sortable records by range into roughly
      * equal ranges. The ranges are determined by sampling the content of the RDD passed in.
      */
    -class RangePartitioner[K <% Ordered[K]: ClassTag, V](
    +class RangePartitioner[K : Ordering : ClassTag, V](
    --- End diff --
    
    Can the user still pass in an Ordering if they want a specific Ordering?


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40830826
  
    Merged build started. 


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40658326
  
    Merged build finished. All automated tests passed.


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#discussion_r11714752
  
    --- Diff: core/src/main/scala/org/apache/spark/rdd/OrderedRDDFunctions.scala ---
    @@ -24,15 +24,31 @@ import org.apache.spark.{Logging, RangePartitioner}
     /**
      * Extra functions available on RDDs of (key, value) pairs where the key is sortable through
      * an implicit conversion. Import `org.apache.spark.SparkContext._` at the top of your program to
    - * use these functions. They will work with any key type that has a `scala.math.Ordered`
    - * implementation.
    + * use these functions. They will work with any key type `K` that has an implicit `Ordering[K]` in
    + * scope.  Ordering objects already exist for all of the standard primitive types.  Users can also
    + * define their own orderings for custom types, or to override the default ordering.  The implicit
    + * ordering that is in the closest scope will be used.
    + *
    + * {{{
    + *   import org.apache.spark.SparkContext._
    + *
    + *   val rdd: RDD[(String, Int)] = ...
    + *   implicit val caseInsensitiveOrdering = new Ordering[String] {
    + *     override def compare(a: String, b: String) = a.toLowerCase.compare(b.toLowerCase)
    + *   }
    + *
    + *   // Sort by key, using the above case insensitive ordering.
    + *   rdd.sortByKey
    --- End diff --
    
    a nit: put parenthesis around sortByKey 


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#discussion_r11708647
  
    --- Diff: core/src/main/scala/org/apache/spark/rdd/OrderedRDDFunctions.scala ---
    @@ -27,12 +27,14 @@ import org.apache.spark.{Logging, RangePartitioner}
      * use these functions. They will work with any key type that has a `scala.math.Ordered`
      * implementation.
      */
    -class OrderedRDDFunctions[K <% Ordered[K]: ClassTag,
    +class OrderedRDDFunctions[K : Ordering : ClassTag,
    --- End diff --
    
    done.


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40451063
  
    All automated tests passed.
    Refer to this link for build results: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/14135/


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40451062
  
    Merged build finished. All automated tests passed.


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40655134
  
    Merged build started. 


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40669391
  
    We should also update the Java API to create Ordering directly (instead of Ordered). It still compiles now, but kind of strange to have Ordered in the Java API that gets converted into Ordering.


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#discussion_r11621180
  
    --- Diff: core/src/main/scala/org/apache/spark/Partitioner.scala ---
    @@ -89,12 +89,14 @@ class HashPartitioner(partitions: Int) extends Partitioner {
      * A [[org.apache.spark.Partitioner]] that partitions sortable records by range into roughly
      * equal ranges. The ranges are determined by sampling the content of the RDD passed in.
      */
    -class RangePartitioner[K <% Ordered[K]: ClassTag, V](
    +class RangePartitioner[K : Ordering : ClassTag, V](
    --- End diff --
    
    Yes, when de-sugared there is an implicit Ordering -- that's why Michael can recover and bind it explicitly at line 98.


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

[GitHub] spark pull request: SPARK-1456 Remove view bounds on Ordered in fa...

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

    https://github.com/apache/spark/pull/410#issuecomment-40834076
  
    Merged build finished. All automated tests passed.


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