diff options
author | Stephen Haberman <stephen@exigencecorp.com> | 2013-01-05 10:54:05 -0600 |
---|---|---|
committer | Stephen Haberman <stephen@exigencecorp.com> | 2013-01-05 12:04:01 -0600 |
commit | 8d57c78c83f74e45ce3c119e2e3915d5eac264e7 (patch) | |
tree | 37ad97ed09bc92daf4c7106978d0a9fc20766121 /core | |
parent | 55809fbc6db5929332cd45fa3281f9190098c6c6 (diff) | |
download | spark-8d57c78c83f74e45ce3c119e2e3915d5eac264e7.tar.gz spark-8d57c78c83f74e45ce3c119e2e3915d5eac264e7.tar.bz2 spark-8d57c78c83f74e45ce3c119e2e3915d5eac264e7.zip |
Add PairRDDFunctions.keys and values.
Diffstat (limited to 'core')
-rw-r--r-- | core/src/main/scala/spark/PairRDDFunctions.scala | 10 | ||||
-rw-r--r-- | core/src/test/scala/spark/ShuffleSuite.scala | 7 |
2 files changed, 17 insertions, 0 deletions
diff --git a/core/src/main/scala/spark/PairRDDFunctions.scala b/core/src/main/scala/spark/PairRDDFunctions.scala index 413c944a66..ce48cea903 100644 --- a/core/src/main/scala/spark/PairRDDFunctions.scala +++ b/core/src/main/scala/spark/PairRDDFunctions.scala @@ -615,6 +615,16 @@ class PairRDDFunctions[K: ClassManifest, V: ClassManifest]( writer.cleanup() } + /** + * Return an RDD with the keys of each tuple. + */ + def keys: RDD[K] = self.map(_._1) + + /** + * Return an RDD with the values of each tuple. + */ + def values: RDD[V] = self.map(_._2) + private[spark] def getKeyClass() = implicitly[ClassManifest[K]].erasure private[spark] def getValueClass() = implicitly[ClassManifest[V]].erasure diff --git a/core/src/test/scala/spark/ShuffleSuite.scala b/core/src/test/scala/spark/ShuffleSuite.scala index 8170100f1d..5a867016f2 100644 --- a/core/src/test/scala/spark/ShuffleSuite.scala +++ b/core/src/test/scala/spark/ShuffleSuite.scala @@ -216,6 +216,13 @@ class ShuffleSuite extends FunSuite with ShouldMatchers with BeforeAndAfter { // Test that a shuffle on the file works, because this used to be a bug assert(file.map(line => (line, 1)).reduceByKey(_ + _).collect().toList === Nil) } + + test("kesy and values") { + sc = new SparkContext("local", "test") + val rdd = sc.parallelize(Array((1, "a"), (2, "b"))) + assert(rdd.keys.collect().toList === List(1, 2)) + assert(rdd.values.collect().toList === List("a", "b")) + } } object ShuffleSuite { |