Skip to content

Commit 06140a4

Browse files
committed
Merge remote-tracking branch 'apache/master' into SPARK-5991
2 parents 282ec8d + 5f7f3b9 commit 06140a4

File tree

8 files changed

+62
-46
lines changed

8 files changed

+62
-46
lines changed

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

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -1389,17 +1389,17 @@ class SparkContext(config: SparkConf) extends Logging with ExecutorAllocationCli
13891389
stopped = true
13901390
env.metricsSystem.report()
13911391
metadataCleaner.cancel()
1392-
env.actorSystem.stop(heartbeatReceiver)
13931392
cleaner.foreach(_.stop())
13941393
dagScheduler.stop()
13951394
dagScheduler = null
1395+
listenerBus.stop()
1396+
eventLogger.foreach(_.stop())
1397+
env.actorSystem.stop(heartbeatReceiver)
13961398
progressBar.foreach(_.stop())
13971399
taskScheduler = null
13981400
// TODO: Cache.stop()?
13991401
env.stop()
14001402
SparkEnv.set(null)
1401-
listenerBus.stop()
1402-
eventLogger.foreach(_.stop())
14031403
logInfo("Successfully stopped SparkContext")
14041404
SparkContext.clearActiveContext()
14051405
} else {

docs/mllib-decision-tree.md

Lines changed: 12 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -223,8 +223,9 @@ val testErr = labelAndPreds.filter(r => r._1 != r._2).count.toDouble / testData.
223223
println("Test Error = " + testErr)
224224
println("Learned classification tree model:\n" + model.toDebugString)
225225

226-
model.save("myModelPath")
227-
val sameModel = DecisionTreeModel.load("myModelPath")
226+
// Save and load model
227+
model.save(sc, "myModelPath")
228+
val sameModel = DecisionTreeModel.load(sc, "myModelPath")
228229
{% endhighlight %}
229230
</div>
230231

@@ -284,8 +285,9 @@ Double testErr =
284285
System.out.println("Test Error: " + testErr);
285286
System.out.println("Learned classification tree model:\n" + model.toDebugString());
286287

287-
model.save("myModelPath");
288-
DecisionTreeModel sameModel = DecisionTreeModel.load("myModelPath");
288+
// Save and load model
289+
model.save(sc.sc(), "myModelPath");
290+
DecisionTreeModel sameModel = DecisionTreeModel.load(sc.sc(), "myModelPath");
289291
{% endhighlight %}
290292
</div>
291293

@@ -362,8 +364,9 @@ val testMSE = labelsAndPredictions.map{ case(v, p) => math.pow((v - p), 2)}.mean
362364
println("Test Mean Squared Error = " + testMSE)
363365
println("Learned regression tree model:\n" + model.toDebugString)
364366

365-
model.save("myModelPath")
366-
val sameModel = DecisionTreeModel.load("myModelPath")
367+
// Save and load model
368+
model.save(sc, "myModelPath")
369+
val sameModel = DecisionTreeModel.load(sc, "myModelPath")
367370
{% endhighlight %}
368371
</div>
369372

@@ -429,8 +432,9 @@ Double testMSE =
429432
System.out.println("Test Mean Squared Error: " + testMSE);
430433
System.out.println("Learned regression tree model:\n" + model.toDebugString());
431434

432-
model.save("myModelPath");
433-
DecisionTreeModel sameModel = DecisionTreeModel.load("myModelPath");
435+
// Save and load model
436+
model.save(sc.sc(), "myModelPath");
437+
DecisionTreeModel sameModel = DecisionTreeModel.load(sc.sc(), "myModelPath");
434438
{% endhighlight %}
435439
</div>
436440

docs/mllib-ensembles.md

Lines changed: 24 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -129,8 +129,9 @@ val testErr = labelAndPreds.filter(r => r._1 != r._2).count.toDouble / testData.
129129
println("Test Error = " + testErr)
130130
println("Learned classification forest model:\n" + model.toDebugString)
131131

132-
model.save("myModelPath")
133-
val sameModel = RandomForestModel.load("myModelPath")
132+
// Save and load model
133+
model.save(sc, "myModelPath")
134+
val sameModel = RandomForestModel.load(sc, "myModelPath")
134135
{% endhighlight %}
135136
</div>
136137

@@ -193,8 +194,9 @@ Double testErr =
193194
System.out.println("Test Error: " + testErr);
194195
System.out.println("Learned classification forest model:\n" + model.toDebugString());
195196

196-
model.save("myModelPath");
197-
RandomForestModel sameModel = RandomForestModel.load("myModelPath");
197+
// Save and load model
198+
model.save(sc.sc(), "myModelPath");
199+
RandomForestModel sameModel = RandomForestModel.load(sc.sc(), "myModelPath");
198200
{% endhighlight %}
199201
</div>
200202

@@ -276,8 +278,9 @@ val testMSE = labelsAndPredictions.map{ case(v, p) => math.pow((v - p), 2)}.mean
276278
println("Test Mean Squared Error = " + testMSE)
277279
println("Learned regression forest model:\n" + model.toDebugString)
278280

279-
model.save("myModelPath")
280-
val sameModel = RandomForestModel.load("myModelPath")
281+
// Save and load model
282+
model.save(sc, "myModelPath")
283+
val sameModel = RandomForestModel.load(sc, "myModelPath")
281284
{% endhighlight %}
282285
</div>
283286

@@ -343,8 +346,9 @@ Double testMSE =
343346
System.out.println("Test Mean Squared Error: " + testMSE);
344347
System.out.println("Learned regression forest model:\n" + model.toDebugString());
345348

346-
model.save("myModelPath");
347-
RandomForestModel sameModel = RandomForestModel.load("myModelPath");
349+
// Save and load model
350+
model.save(sc.sc(), "myModelPath");
351+
RandomForestModel sameModel = RandomForestModel.load(sc.sc(), "myModelPath");
348352
{% endhighlight %}
349353
</div>
350354

@@ -504,8 +508,9 @@ val testErr = labelAndPreds.filter(r => r._1 != r._2).count.toDouble / testData.
504508
println("Test Error = " + testErr)
505509
println("Learned classification GBT model:\n" + model.toDebugString)
506510

507-
model.save("myModelPath")
508-
val sameModel = GradientBoostedTreesModel.load("myModelPath")
511+
// Save and load model
512+
model.save(sc, "myModelPath")
513+
val sameModel = GradientBoostedTreesModel.load(sc, "myModelPath")
509514
{% endhighlight %}
510515
</div>
511516

@@ -568,8 +573,9 @@ Double testErr =
568573
System.out.println("Test Error: " + testErr);
569574
System.out.println("Learned classification GBT model:\n" + model.toDebugString());
570575

571-
model.save("myModelPath");
572-
GradientBoostedTreesModel sameModel = GradientBoostedTreesModel.load("myModelPath");
576+
// Save and load model
577+
model.save(sc.sc(), "myModelPath");
578+
GradientBoostedTreesModel sameModel = GradientBoostedTreesModel.load(sc.sc(), "myModelPath");
573579
{% endhighlight %}
574580
</div>
575581

@@ -647,8 +653,9 @@ val testMSE = labelsAndPredictions.map{ case(v, p) => math.pow((v - p), 2)}.mean
647653
println("Test Mean Squared Error = " + testMSE)
648654
println("Learned regression GBT model:\n" + model.toDebugString)
649655

650-
model.save("myModelPath")
651-
val sameModel = GradientBoostedTreesModel.load("myModelPath")
656+
// Save and load model
657+
model.save(sc, "myModelPath")
658+
val sameModel = GradientBoostedTreesModel.load(sc, "myModelPath")
652659
{% endhighlight %}
653660
</div>
654661

@@ -717,8 +724,9 @@ Double testMSE =
717724
System.out.println("Test Mean Squared Error: " + testMSE);
718725
System.out.println("Learned regression GBT model:\n" + model.toDebugString());
719726

720-
model.save("myModelPath");
721-
GradientBoostedTreesModel sameModel = GradientBoostedTreesModel.load("myModelPath");
727+
// Save and load model
728+
model.save(sc.sc(), "myModelPath");
729+
GradientBoostedTreesModel sameModel = GradientBoostedTreesModel.load(sc.sc(), "myModelPath");
722730
{% endhighlight %}
723731
</div>
724732

docs/mllib-linear-methods.md

Lines changed: 12 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -223,8 +223,9 @@ val auROC = metrics.areaUnderROC()
223223

224224
println("Area under ROC = " + auROC)
225225

226-
model.save("myModelPath")
227-
val sameModel = SVMModel.load("myModelPath")
226+
// Save and load model
227+
model.save(sc, "myModelPath")
228+
val sameModel = SVMModel.load(sc, "myModelPath")
228229
{% endhighlight %}
229230

230231
The `SVMWithSGD.train()` method by default performs L2 regularization with the
@@ -308,8 +309,9 @@ public class SVMClassifier {
308309

309310
System.out.println("Area under ROC = " + auROC);
310311

311-
model.save("myModelPath");
312-
SVMModel sameModel = SVMModel.load("myModelPath");
312+
// Save and load model
313+
model.save(sc.sc(), "myModelPath");
314+
SVMModel sameModel = SVMModel.load(sc.sc(), "myModelPath");
313315
}
314316
}
315317
{% endhighlight %}
@@ -423,8 +425,9 @@ val valuesAndPreds = parsedData.map { point =>
423425
val MSE = valuesAndPreds.map{case(v, p) => math.pow((v - p), 2)}.mean()
424426
println("training Mean Squared Error = " + MSE)
425427

426-
model.save("myModelPath")
427-
val sameModel = LinearRegressionModel.load("myModelPath")
428+
// Save and load model
429+
model.save(sc, "myModelPath")
430+
val sameModel = LinearRegressionModel.load(sc, "myModelPath")
428431
{% endhighlight %}
429432

430433
[`RidgeRegressionWithSGD`](api/scala/index.html#org.apache.spark.mllib.regression.RidgeRegressionWithSGD)
@@ -496,8 +499,9 @@ public class LinearRegression {
496499
).rdd()).mean();
497500
System.out.println("training Mean Squared Error = " + MSE);
498501

499-
model.save("myModelPath");
500-
LinearRegressionModel sameModel = LinearRegressionModel.load("myModelPath");
502+
// Save and load model
503+
model.save(sc.sc(), "myModelPath");
504+
LinearRegressionModel sameModel = LinearRegressionModel.load(sc.sc(), "myModelPath");
501505
}
502506
}
503507
{% endhighlight %}

docs/mllib-naive-bayes.md

Lines changed: 6 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -56,8 +56,9 @@ val model = NaiveBayes.train(training, lambda = 1.0)
5656
val predictionAndLabel = test.map(p => (model.predict(p.features), p.label))
5757
val accuracy = 1.0 * predictionAndLabel.filter(x => x._1 == x._2).count() / test.count()
5858

59-
model.save("myModelPath")
60-
val sameModel = NaiveBayesModel.load("myModelPath")
59+
// Save and load model
60+
model.save(sc, "myModelPath")
61+
val sameModel = NaiveBayesModel.load(sc, "myModelPath")
6162
{% endhighlight %}
6263
</div>
6364

@@ -97,8 +98,9 @@ double accuracy = predictionAndLabel.filter(new Function<Tuple2<Double, Double>,
9798
}
9899
}).count() / (double) test.count();
99100

100-
model.save("myModelPath");
101-
NaiveBayesModel sameModel = NaiveBayesModel.load("myModelPath");
101+
// Save and load model
102+
model.save(sc.sc(), "myModelPath");
103+
NaiveBayesModel sameModel = NaiveBayesModel.load(sc.sc(), "myModelPath");
102104
{% endhighlight %}
103105
</div>
104106

docs/spark-standalone.md

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -222,8 +222,7 @@ SPARK_WORKER_OPTS supports the following system properties:
222222
<td>false</td>
223223
<td>
224224
Enable periodic cleanup of worker / application directories. Note that this only affects standalone
225-
mode, as YARN works differently. Applications directories are cleaned up regardless of whether
226-
the application is still running.
225+
mode, as YARN works differently. Only the directories of stopped applications are cleaned up.
227226
</td>
228227
</tr>
229228
<tr>

external/kafka/src/main/scala/org/apache/spark/streaming/kafka/KafkaUtils.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -512,7 +512,7 @@ object KafkaUtils {
512512
* @param topics Names of the topics to consume
513513
*/
514514
@Experimental
515-
def createDirectStream[K, V, KD <: Decoder[K], VD <: Decoder[V], R](
515+
def createDirectStream[K, V, KD <: Decoder[K], VD <: Decoder[V]](
516516
jssc: JavaStreamingContext,
517517
keyClass: Class[K],
518518
valueClass: Class[V],

yarn/src/main/scala/org/apache/spark/deploy/yarn/ApplicationMaster.scala

Lines changed: 3 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -68,8 +68,8 @@ private[spark] class ApplicationMaster(
6868
@volatile private var finalMsg: String = ""
6969
@volatile private var userClassThread: Thread = _
7070

71-
private var reporterThread: Thread = _
72-
private var allocator: YarnAllocator = _
71+
@volatile private var reporterThread: Thread = _
72+
@volatile private var allocator: YarnAllocator = _
7373

7474
// Fields used in client mode.
7575
private var actorSystem: ActorSystem = null
@@ -486,11 +486,10 @@ private[spark] class ApplicationMaster(
486486
case _: InterruptedException =>
487487
// Reporter thread can interrupt to stop user class
488488
case cause: Throwable =>
489+
logError("User class threw exception: " + cause.getMessage, cause)
489490
finish(FinalApplicationStatus.FAILED,
490491
ApplicationMaster.EXIT_EXCEPTION_USER_CLASS,
491492
"User class threw exception: " + cause.getMessage)
492-
// re-throw to get it logged
493-
throw cause
494493
}
495494
}
496495
}

0 commit comments

Comments
 (0)