@@ -469,6 +469,22 @@ class JavaPairRDD[K, V](val rdd: RDD[(K, V)])
469469 fromRDD(joinResult.mapValues{case (v, w) => (JavaUtils .optionToOptional(v), w)})
470470 }
471471
472+ /**
473+ * Perform a full outer join of `this` and `other`. For each element (k, v) in `this`, the
474+ * resulting RDD will either contain all pairs (k, (Some(v), Some(w))) for w in `other`, or
475+ * the pair (k, (Some(v), None)) if no elements in `other` have key k. Similarly, for each
476+ * element (k, w) in `other`, the resulting RDD will either contain all pairs
477+ * (k, (Some(v), Some(w))) for v in `this`, or the pair (k, (None, Some(w))) if no elements
478+ * in `this` have key k. Uses the given Partitioner to partition the output RDD.
479+ */
480+ def fullOuterJoin [W ](other : JavaPairRDD [K , W ], partitioner : Partitioner )
481+ : JavaPairRDD [K , (Optional [V ], Optional [W ])] = {
482+ val joinResult = rdd.fullOuterJoin(other, partitioner)
483+ fromRDD(joinResult.mapValues{ case (v, w) =>
484+ (JavaUtils .optionToOptional(v), JavaUtils .optionToOptional(w))
485+ })
486+ }
487+
472488 /**
473489 * Simplified version of combineByKey that hash-partitions the resulting RDD using the existing
474490 * partitioner/parallelism level.
@@ -563,6 +579,38 @@ class JavaPairRDD[K, V](val rdd: RDD[(K, V)])
563579 fromRDD(joinResult.mapValues{case (v, w) => (JavaUtils .optionToOptional(v), w)})
564580 }
565581
582+ /**
583+ * Perform a full outer join of `this` and `other`. For each element (k, v) in `this`, the
584+ * resulting RDD will either contain all pairs (k, (Some(v), Some(w))) for w in `other`, or
585+ * the pair (k, (Some(v), None)) if no elements in `other` have key k. Similarly, for each
586+ * element (k, w) in `other`, the resulting RDD will either contain all pairs
587+ * (k, (Some(v), Some(w))) for v in `this`, or the pair (k, (None, Some(w))) if no elements
588+ * in `this` have key k. Hash-partitions the resulting RDD using the existing partitioner/
589+ * parallelism level.
590+ */
591+ def fullOuterJoin [W ](other : JavaPairRDD [K , W ]): JavaPairRDD [K , (Optional [V ], Optional [W ])] = {
592+ val joinResult = rdd.fullOuterJoin(other)
593+ fromRDD(joinResult.mapValues{ case (v, w) =>
594+ (JavaUtils .optionToOptional(v), JavaUtils .optionToOptional(w))
595+ })
596+ }
597+
598+ /**
599+ * Perform a full outer join of `this` and `other`. For each element (k, v) in `this`, the
600+ * resulting RDD will either contain all pairs (k, (Some(v), Some(w))) for w in `other`, or
601+ * the pair (k, (Some(v), None)) if no elements in `other` have key k. Similarly, for each
602+ * element (k, w) in `other`, the resulting RDD will either contain all pairs
603+ * (k, (Some(v), Some(w))) for v in `this`, or the pair (k, (None, Some(w))) if no elements
604+ * in `this` have key k. Hash-partitions the resulting RDD into the given number of partitions.
605+ */
606+ def fullOuterJoin [W ](other : JavaPairRDD [K , W ], numPartitions : Int )
607+ : JavaPairRDD [K , (Optional [V ], Optional [W ])] = {
608+ val joinResult = rdd.fullOuterJoin(other, numPartitions)
609+ fromRDD(joinResult.mapValues{ case (v, w) =>
610+ (JavaUtils .optionToOptional(v), JavaUtils .optionToOptional(w))
611+ })
612+ }
613+
566614 /**
567615 * Return the key-value pairs in this RDD to the master as a Map.
568616 */
0 commit comments