Skip to content

Commit dc7bdb3

Browse files
committed
solve conflicts
2 parents d7c3e1e + 7e0cc36 commit dc7bdb3

File tree

79 files changed

+2175
-835
lines changed

Some content is hidden

Large Commits have some content hidden by default. Use the searchbox below for content that may be hidden.

79 files changed

+2175
-835
lines changed

assembly/pom.xml

Lines changed: 3 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -141,7 +141,9 @@
141141
<include>com.google.common.**</include>
142142
</includes>
143143
<excludes>
144-
<exclude>com.google.common.base.Optional**</exclude>
144+
<exclude>com/google/common/base/Absent*</exclude>
145+
<exclude>com/google/common/base/Optional*</exclude>
146+
<exclude>com/google/common/base/Present*</exclude>
145147
</excludes>
146148
</relocation>
147149
</relocations>

core/pom.xml

Lines changed: 2 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -343,7 +343,9 @@
343343
<filter>
344344
<artifact>com.google.guava:guava</artifact>
345345
<includes>
346+
<include>com/google/common/base/Absent*</include>
346347
<include>com/google/common/base/Optional*</include>
348+
<include>com/google/common/base/Present*</include>
347349
</includes>
348350
</filter>
349351
</filters>

core/src/main/scala/org/apache/spark/SparkContext.scala

Lines changed: 22 additions & 10 deletions
Original file line numberDiff line numberDiff line change
@@ -1030,28 +1030,40 @@ class SparkContext(config: SparkConf) extends Logging {
10301030
}
10311031

10321032
/**
1033-
* Support function for API backtraces.
1033+
* Set the thread-local property for overriding the call sites
1034+
* of actions and RDDs.
10341035
*/
1035-
def setCallSite(site: String) {
1036-
setLocalProperty("externalCallSite", site)
1036+
def setCallSite(shortCallSite: String) {
1037+
setLocalProperty(CallSite.SHORT_FORM, shortCallSite)
10371038
}
10381039

10391040
/**
1040-
* Support function for API backtraces.
1041+
* Set the thread-local property for overriding the call sites
1042+
* of actions and RDDs.
1043+
*/
1044+
private[spark] def setCallSite(callSite: CallSite) {
1045+
setLocalProperty(CallSite.SHORT_FORM, callSite.shortForm)
1046+
setLocalProperty(CallSite.LONG_FORM, callSite.longForm)
1047+
}
1048+
1049+
/**
1050+
* Clear the thread-local property for overriding the call sites
1051+
* of actions and RDDs.
10411052
*/
10421053
def clearCallSite() {
1043-
setLocalProperty("externalCallSite", null)
1054+
setLocalProperty(CallSite.SHORT_FORM, null)
1055+
setLocalProperty(CallSite.LONG_FORM, null)
10441056
}
10451057

10461058
/**
10471059
* Capture the current user callsite and return a formatted version for printing. If the user
1048-
* has overridden the call site, this will return the user's version.
1060+
* has overridden the call site using `setCallSite()`, this will return the user's version.
10491061
*/
10501062
private[spark] def getCallSite(): CallSite = {
1051-
Option(getLocalProperty("externalCallSite")) match {
1052-
case Some(callSite) => CallSite(callSite, longForm = "")
1053-
case None => Utils.getCallSite
1054-
}
1063+
Option(getLocalProperty(CallSite.SHORT_FORM)).map { case shortCallSite =>
1064+
val longCallSite = Option(getLocalProperty(CallSite.LONG_FORM)).getOrElse("")
1065+
CallSite(shortCallSite, longCallSite)
1066+
}.getOrElse(Utils.getCallSite())
10551067
}
10561068

10571069
/**

core/src/main/scala/org/apache/spark/api/java/JavaPairRDD.scala

Lines changed: 48 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -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
*/

core/src/main/scala/org/apache/spark/deploy/SparkSubmitArguments.scala

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -75,6 +75,10 @@ private[spark] class SparkSubmitArguments(args: Seq[String]) {
7575
defaultProperties
7676
}
7777

78+
// Respect SPARK_*_MEMORY for cluster mode
79+
driverMemory = sys.env.get("SPARK_DRIVER_MEMORY").orNull
80+
executorMemory = sys.env.get("SPARK_EXECUTOR_MEMORY").orNull
81+
7882
parseOpts(args.toList)
7983
mergeSparkProperties()
8084
checkRequiredArguments()

core/src/main/scala/org/apache/spark/executor/Executor.scala

Lines changed: 11 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -24,6 +24,7 @@ import java.util.concurrent._
2424

2525
import scala.collection.JavaConversions._
2626
import scala.collection.mutable.{ArrayBuffer, HashMap}
27+
import scala.util.control.NonFatal
2728

2829
import org.apache.spark._
2930
import org.apache.spark.deploy.SparkHadoopUtil
@@ -375,12 +376,17 @@ private[spark] class Executor(
375376
}
376377

377378
val message = Heartbeat(executorId, tasksMetrics.toArray, env.blockManager.blockManagerId)
378-
val response = AkkaUtils.askWithReply[HeartbeatResponse](message, heartbeatReceiverRef,
379-
retryAttempts, retryIntervalMs, timeout)
380-
if (response.reregisterBlockManager) {
381-
logWarning("Told to re-register on heartbeat")
382-
env.blockManager.reregister()
379+
try {
380+
val response = AkkaUtils.askWithReply[HeartbeatResponse](message, heartbeatReceiverRef,
381+
retryAttempts, retryIntervalMs, timeout)
382+
if (response.reregisterBlockManager) {
383+
logWarning("Told to re-register on heartbeat")
384+
env.blockManager.reregister()
385+
}
386+
} catch {
387+
case NonFatal(t) => logWarning("Issue communicating with driver in heartbeater", t)
383388
}
389+
384390
Thread.sleep(interval)
385391
}
386392
}

core/src/main/scala/org/apache/spark/rdd/PairRDDFunctions.scala

Lines changed: 42 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -506,6 +506,23 @@ class PairRDDFunctions[K, V](self: RDD[(K, V)])
506506
}
507507
}
508508

509+
/**
510+
* Perform a full outer join of `this` and `other`. For each element (k, v) in `this`, the
511+
* resulting RDD will either contain all pairs (k, (Some(v), Some(w))) for w in `other`, or
512+
* the pair (k, (Some(v), None)) if no elements in `other` have key k. Similarly, for each
513+
* element (k, w) in `other`, the resulting RDD will either contain all pairs
514+
* (k, (Some(v), Some(w))) for v in `this`, or the pair (k, (None, Some(w))) if no elements
515+
* in `this` have key k. Uses the given Partitioner to partition the output RDD.
516+
*/
517+
def fullOuterJoin[W](other: RDD[(K, W)], partitioner: Partitioner)
518+
: RDD[(K, (Option[V], Option[W]))] = {
519+
this.cogroup(other, partitioner).flatMapValues {
520+
case (vs, Seq()) => vs.map(v => (Some(v), None))
521+
case (Seq(), ws) => ws.map(w => (None, Some(w)))
522+
case (vs, ws) => for (v <- vs; w <- ws) yield (Some(v), Some(w))
523+
}
524+
}
525+
509526
/**
510527
* Simplified version of combineByKey that hash-partitions the resulting RDD using the
511528
* existing partitioner/parallelism level.
@@ -585,6 +602,31 @@ class PairRDDFunctions[K, V](self: RDD[(K, V)])
585602
rightOuterJoin(other, new HashPartitioner(numPartitions))
586603
}
587604

605+
/**
606+
* Perform a full outer join of `this` and `other`. For each element (k, v) in `this`, the
607+
* resulting RDD will either contain all pairs (k, (Some(v), Some(w))) for w in `other`, or
608+
* the pair (k, (Some(v), None)) if no elements in `other` have key k. Similarly, for each
609+
* element (k, w) in `other`, the resulting RDD will either contain all pairs
610+
* (k, (Some(v), Some(w))) for v in `this`, or the pair (k, (None, Some(w))) if no elements
611+
* in `this` have key k. Hash-partitions the resulting RDD using the existing partitioner/
612+
* parallelism level.
613+
*/
614+
def fullOuterJoin[W](other: RDD[(K, W)]): RDD[(K, (Option[V], Option[W]))] = {
615+
fullOuterJoin(other, defaultPartitioner(self, other))
616+
}
617+
618+
/**
619+
* Perform a full outer join of `this` and `other`. For each element (k, v) in `this`, the
620+
* resulting RDD will either contain all pairs (k, (Some(v), Some(w))) for w in `other`, or
621+
* the pair (k, (Some(v), None)) if no elements in `other` have key k. Similarly, for each
622+
* element (k, w) in `other`, the resulting RDD will either contain all pairs
623+
* (k, (Some(v), Some(w))) for v in `this`, or the pair (k, (None, Some(w))) if no elements
624+
* in `this` have key k. Hash-partitions the resulting RDD into the given number of partitions.
625+
*/
626+
def fullOuterJoin[W](other: RDD[(K, W)], numPartitions: Int): RDD[(K, (Option[V], Option[W]))] = {
627+
fullOuterJoin(other, new HashPartitioner(numPartitions))
628+
}
629+
588630
/**
589631
* Return the key-value pairs in this RDD to the master as a Map.
590632
*

core/src/main/scala/org/apache/spark/rdd/RDD.scala

Lines changed: 4 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -17,7 +17,7 @@
1717

1818
package org.apache.spark.rdd
1919

20-
import java.util.Random
20+
import java.util.{Properties, Random}
2121

2222
import scala.collection.{mutable, Map}
2323
import scala.collection.mutable.ArrayBuffer
@@ -41,7 +41,7 @@ import org.apache.spark.partial.CountEvaluator
4141
import org.apache.spark.partial.GroupedCountEvaluator
4242
import org.apache.spark.partial.PartialResult
4343
import org.apache.spark.storage.StorageLevel
44-
import org.apache.spark.util.{BoundedPriorityQueue, Utils}
44+
import org.apache.spark.util.{BoundedPriorityQueue, Utils, CallSite}
4545
import org.apache.spark.util.collection.OpenHashMap
4646
import org.apache.spark.util.random.{BernoulliSampler, PoissonSampler, SamplingUtils}
4747

@@ -1224,7 +1224,8 @@ abstract class RDD[T: ClassTag](
12241224
private var storageLevel: StorageLevel = StorageLevel.NONE
12251225

12261226
/** User code that created this RDD (e.g. `textFile`, `parallelize`). */
1227-
@transient private[spark] val creationSite = Utils.getCallSite
1227+
@transient private[spark] val creationSite = sc.getCallSite()
1228+
12281229
private[spark] def getCreationSite: String = Option(creationSite).map(_.shortForm).getOrElse("")
12291230

12301231
private[spark] def elementClassTag: ClassTag[T] = classTag[T]

core/src/main/scala/org/apache/spark/util/Utils.scala

Lines changed: 20 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -49,6 +49,11 @@ import org.apache.spark.serializer.{DeserializationStream, SerializationStream,
4949
/** CallSite represents a place in user code. It can have a short and a long form. */
5050
private[spark] case class CallSite(shortForm: String, longForm: String)
5151

52+
private[spark] object CallSite {
53+
val SHORT_FORM = "callSite.short"
54+
val LONG_FORM = "callSite.long"
55+
}
56+
5257
/**
5358
* Various utility methods used by Spark.
5459
*/
@@ -859,18 +864,26 @@ private[spark] object Utils extends Logging {
859864
}
860865
}
861866

862-
/**
863-
* A regular expression to match classes of the "core" Spark API that we want to skip when
864-
* finding the call site of a method.
865-
*/
866-
private val SPARK_CLASS_REGEX = """^org\.apache\.spark(\.api\.java)?(\.util)?(\.rdd)?\.[A-Z]""".r
867+
/** Default filtering function for finding call sites using `getCallSite`. */
868+
private def coreExclusionFunction(className: String): Boolean = {
869+
// A regular expression to match classes of the "core" Spark API that we want to skip when
870+
// finding the call site of a method.
871+
val SPARK_CORE_CLASS_REGEX = """^org\.apache\.spark(\.api\.java)?(\.util)?(\.rdd)?\.[A-Z]""".r
872+
val SCALA_CLASS_REGEX = """^scala""".r
873+
val isSparkCoreClass = SPARK_CORE_CLASS_REGEX.findFirstIn(className).isDefined
874+
val isScalaClass = SCALA_CLASS_REGEX.findFirstIn(className).isDefined
875+
// If the class is a Spark internal class or a Scala class, then exclude.
876+
isSparkCoreClass || isScalaClass
877+
}
867878

868879
/**
869880
* When called inside a class in the spark package, returns the name of the user code class
870881
* (outside the spark package) that called into Spark, as well as which Spark method they called.
871882
* This is used, for example, to tell users where in their code each RDD got created.
883+
*
884+
* @param skipClass Function that is used to exclude non-user-code classes.
872885
*/
873-
def getCallSite: CallSite = {
886+
def getCallSite(skipClass: String => Boolean = coreExclusionFunction): CallSite = {
874887
val trace = Thread.currentThread.getStackTrace()
875888
.filterNot { ste:StackTraceElement =>
876889
// When running under some profilers, the current stack trace might contain some bogus
@@ -891,7 +904,7 @@ private[spark] object Utils extends Logging {
891904

892905
for (el <- trace) {
893906
if (insideSpark) {
894-
if (SPARK_CLASS_REGEX.findFirstIn(el.getClassName).isDefined) {
907+
if (skipClass(el.getClassName)) {
895908
lastSparkMethod = if (el.getMethodName == "<init>") {
896909
// Spark method is a constructor; get its class name
897910
el.getClassName.substring(el.getClassName.lastIndexOf('.') + 1)

core/src/test/java/org/apache/spark/JavaAPISuite.java

Lines changed: 26 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1307,4 +1307,30 @@ public void collectUnderlyingScalaRDD() {
13071307
SomeCustomClass[] collected = (SomeCustomClass[]) rdd.rdd().retag(SomeCustomClass.class).collect();
13081308
Assert.assertEquals(data.size(), collected.length);
13091309
}
1310+
1311+
/**
1312+
* Test for SPARK-3647. This test needs to use the maven-built assembly to trigger the issue,
1313+
* since that's the only artifact where Guava classes have been relocated.
1314+
*/
1315+
@Test
1316+
public void testGuavaOptional() {
1317+
// Stop the context created in setUp() and start a local-cluster one, to force usage of the
1318+
// assembly.
1319+
sc.stop();
1320+
JavaSparkContext localCluster = new JavaSparkContext("local-cluster[1,1,512]", "JavaAPISuite");
1321+
try {
1322+
JavaRDD<Integer> rdd1 = localCluster.parallelize(Arrays.asList(1, 2, null), 3);
1323+
JavaRDD<Optional<Integer>> rdd2 = rdd1.map(
1324+
new Function<Integer, Optional<Integer>>() {
1325+
@Override
1326+
public Optional<Integer> call(Integer i) {
1327+
return Optional.fromNullable(i);
1328+
}
1329+
});
1330+
rdd2.collect();
1331+
} finally {
1332+
localCluster.stop();
1333+
}
1334+
}
1335+
13101336
}

0 commit comments

Comments
 (0)