aboutsummaryrefslogtreecommitdiff
path: root/examples/src/main/scala
diff options
context:
space:
mode:
authorDongjoon Hyun <dongjoon@apache.org>2016-05-04 14:31:36 -0700
committerAndrew Or <andrew@databricks.com>2016-05-04 14:31:36 -0700
commitcdce4e62a5674e2034e5d395578b1a60e3d8c435 (patch)
treec715f2555dad353683f82820962576f89b2db452 /examples/src/main/scala
parentcf2e9da612397233ae7bca0e9ce57309f16226b5 (diff)
downloadspark-cdce4e62a5674e2034e5d395578b1a60e3d8c435.tar.gz
spark-cdce4e62a5674e2034e5d395578b1a60e3d8c435.tar.bz2
spark-cdce4e62a5674e2034e5d395578b1a60e3d8c435.zip
[SPARK-15031][EXAMPLE] Use SparkSession in Scala/Python/Java example.
## What changes were proposed in this pull request? This PR aims to update Scala/Python/Java examples by replacing `SQLContext` with newly added `SparkSession`. - Use **SparkSession Builder Pattern** in 154(Scala 55, Java 52, Python 47) files. - Add `getConf` in Python SparkContext class: `python/pyspark/context.py` - Replace **SQLContext Singleton Pattern** with **SparkSession Singleton Pattern**: - `SqlNetworkWordCount.scala` - `JavaSqlNetworkWordCount.java` - `sql_network_wordcount.py` Now, `SQLContexts` are used only in R examples and the following two Python examples. The python examples are untouched in this PR since it already fails some unknown issue. - `simple_params_example.py` - `aft_survival_regression.py` ## How was this patch tested? Manual. Author: Dongjoon Hyun <dongjoon@apache.org> Closes #12809 from dongjoon-hyun/SPARK-15031.
Diffstat (limited to 'examples/src/main/scala')
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/AFTSurvivalRegressionExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/ALSExample.scala14
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/BinarizerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/BucketizerExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/ChiSqSelectorExample.scala14
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/CountVectorizerExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/DCTExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/DataFrameExample.scala14
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeClassificationExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeExample.scala18
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeRegressionExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/DeveloperApiExample.scala17
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/ElementwiseProductExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/EstimatorTransformerParamExample.scala13
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeClassifierExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeRegressorExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/IndexToStringExample.scala13
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/KMeansExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/LDAExample.scala13
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/LinearRegressionWithElasticNetExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionSummaryExample.scala13
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionWithElasticNetExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/MaxAbsScalerExample.scala14
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/MinMaxScalerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaCrossValidationExample.scala14
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaTrainValidationSplitExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/MultilayerPerceptronClassifierExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/NGramExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/NaiveBayesExample.scala13
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/NormalizerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/OneHotEncoderExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/OneVsRestExample.scala13
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/PCAExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/PipelineExample.scala13
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/PolynomialExpansionExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/QuantileDiscretizerExample.scala16
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/RFormulaExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/RandomForestClassifierExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/RandomForestRegressorExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/SQLTransformerExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/SimpleParamsExample.scala19
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/SimpleTextClassificationPipeline.scala15
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/StandardScalerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/StopWordsRemoverExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/StringIndexerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/TfIdfExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/TokenizerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/VectorAssemblerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/VectorIndexerExample.scala12
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/VectorSlicerExample.scala17
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/ml/Word2VecExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/mllib/LDAExample.scala6
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/mllib/RankingMetricsExample.scala11
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/mllib/RegressionMetricsExample.scala18
-rw-r--r--examples/src/main/scala/org/apache/spark/examples/streaming/SqlNetworkWordCount.scala21
55 files changed, 290 insertions, 410 deletions
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/AFTSurvivalRegressionExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/AFTSurvivalRegressionExample.scala
index 21f58ddf3c..3795af8309 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/AFTSurvivalRegressionExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/AFTSurvivalRegressionExample.scala
@@ -18,12 +18,11 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.regression.AFTSurvivalRegression
import org.apache.spark.mllib.linalg.Vectors
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
/**
* An example for AFTSurvivalRegression.
@@ -31,12 +30,10 @@ import org.apache.spark.sql.SQLContext
object AFTSurvivalRegressionExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("AFTSurvivalRegressionExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("AFTSurvivalRegressionExample").getOrCreate()
// $example on$
- val training = sqlContext.createDataFrame(Seq(
+ val training = spark.createDataFrame(Seq(
(1.218, 1.0, Vectors.dense(1.560, -0.605)),
(2.949, 0.0, Vectors.dense(0.346, 2.158)),
(3.627, 0.0, Vectors.dense(1.380, 0.231)),
@@ -56,7 +53,7 @@ object AFTSurvivalRegressionExample {
model.transform(training).show(false)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/ALSExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/ALSExample.scala
index a79e15c767..41750ca779 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/ALSExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/ALSExample.scala
@@ -18,12 +18,11 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.evaluation.RegressionEvaluator
import org.apache.spark.ml.recommendation.ALS
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
// $example on$
import org.apache.spark.sql.functions._
import org.apache.spark.sql.types.DoubleType
@@ -43,13 +42,11 @@ object ALSExample {
// $example off$
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("ALSExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("ALSExample").getOrCreate()
+ import spark.implicits._
// $example on$
- val ratings = sc.textFile("data/mllib/als/sample_movielens_ratings.txt")
+ val ratings = spark.read.text("data/mllib/als/sample_movielens_ratings.txt")
.map(Rating.parseRating)
.toDF()
val Array(training, test) = ratings.randomSplit(Array(0.8, 0.2))
@@ -75,7 +72,8 @@ object ALSExample {
val rmse = evaluator.evaluate(predictions)
println(s"Root-mean-square error = $rmse")
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/BinarizerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/BinarizerExample.scala
index 2ed8101c13..93c153f923 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/BinarizerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/BinarizerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.Binarizer
// $example off$
-import org.apache.spark.sql.{DataFrame, SQLContext}
+import org.apache.spark.sql.{DataFrame, SparkSession}
object BinarizerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("BinarizerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("BinarizerExample").getOrCreate()
// $example on$
val data = Array((0, 0.1), (1, 0.8), (2, 0.2))
- val dataFrame: DataFrame = sqlContext.createDataFrame(data).toDF("label", "feature")
+ val dataFrame: DataFrame = spark.createDataFrame(data).toDF("label", "feature")
val binarizer: Binarizer = new Binarizer()
.setInputCol("feature")
@@ -42,7 +39,8 @@ object BinarizerExample {
val binarizedFeatures = binarizedDataFrame.select("binarized_feature")
binarizedFeatures.collect().foreach(println)
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/BucketizerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/BucketizerExample.scala
index 6f6236a2b0..779ad33dbd 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/BucketizerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/BucketizerExample.scala
@@ -18,23 +18,20 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.Bucketizer
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object BucketizerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("BucketizerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("BucketizerExample").getOrCreate()
// $example on$
val splits = Array(Double.NegativeInfinity, -0.5, 0.0, 0.5, Double.PositiveInfinity)
val data = Array(-0.5, -0.3, 0.0, 0.2)
- val dataFrame = sqlContext.createDataFrame(data.map(Tuple1.apply)).toDF("features")
+ val dataFrame = spark.createDataFrame(data.map(Tuple1.apply)).toDF("features")
val bucketizer = new Bucketizer()
.setInputCol("features")
@@ -45,7 +42,7 @@ object BucketizerExample {
val bucketedData = bucketizer.transform(dataFrame)
bucketedData.show()
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/ChiSqSelectorExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/ChiSqSelectorExample.scala
index 2be61537e6..84ca1f0b56 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/ChiSqSelectorExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/ChiSqSelectorExample.scala
@@ -18,20 +18,16 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.ChiSqSelector
import org.apache.spark.mllib.linalg.Vectors
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object ChiSqSelectorExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("ChiSqSelectorExample")
- val sc = new SparkContext(conf)
-
- val sqlContext = SQLContext.getOrCreate(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("ChiSqSelectorExample").getOrCreate()
+ import spark.implicits._
// $example on$
val data = Seq(
@@ -40,7 +36,7 @@ object ChiSqSelectorExample {
(9, Vectors.dense(1.0, 0.0, 15.0, 0.1), 0.0)
)
- val df = sc.parallelize(data).toDF("id", "features", "clicked")
+ val df = spark.createDataset(data).toDF("id", "features", "clicked")
val selector = new ChiSqSelector()
.setNumTopFeatures(1)
@@ -51,7 +47,7 @@ object ChiSqSelectorExample {
val result = selector.fit(df).transform(df)
result.show()
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/CountVectorizerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/CountVectorizerExample.scala
index 7d07fc7dd1..9ab43a48bf 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/CountVectorizerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/CountVectorizerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.{CountVectorizer, CountVectorizerModel}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object CountVectorizerExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("CounterVectorizerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("CounterVectorizerExample").getOrCreate()
// $example on$
- val df = sqlContext.createDataFrame(Seq(
+ val df = spark.createDataFrame(Seq(
(0, Array("a", "b", "c")),
(1, Array("a", "b", "b", "c", "a"))
)).toDF("id", "words")
@@ -51,6 +48,8 @@ object CountVectorizerExample {
cvModel.transform(df).select("features").show()
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/DCTExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/DCTExample.scala
index dc26b55a76..b415333c71 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/DCTExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/DCTExample.scala
@@ -18,18 +18,15 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.DCT
import org.apache.spark.mllib.linalg.Vectors
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object DCTExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("DCTExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("DCTExample").getOrCreate()
// $example on$
val data = Seq(
@@ -37,7 +34,7 @@ object DCTExample {
Vectors.dense(-1.0, 2.0, 4.0, -7.0),
Vectors.dense(14.0, -2.0, -5.0, 1.0))
- val df = sqlContext.createDataFrame(data.map(Tuple1.apply)).toDF("features")
+ val df = spark.createDataFrame(data.map(Tuple1.apply)).toDF("features")
val dct = new DCT()
.setInputCol("features")
@@ -47,7 +44,8 @@ object DCTExample {
val dctDf = dct.transform(df)
dctDf.select("featuresDCT").show(3)
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/DataFrameExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/DataFrameExample.scala
index 7e608a2812..2f892f8d72 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/DataFrameExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/DataFrameExample.scala
@@ -23,11 +23,10 @@ import java.io.File
import com.google.common.io.Files
import scopt.OptionParser
-import org.apache.spark.{SparkConf, SparkContext}
import org.apache.spark.examples.mllib.AbstractParams
import org.apache.spark.mllib.linalg.Vector
import org.apache.spark.mllib.stat.MultivariateOnlineSummarizer
-import org.apache.spark.sql.{DataFrame, Row, SQLContext}
+import org.apache.spark.sql.{DataFrame, Row, SparkSession}
/**
* An example of how to use [[org.apache.spark.sql.DataFrame]] for ML. Run with
@@ -62,14 +61,11 @@ object DataFrameExample {
}
def run(params: Params) {
-
- val conf = new SparkConf().setAppName(s"DataFrameExample with $params")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName(s"DataFrameExample with $params").getOrCreate()
// Load input data
println(s"Loading LIBSVM file with UDT from ${params.input}.")
- val df: DataFrame = sqlContext.read.format("libsvm").load(params.input).cache()
+ val df: DataFrame = spark.read.format("libsvm").load(params.input).cache()
println("Schema from LIBSVM:")
df.printSchema()
println(s"Loaded training data as a DataFrame with ${df.count()} records.")
@@ -94,11 +90,11 @@ object DataFrameExample {
// Load the records back.
println(s"Loading Parquet file with UDT from $outputDir.")
- val newDF = sqlContext.read.parquet(outputDir)
+ val newDF = spark.read.parquet(outputDir)
println(s"Schema from Parquet:")
newDF.printSchema()
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeClassificationExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeClassificationExample.scala
index 224d8da5f0..a0a2e1fb33 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeClassificationExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeClassificationExample.scala
@@ -18,7 +18,6 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.classification.DecisionTreeClassificationModel
@@ -26,16 +25,14 @@ import org.apache.spark.ml.classification.DecisionTreeClassifier
import org.apache.spark.ml.evaluation.MulticlassClassificationEvaluator
import org.apache.spark.ml.feature.{IndexToString, StringIndexer, VectorIndexer}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object DecisionTreeClassificationExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("DecisionTreeClassificationExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("DecisionTreeClassificationExample").getOrCreate()
// $example on$
// Load the data stored in LIBSVM format as a DataFrame.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Index labels, adding metadata to the label column.
// Fit on whole dataset to include all labels in index.
@@ -88,6 +85,8 @@ object DecisionTreeClassificationExample {
val treeModel = model.stages(2).asInstanceOf[DecisionTreeClassificationModel]
println("Learned classification tree model:\n" + treeModel.toDebugString)
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeExample.scala
index d2560cc00b..cea1d801aa 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeExample.scala
@@ -33,7 +33,7 @@ import org.apache.spark.ml.util.MetadataUtils
import org.apache.spark.mllib.evaluation.{MulticlassMetrics, RegressionMetrics}
import org.apache.spark.mllib.linalg.Vector
import org.apache.spark.mllib.util.MLUtils
-import org.apache.spark.sql.{DataFrame, SQLContext}
+import org.apache.spark.sql.{DataFrame, SparkSession}
/**
* An example runner for decision trees. Run with
@@ -134,18 +134,18 @@ object DecisionTreeExample {
/** Load a dataset from the given path, using the given format */
private[ml] def loadData(
- sqlContext: SQLContext,
+ spark: SparkSession,
path: String,
format: String,
expectedNumFeatures: Option[Int] = None): DataFrame = {
- import sqlContext.implicits._
+ import spark.implicits._
format match {
- case "dense" => MLUtils.loadLabeledPoints(sqlContext.sparkContext, path).toDF()
+ case "dense" => MLUtils.loadLabeledPoints(spark.sparkContext, path).toDF()
case "libsvm" => expectedNumFeatures match {
- case Some(numFeatures) => sqlContext.read.option("numFeatures", numFeatures.toString)
+ case Some(numFeatures) => spark.read.option("numFeatures", numFeatures.toString)
.format("libsvm").load(path)
- case None => sqlContext.read.format("libsvm").load(path)
+ case None => spark.read.format("libsvm").load(path)
}
case _ => throw new IllegalArgumentException(s"Bad data format: $format")
}
@@ -167,17 +167,17 @@ object DecisionTreeExample {
testInput: String,
algo: String,
fracTest: Double): (DataFrame, DataFrame) = {
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.getOrCreate()
// Load training data
- val origExamples: DataFrame = loadData(sqlContext, input, dataFormat)
+ val origExamples: DataFrame = loadData(spark, input, dataFormat)
// Load or create test set
val dataframes: Array[DataFrame] = if (testInput != "") {
// Load testInput.
val numFeatures = origExamples.first().getAs[Vector](1).size
val origTestExamples: DataFrame =
- loadData(sqlContext, testInput, dataFormat, Some(numFeatures))
+ loadData(spark, testInput, dataFormat, Some(numFeatures))
Array(origExamples, origTestExamples)
} else {
// Split input into training, test.
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeRegressionExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeRegressionExample.scala
index ad32e5635a..26b52d0489 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeRegressionExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/DecisionTreeRegressionExample.scala
@@ -18,7 +18,6 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.evaluation.RegressionEvaluator
@@ -26,17 +25,15 @@ import org.apache.spark.ml.feature.VectorIndexer
import org.apache.spark.ml.regression.DecisionTreeRegressionModel
import org.apache.spark.ml.regression.DecisionTreeRegressor
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object DecisionTreeRegressionExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("DecisionTreeRegressionExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("DecisionTreeRegressionExample").getOrCreate()
// $example on$
// Load the data stored in LIBSVM format as a DataFrame.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Automatically identify categorical features, and index them.
// Here, we treat features with > 4 distinct values as continuous.
@@ -78,6 +75,8 @@ object DecisionTreeRegressionExample {
val treeModel = model.stages(1).asInstanceOf[DecisionTreeRegressionModel]
println("Learned regression tree model:\n" + treeModel.toDebugString)
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/DeveloperApiExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/DeveloperApiExample.scala
index 8d127f9b35..2aa1ab1ec8 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/DeveloperApiExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/DeveloperApiExample.scala
@@ -18,13 +18,12 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
import org.apache.spark.ml.classification.{ClassificationModel, Classifier, ClassifierParams}
import org.apache.spark.ml.param.{IntParam, ParamMap}
import org.apache.spark.ml.util.Identifiable
import org.apache.spark.mllib.linalg.{BLAS, Vector, Vectors}
import org.apache.spark.mllib.regression.LabeledPoint
-import org.apache.spark.sql.{DataFrame, Dataset, Row, SQLContext}
+import org.apache.spark.sql.{Dataset, Row, SparkSession}
/**
* A simple example demonstrating how to write your own learning algorithm using Estimator,
@@ -38,13 +37,11 @@ import org.apache.spark.sql.{DataFrame, Dataset, Row, SQLContext}
object DeveloperApiExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("DeveloperApiExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("DeveloperApiExample").getOrCreate()
+ import spark.implicits._
// Prepare training data.
- val training = sc.parallelize(Seq(
+ val training = spark.createDataFrame(Seq(
LabeledPoint(1.0, Vectors.dense(0.0, 1.1, 0.1)),
LabeledPoint(0.0, Vectors.dense(2.0, 1.0, -1.0)),
LabeledPoint(0.0, Vectors.dense(2.0, 1.3, 1.0)),
@@ -62,13 +59,13 @@ object DeveloperApiExample {
val model = lr.fit(training.toDF())
// Prepare test data.
- val test = sc.parallelize(Seq(
+ val test = spark.createDataFrame(Seq(
LabeledPoint(1.0, Vectors.dense(-1.0, 1.5, 1.3)),
LabeledPoint(0.0, Vectors.dense(3.0, 2.0, -0.1)),
LabeledPoint(1.0, Vectors.dense(0.0, 2.2, -1.5))))
// Make predictions on test data.
- val sumPredictions: Double = model.transform(test.toDF())
+ val sumPredictions: Double = model.transform(test)
.select("features", "label", "prediction")
.collect()
.map { case Row(features: Vector, label: Double, prediction: Double) =>
@@ -77,7 +74,7 @@ object DeveloperApiExample {
assert(sumPredictions == 0.0,
"MyLogisticRegression predicted something other than 0, even though all coefficients are 0!")
- sc.stop()
+ spark.stop()
}
}
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/ElementwiseProductExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/ElementwiseProductExample.scala
index 629d322c43..f289c28df9 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/ElementwiseProductExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/ElementwiseProductExample.scala
@@ -18,22 +18,19 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.ElementwiseProduct
import org.apache.spark.mllib.linalg.Vectors
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object ElementwiseProductExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("ElementwiseProductExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("ElementwiseProductExample").getOrCreate()
// $example on$
// Create some vector data; also works for sparse vectors
- val dataFrame = sqlContext.createDataFrame(Seq(
+ val dataFrame = spark.createDataFrame(Seq(
("a", Vectors.dense(1.0, 2.0, 3.0)),
("b", Vectors.dense(4.0, 5.0, 6.0)))).toDF("id", "vector")
@@ -46,7 +43,8 @@ object ElementwiseProductExample {
// Batch transform the vectors to create new column:
transformer.transform(dataFrame).show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/EstimatorTransformerParamExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/EstimatorTransformerParamExample.scala
index 65e3c365ab..91076ccbc1 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/EstimatorTransformerParamExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/EstimatorTransformerParamExample.scala
@@ -18,25 +18,22 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.classification.LogisticRegression
import org.apache.spark.ml.param.ParamMap
import org.apache.spark.mllib.linalg.{Vector, Vectors}
import org.apache.spark.sql.Row
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object EstimatorTransformerParamExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("EstimatorTransformerParamExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("EstimatorTransformerParamExample").getOrCreate()
// $example on$
// Prepare training data from a list of (label, features) tuples.
- val training = sqlContext.createDataFrame(Seq(
+ val training = spark.createDataFrame(Seq(
(1.0, Vectors.dense(0.0, 1.1, 0.1)),
(0.0, Vectors.dense(2.0, 1.0, -1.0)),
(0.0, Vectors.dense(2.0, 1.3, 1.0)),
@@ -76,7 +73,7 @@ object EstimatorTransformerParamExample {
println("Model 2 was fit using parameters: " + model2.parent.extractParamMap)
// Prepare test data.
- val test = sqlContext.createDataFrame(Seq(
+ val test = spark.createDataFrame(Seq(
(1.0, Vectors.dense(-1.0, 1.5, 1.3)),
(0.0, Vectors.dense(3.0, 2.0, -0.1)),
(1.0, Vectors.dense(0.0, 2.2, -1.5))
@@ -94,7 +91,7 @@ object EstimatorTransformerParamExample {
}
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeClassifierExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeClassifierExample.scala
index cd62a80382..412c54db7d 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeClassifierExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeClassifierExample.scala
@@ -18,24 +18,21 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.classification.{GBTClassificationModel, GBTClassifier}
import org.apache.spark.ml.evaluation.MulticlassClassificationEvaluator
import org.apache.spark.ml.feature.{IndexToString, StringIndexer, VectorIndexer}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object GradientBoostedTreeClassifierExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("GradientBoostedTreeClassifierExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("GradientBoostedTreeClassifierExample").getOrCreate()
// $example on$
// Load and parse the data file, converting it to a DataFrame.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Index labels, adding metadata to the label column.
// Fit on whole dataset to include all labels in index.
@@ -91,7 +88,7 @@ object GradientBoostedTreeClassifierExample {
println("Learned classification GBT model:\n" + gbtModel.toDebugString)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeRegressorExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeRegressorExample.scala
index b8cf9629bb..fd43553cc6 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeRegressorExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/GradientBoostedTreeRegressorExample.scala
@@ -18,24 +18,21 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.evaluation.RegressionEvaluator
import org.apache.spark.ml.feature.VectorIndexer
import org.apache.spark.ml.regression.{GBTRegressionModel, GBTRegressor}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object GradientBoostedTreeRegressorExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("GradientBoostedTreeRegressorExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("GradientBoostedTreeRegressorExample").getOrCreate()
// $example on$
// Load and parse the data file, converting it to a DataFrame.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Automatically identify categorical features, and index them.
// Set maxCategories so features with > 4 distinct values are treated as continuous.
@@ -79,7 +76,7 @@ object GradientBoostedTreeRegressorExample {
println("Learned regression GBT model:\n" + gbtModel.toDebugString)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/IndexToStringExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/IndexToStringExample.scala
index 4cea09ba12..d873618726 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/IndexToStringExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/IndexToStringExample.scala
@@ -18,21 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.{IndexToString, StringIndexer}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object IndexToStringExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("IndexToStringExample")
- val sc = new SparkContext(conf)
-
- val sqlContext = SQLContext.getOrCreate(sc)
+ val spark = SparkSession.builder.appName("IndexToStringExample").getOrCreate()
// $example on$
- val df = sqlContext.createDataFrame(Seq(
+ val df = spark.createDataFrame(Seq(
(0, "a"),
(1, "b"),
(2, "c"),
@@ -54,7 +50,8 @@ object IndexToStringExample {
val converted = converter.transform(indexed)
converted.select("id", "originalCategory").show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/KMeansExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/KMeansExample.scala
index 7af011571f..d2573fad35 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/KMeansExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/KMeansExample.scala
@@ -19,11 +19,10 @@ package org.apache.spark.examples.ml
// scalastyle:off println
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.clustering.KMeans
import org.apache.spark.mllib.linalg.Vectors
-import org.apache.spark.sql.{DataFrame, SQLContext}
+import org.apache.spark.sql.{DataFrame, SparkSession}
// $example off$
/**
@@ -37,13 +36,11 @@ object KMeansExample {
def main(args: Array[String]): Unit = {
// Creates a Spark context and a SQL context
- val conf = new SparkConf().setAppName(s"${this.getClass.getSimpleName}")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName(s"${this.getClass.getSimpleName}").getOrCreate()
// $example on$
// Crates a DataFrame
- val dataset: DataFrame = sqlContext.createDataFrame(Seq(
+ val dataset: DataFrame = spark.createDataFrame(Seq(
(1, Vectors.dense(0.0, 0.0, 0.0)),
(2, Vectors.dense(0.1, 0.1, 0.1)),
(3, Vectors.dense(0.2, 0.2, 0.2)),
@@ -64,7 +61,7 @@ object KMeansExample {
model.clusterCenters.foreach(println)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/LDAExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/LDAExample.scala
index f9ddac7709..c23adee1a3 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/LDAExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/LDAExample.scala
@@ -18,11 +18,10 @@
package org.apache.spark.examples.ml
// scalastyle:off println
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.clustering.LDA
import org.apache.spark.mllib.linalg.{Vectors, VectorUDT}
-import org.apache.spark.sql.{Row, SQLContext}
+import org.apache.spark.sql.{Row, SparkSession}
import org.apache.spark.sql.types.{StructField, StructType}
// $example off$
@@ -41,16 +40,14 @@ object LDAExample {
val input = "data/mllib/sample_lda_data.txt"
// Creates a Spark context and a SQL context
- val conf = new SparkConf().setAppName(s"${this.getClass.getSimpleName}")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName(s"${this.getClass.getSimpleName}").getOrCreate()
// $example on$
// Loads data
- val rowRDD = sc.textFile(input).filter(_.nonEmpty)
+ val rowRDD = spark.read.text(input).rdd.filter(_.nonEmpty)
.map(_.split(" ").map(_.toDouble)).map(Vectors.dense).map(Row(_))
val schema = StructType(Array(StructField(FEATURES_COL, new VectorUDT, false)))
- val dataset = sqlContext.createDataFrame(rowRDD, schema)
+ val dataset = spark.createDataFrame(rowRDD, schema)
// Trains a LDA model
val lda = new LDA()
@@ -71,7 +68,7 @@ object LDAExample {
transformed.show(false)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/LinearRegressionWithElasticNetExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/LinearRegressionWithElasticNetExample.scala
index f68aef7082..cb6e2492f5 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/LinearRegressionWithElasticNetExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/LinearRegressionWithElasticNetExample.scala
@@ -18,22 +18,19 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.regression.LinearRegression
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object LinearRegressionWithElasticNetExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("LinearRegressionWithElasticNetExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("LinearRegressionWithElasticNetExample").getOrCreate()
// $example on$
// Load training data
- val training = sqlContext.read.format("libsvm")
+ val training = spark.read.format("libsvm")
.load("data/mllib/sample_linear_regression_data.txt")
val lr = new LinearRegression()
@@ -56,7 +53,7 @@ object LinearRegressionWithElasticNetExample {
println(s"r2: ${trainingSummary.r2}")
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionSummaryExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionSummaryExample.scala
index 89c5edf1ac..50670d7b38 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionSummaryExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionSummaryExample.scala
@@ -18,23 +18,20 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.classification.{BinaryLogisticRegressionSummary, LogisticRegression}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.functions.max
object LogisticRegressionSummaryExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("LogisticRegressionSummaryExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("LogisticRegressionSummaryExample").getOrCreate()
+ import spark.implicits._
// Load training data
- val training = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val training = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
val lr = new LogisticRegression()
.setMaxIter(10)
@@ -71,7 +68,7 @@ object LogisticRegressionSummaryExample {
lrModel.setThreshold(bestThreshold)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionWithElasticNetExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionWithElasticNetExample.scala
index 6e27571f1d..fcba813d5b 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionWithElasticNetExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/LogisticRegressionWithElasticNetExample.scala
@@ -18,22 +18,20 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.classification.LogisticRegression
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object LogisticRegressionWithElasticNetExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("LogisticRegressionWithElasticNetExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession
+ .builder.appName("LogisticRegressionWithElasticNetExample").getOrCreate()
// $example on$
// Load training data
- val training = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val training = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
val lr = new LogisticRegression()
.setMaxIter(10)
@@ -47,7 +45,7 @@ object LogisticRegressionWithElasticNetExample {
println(s"Coefficients: ${lrModel.coefficients} Intercept: ${lrModel.intercept}")
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/MaxAbsScalerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/MaxAbsScalerExample.scala
index aafb5efd69..896d8fadbe 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/MaxAbsScalerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/MaxAbsScalerExample.scala
@@ -15,23 +15,19 @@
* limitations under the License.
*/
-// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.MaxAbsScaler
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object MaxAbsScalerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("MaxAbsScalerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("MaxAbsScalerExample").getOrCreate()
// $example on$
- val dataFrame = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val dataFrame = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
val scaler = new MaxAbsScaler()
.setInputCol("features")
.setOutputCol("scaledFeatures")
@@ -43,7 +39,7 @@ object MaxAbsScalerExample {
val scaledData = scalerModel.transform(dataFrame)
scaledData.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
-// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/MinMaxScalerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/MinMaxScalerExample.scala
index 9a03f69f5a..bcdca0fa04 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/MinMaxScalerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/MinMaxScalerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.MinMaxScaler
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object MinMaxScalerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("MinMaxScalerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("MinMaxScalerExample").getOrCreate()
// $example on$
- val dataFrame = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val dataFrame = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
val scaler = new MinMaxScaler()
.setInputCol("features")
@@ -44,7 +41,8 @@ object MinMaxScalerExample {
val scaledData = scalerModel.transform(dataFrame)
scaledData.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaCrossValidationExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaCrossValidationExample.scala
index d1441b5497..5fb3536060 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaCrossValidationExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaCrossValidationExample.scala
@@ -18,7 +18,6 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.classification.LogisticRegression
@@ -28,7 +27,7 @@ import org.apache.spark.ml.tuning.{CrossValidator, ParamGridBuilder}
import org.apache.spark.mllib.linalg.Vector
import org.apache.spark.sql.Row
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
/**
* A simple example demonstrating model selection using CrossValidator.
@@ -42,13 +41,12 @@ import org.apache.spark.sql.SQLContext
object ModelSelectionViaCrossValidationExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("ModelSelectionViaCrossValidationExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession
+ .builder.appName("ModelSelectionViaCrossValidationExample").getOrCreate()
// $example on$
// Prepare training data from a list of (id, text, label) tuples.
- val training = sqlContext.createDataFrame(Seq(
+ val training = spark.createDataFrame(Seq(
(0L, "a b c d e spark", 1.0),
(1L, "b d", 0.0),
(2L, "spark f g h", 1.0),
@@ -98,7 +96,7 @@ object ModelSelectionViaCrossValidationExample {
val cvModel = cv.fit(training)
// Prepare test documents, which are unlabeled (id, text) tuples.
- val test = sqlContext.createDataFrame(Seq(
+ val test = spark.createDataFrame(Seq(
(4L, "spark i j k"),
(5L, "l m n"),
(6L, "mapreduce spark"),
@@ -114,7 +112,7 @@ object ModelSelectionViaCrossValidationExample {
}
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaTrainValidationSplitExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaTrainValidationSplitExample.scala
index fcad17a817..6bc082982c 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaTrainValidationSplitExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/ModelSelectionViaTrainValidationSplitExample.scala
@@ -17,13 +17,12 @@
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.evaluation.RegressionEvaluator
import org.apache.spark.ml.regression.LinearRegression
import org.apache.spark.ml.tuning.{ParamGridBuilder, TrainValidationSplit}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
/**
* A simple example demonstrating model selection using TrainValidationSplit.
@@ -36,13 +35,12 @@ import org.apache.spark.sql.SQLContext
object ModelSelectionViaTrainValidationSplitExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("ModelSelectionViaTrainValidationSplitExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession
+ .builder.appName("ModelSelectionViaTrainValidationSplitExample").getOrCreate()
// $example on$
// Prepare training and test data.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_linear_regression_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_linear_regression_data.txt")
val Array(training, test) = data.randomSplit(Array(0.9, 0.1), seed = 12345)
val lr = new LinearRegression()
@@ -75,6 +73,6 @@ object ModelSelectionViaTrainValidationSplitExample {
.show()
// $example off$
- sc.stop()
+ spark.stop()
}
}
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/MultilayerPerceptronClassifierExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/MultilayerPerceptronClassifierExample.scala
index d7d1e82f6f..a11fe1b4b2 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/MultilayerPerceptronClassifierExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/MultilayerPerceptronClassifierExample.scala
@@ -18,12 +18,11 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.classification.MultilayerPerceptronClassifier
import org.apache.spark.ml.evaluation.MulticlassClassificationEvaluator
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
/**
* An example for Multilayer Perceptron Classification.
@@ -31,13 +30,11 @@ import org.apache.spark.sql.SQLContext
object MultilayerPerceptronClassifierExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("MultilayerPerceptronClassifierExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("MultilayerPerceptronClassifierExample").getOrCreate()
// $example on$
// Load the data stored in LIBSVM format as a DataFrame.
- val data = sqlContext.read.format("libsvm")
+ val data = spark.read.format("libsvm")
.load("data/mllib/sample_multiclass_classification_data.txt")
// Split the data into train and test
val splits = data.randomSplit(Array(0.6, 0.4), seed = 1234L)
@@ -63,7 +60,7 @@ object MultilayerPerceptronClassifierExample {
println("Precision:" + evaluator.evaluate(predictionAndLabels))
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/NGramExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/NGramExample.scala
index 77b913aaa3..1b71a39890 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/NGramExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/NGramExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.NGram
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object NGramExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("NGramExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("NGramExample").getOrCreate()
// $example on$
- val wordDataFrame = sqlContext.createDataFrame(Seq(
+ val wordDataFrame = spark.createDataFrame(Seq(
(0, Array("Hi", "I", "heard", "about", "Spark")),
(1, Array("I", "wish", "Java", "could", "use", "case", "classes")),
(2, Array("Logistic", "regression", "models", "are", "neat"))
@@ -41,7 +38,8 @@ object NGramExample {
val ngramDataFrame = ngram.transform(wordDataFrame)
ngramDataFrame.take(3).map(_.getAs[Stream[String]]("ngrams").toList).foreach(println)
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/NaiveBayesExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/NaiveBayesExample.scala
index 5ea1270c97..8d54555cd3 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/NaiveBayesExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/NaiveBayesExample.scala
@@ -18,21 +18,18 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
-import org.apache.spark.ml.classification.{NaiveBayes}
+import org.apache.spark.ml.classification.NaiveBayes
import org.apache.spark.ml.evaluation.MulticlassClassificationEvaluator
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object NaiveBayesExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("NaiveBayesExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("NaiveBayesExample").getOrCreate()
// $example on$
// Load the data stored in LIBSVM format as a DataFrame.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Split the data into training and test sets (30% held out for testing)
val Array(trainingData, testData) = data.randomSplit(Array(0.7, 0.3))
@@ -53,6 +50,8 @@ object NaiveBayesExample {
val precision = evaluator.evaluate(predictions)
println("Precision:" + precision)
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/NormalizerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/NormalizerExample.scala
index 6b33c16c74..4622d69ef9 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/NormalizerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/NormalizerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.Normalizer
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object NormalizerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("NormalizerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("NormalizerExample").getOrCreate()
// $example on$
- val dataFrame = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val dataFrame = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Normalize each Vector using $L^1$ norm.
val normalizer = new Normalizer()
@@ -46,7 +43,8 @@ object NormalizerExample {
val lInfNormData = normalizer.transform(dataFrame, normalizer.p -> Double.PositiveInfinity)
lInfNormData.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/OneHotEncoderExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/OneHotEncoderExample.scala
index cb9fe65a85..338436100c 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/OneHotEncoderExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/OneHotEncoderExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.{OneHotEncoder, StringIndexer}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object OneHotEncoderExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("OneHotEncoderExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("OneHotEncoderExample").getOrCreate()
// $example on$
- val df = sqlContext.createDataFrame(Seq(
+ val df = spark.createDataFrame(Seq(
(0, "a"),
(1, "b"),
(2, "c"),
@@ -52,7 +49,8 @@ object OneHotEncoderExample {
val encoded = encoder.transform(indexed)
encoded.select("id", "categoryVec").show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/OneVsRestExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/OneVsRestExample.scala
index 0b5d31c0ff..e2351c682d 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/OneVsRestExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/OneVsRestExample.scala
@@ -22,7 +22,6 @@ import java.util.concurrent.TimeUnit.{NANOSECONDS => NANO}
import scopt.OptionParser
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.examples.mllib.AbstractParams
import org.apache.spark.ml.classification.{LogisticRegression, OneVsRest}
@@ -31,7 +30,7 @@ import org.apache.spark.mllib.evaluation.MulticlassMetrics
import org.apache.spark.mllib.linalg.Vector
import org.apache.spark.sql.DataFrame
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
/**
* An example runner for Multiclass to Binary Reduction with One Vs Rest.
@@ -110,18 +109,16 @@ object OneVsRestExample {
}
private def run(params: Params) {
- val conf = new SparkConf().setAppName(s"OneVsRestExample with $params")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName(s"OneVsRestExample with $params").getOrCreate()
// $example on$
- val inputData = sqlContext.read.format("libsvm").load(params.input)
+ val inputData = spark.read.format("libsvm").load(params.input)
// compute the train/test split: if testInput is not provided use part of input.
val data = params.testInput match {
case Some(t) =>
// compute the number of features in the training set.
val numFeatures = inputData.first().getAs[Vector](1).size
- val testData = sqlContext.read.option("numFeatures", numFeatures.toString)
+ val testData = spark.read.option("numFeatures", numFeatures.toString)
.format("libsvm").load(t)
Array[DataFrame](inputData, testData)
case None =>
@@ -175,7 +172,7 @@ object OneVsRestExample {
println(fprs.map {case (label, fpr) => label + "\t" + fpr}.mkString("\n"))
// $example off$
- sc.stop()
+ spark.stop()
}
private def time[R](block: => R): (Long, R) = {
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/PCAExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/PCAExample.scala
index 535652ec6c..14394d5624 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/PCAExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/PCAExample.scala
@@ -18,18 +18,15 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.PCA
import org.apache.spark.mllib.linalg.Vectors
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object PCAExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("PCAExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("PCAExample").getOrCreate()
// $example on$
val data = Array(
@@ -37,7 +34,7 @@ object PCAExample {
Vectors.dense(2.0, 0.0, 3.0, 4.0, 5.0),
Vectors.dense(4.0, 0.0, 0.0, 6.0, 7.0)
)
- val df = sqlContext.createDataFrame(data.map(Tuple1.apply)).toDF("features")
+ val df = spark.createDataFrame(data.map(Tuple1.apply)).toDF("features")
val pca = new PCA()
.setInputCol("features")
.setOutputCol("pcaFeatures")
@@ -47,7 +44,8 @@ object PCAExample {
val result = pcaDF.select("pcaFeatures")
result.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/PipelineExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/PipelineExample.scala
index 6c29063626..61b34aebd9 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/PipelineExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/PipelineExample.scala
@@ -18,7 +18,6 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.{Pipeline, PipelineModel}
import org.apache.spark.ml.classification.LogisticRegression
@@ -26,18 +25,16 @@ import org.apache.spark.ml.feature.{HashingTF, Tokenizer}
import org.apache.spark.mllib.linalg.Vector
import org.apache.spark.sql.Row
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object PipelineExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("PipelineExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("PipelineExample").getOrCreate()
// $example on$
// Prepare training documents from a list of (id, text, label) tuples.
- val training = sqlContext.createDataFrame(Seq(
+ val training = spark.createDataFrame(Seq(
(0L, "a b c d e spark", 1.0),
(1L, "b d", 0.0),
(2L, "spark f g h", 1.0),
@@ -71,7 +68,7 @@ object PipelineExample {
val sameModel = PipelineModel.load("/tmp/spark-logistic-regression-model")
// Prepare test documents, which are unlabeled (id, text) tuples.
- val test = sqlContext.createDataFrame(Seq(
+ val test = spark.createDataFrame(Seq(
(4L, "spark i j k"),
(5L, "l m n"),
(6L, "mapreduce spark"),
@@ -87,7 +84,7 @@ object PipelineExample {
}
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/PolynomialExpansionExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/PolynomialExpansionExample.scala
index 3014008ea0..4d8c672a55 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/PolynomialExpansionExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/PolynomialExpansionExample.scala
@@ -18,18 +18,15 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.PolynomialExpansion
import org.apache.spark.mllib.linalg.Vectors
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object PolynomialExpansionExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("PolynomialExpansionExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("PolynomialExpansionExample").getOrCreate()
// $example on$
val data = Array(
@@ -37,7 +34,7 @@ object PolynomialExpansionExample {
Vectors.dense(0.0, 0.0),
Vectors.dense(0.6, -1.1)
)
- val df = sqlContext.createDataFrame(data.map(Tuple1.apply)).toDF("features")
+ val df = spark.createDataFrame(data.map(Tuple1.apply)).toDF("features")
val polynomialExpansion = new PolynomialExpansion()
.setInputCol("features")
.setOutputCol("polyFeatures")
@@ -45,7 +42,8 @@ object PolynomialExpansionExample {
val polyDF = polynomialExpansion.transform(df)
polyDF.select("polyFeatures").take(3).foreach(println)
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/QuantileDiscretizerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/QuantileDiscretizerExample.scala
index e64e673a48..0839c609f1 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/QuantileDiscretizerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/QuantileDiscretizerExample.scala
@@ -15,25 +15,21 @@
* limitations under the License.
*/
-// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.QuantileDiscretizer
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object QuantileDiscretizerExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("QuantileDiscretizerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("QuantileDiscretizerExample").getOrCreate()
+ import spark.implicits._
// $example on$
val data = Array((0, 18.0), (1, 19.0), (2, 8.0), (3, 5.0), (4, 2.2))
- val df = sc.parallelize(data).toDF("id", "hour")
+ val df = spark.createDataFrame(data).toDF("id", "hour")
val discretizer = new QuantileDiscretizer()
.setInputCol("hour")
@@ -43,7 +39,7 @@ object QuantileDiscretizerExample {
val result = discretizer.fit(df).transform(df)
result.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
-// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/RFormulaExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/RFormulaExample.scala
index bec831d51c..699b621db9 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/RFormulaExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/RFormulaExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.RFormula
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object RFormulaExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("RFormulaExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("RFormulaExample").getOrCreate()
// $example on$
- val dataset = sqlContext.createDataFrame(Seq(
+ val dataset = spark.createDataFrame(Seq(
(7, "US", 18, 1.0),
(8, "CA", 12, 0.0),
(9, "NZ", 15, 0.0)
@@ -43,7 +40,8 @@ object RFormulaExample {
val output = formula.fit(dataset).transform(dataset)
output.select("features", "label").show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestClassifierExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestClassifierExample.scala
index 6c9b52cf25..4192a9c737 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestClassifierExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestClassifierExample.scala
@@ -18,24 +18,21 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.classification.{RandomForestClassificationModel, RandomForestClassifier}
import org.apache.spark.ml.evaluation.MulticlassClassificationEvaluator
import org.apache.spark.ml.feature.{IndexToString, StringIndexer, VectorIndexer}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object RandomForestClassifierExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("RandomForestClassifierExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("RandomForestClassifierExample").getOrCreate()
// $example on$
// Load and parse the data file, converting it to a DataFrame.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Index labels, adding metadata to the label column.
// Fit on whole dataset to include all labels in index.
@@ -91,7 +88,7 @@ object RandomForestClassifierExample {
println("Learned classification forest model:\n" + rfModel.toDebugString)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestRegressorExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestRegressorExample.scala
index 4d2db017f3..5632f0419a 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestRegressorExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/RandomForestRegressorExample.scala
@@ -18,24 +18,21 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.evaluation.RegressionEvaluator
import org.apache.spark.ml.feature.VectorIndexer
import org.apache.spark.ml.regression.{RandomForestRegressionModel, RandomForestRegressor}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object RandomForestRegressorExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("RandomForestRegressorExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("RandomForestRegressorExample").getOrCreate()
// $example on$
// Load and parse the data file, converting it to a DataFrame.
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
// Automatically identify categorical features, and index them.
// Set maxCategories so features with > 4 distinct values are treated as continuous.
@@ -78,7 +75,7 @@ object RandomForestRegressorExample {
println("Learned regression forest model:\n" + rfModel.toDebugString)
// $example off$
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/SQLTransformerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/SQLTransformerExample.scala
index 202925acad..f03b29ba32 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/SQLTransformerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/SQLTransformerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.SQLTransformer
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object SQLTransformerExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("SQLTransformerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("SQLTransformerExample").getOrCreate()
// $example on$
- val df = sqlContext.createDataFrame(
+ val df = spark.createDataFrame(
Seq((0, 1.0, 3.0), (2, 2.0, 5.0))).toDF("id", "v1", "v2")
val sqlTrans = new SQLTransformer().setStatement(
@@ -39,6 +36,8 @@ object SQLTransformerExample {
sqlTrans.transform(df).show()
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/SimpleParamsExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/SimpleParamsExample.scala
index f4d1fe5785..dff7719507 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/SimpleParamsExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/SimpleParamsExample.scala
@@ -18,12 +18,11 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
import org.apache.spark.ml.classification.LogisticRegression
import org.apache.spark.ml.param.ParamMap
import org.apache.spark.mllib.linalg.{Vector, Vectors}
import org.apache.spark.mllib.regression.LabeledPoint
-import org.apache.spark.sql.{Row, SQLContext}
+import org.apache.spark.sql.{Row, SparkSession}
/**
* A simple example demonstrating ways to specify parameters for Estimators and Transformers.
@@ -35,15 +34,13 @@ import org.apache.spark.sql.{Row, SQLContext}
object SimpleParamsExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("SimpleParamsExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("SimpleParamsExample").getOrCreate()
+ import spark.implicits._
// Prepare training data.
// We use LabeledPoint, which is a case class. Spark SQL can convert RDDs of case classes
// into DataFrames, where it uses the case class metadata to infer the schema.
- val training = sc.parallelize(Seq(
+ val training = spark.createDataFrame(Seq(
LabeledPoint(1.0, Vectors.dense(0.0, 1.1, 0.1)),
LabeledPoint(0.0, Vectors.dense(2.0, 1.0, -1.0)),
LabeledPoint(0.0, Vectors.dense(2.0, 1.3, 1.0)),
@@ -59,7 +56,7 @@ object SimpleParamsExample {
.setRegParam(0.01)
// Learn a LogisticRegression model. This uses the parameters stored in lr.
- val model1 = lr.fit(training.toDF())
+ val model1 = lr.fit(training)
// Since model1 is a Model (i.e., a Transformer produced by an Estimator),
// we can view the parameters it used during fit().
// This prints the parameter (name: value) pairs, where names are unique IDs for this
@@ -82,7 +79,7 @@ object SimpleParamsExample {
println("Model 2 was fit using parameters: " + model2.parent.extractParamMap())
// Prepare test data.
- val test = sc.parallelize(Seq(
+ val test = spark.createDataFrame(Seq(
LabeledPoint(1.0, Vectors.dense(-1.0, 1.5, 1.3)),
LabeledPoint(0.0, Vectors.dense(3.0, 2.0, -0.1)),
LabeledPoint(1.0, Vectors.dense(0.0, 2.2, -1.5))))
@@ -91,14 +88,14 @@ object SimpleParamsExample {
// LogisticRegressionModel.transform will only use the 'features' column.
// Note that model2.transform() outputs a 'myProbability' column instead of the usual
// 'probability' column since we renamed the lr.probabilityCol parameter previously.
- model2.transform(test.toDF())
+ model2.transform(test)
.select("features", "label", "myProbability", "prediction")
.collect()
.foreach { case Row(features: Vector, label: Double, prob: Vector, prediction: Double) =>
println(s"($features, $label) -> prob=$prob, prediction=$prediction")
}
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/SimpleTextClassificationPipeline.scala b/examples/src/main/scala/org/apache/spark/examples/ml/SimpleTextClassificationPipeline.scala
index 960280137c..05199007f0 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/SimpleTextClassificationPipeline.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/SimpleTextClassificationPipeline.scala
@@ -20,12 +20,11 @@ package org.apache.spark.examples.ml
import scala.beans.BeanInfo
-import org.apache.spark.{SparkConf, SparkContext}
import org.apache.spark.ml.Pipeline
import org.apache.spark.ml.classification.LogisticRegression
import org.apache.spark.ml.feature.{HashingTF, Tokenizer}
import org.apache.spark.mllib.linalg.Vector
-import org.apache.spark.sql.{Row, SQLContext}
+import org.apache.spark.sql.{Row, SparkSession}
@BeanInfo
case class LabeledDocument(id: Long, text: String, label: Double)
@@ -43,13 +42,11 @@ case class Document(id: Long, text: String)
object SimpleTextClassificationPipeline {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("SimpleTextClassificationPipeline")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("SimpleTextClassificationPipeline").getOrCreate()
+ import spark.implicits._
// Prepare training documents, which are labeled.
- val training = sc.parallelize(Seq(
+ val training = spark.createDataFrame(Seq(
LabeledDocument(0L, "a b c d e spark", 1.0),
LabeledDocument(1L, "b d", 0.0),
LabeledDocument(2L, "spark f g h", 1.0),
@@ -73,7 +70,7 @@ object SimpleTextClassificationPipeline {
val model = pipeline.fit(training.toDF())
// Prepare test documents, which are unlabeled.
- val test = sc.parallelize(Seq(
+ val test = spark.createDataFrame(Seq(
Document(4L, "spark i j k"),
Document(5L, "l m n"),
Document(6L, "spark hadoop spark"),
@@ -87,7 +84,7 @@ object SimpleTextClassificationPipeline {
println(s"($id, $text) --> prob=$prob, prediction=$prediction")
}
- sc.stop()
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/StandardScalerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/StandardScalerExample.scala
index e3439677e7..55f777c6e2 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/StandardScalerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/StandardScalerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.StandardScaler
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object StandardScalerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("StandardScalerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("StandardScalerExample").getOrCreate()
// $example on$
- val dataFrame = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val dataFrame = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
val scaler = new StandardScaler()
.setInputCol("features")
@@ -46,7 +43,8 @@ object StandardScalerExample {
val scaledData = scalerModel.transform(dataFrame)
scaledData.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/StopWordsRemoverExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/StopWordsRemoverExample.scala
index 8199be12c1..85e79c8cb3 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/StopWordsRemoverExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/StopWordsRemoverExample.scala
@@ -18,31 +18,29 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.StopWordsRemover
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object StopWordsRemoverExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("StopWordsRemoverExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("StopWordsRemoverExample").getOrCreate()
// $example on$
val remover = new StopWordsRemover()
.setInputCol("raw")
.setOutputCol("filtered")
- val dataSet = sqlContext.createDataFrame(Seq(
+ val dataSet = spark.createDataFrame(Seq(
(0, Seq("I", "saw", "the", "red", "baloon")),
(1, Seq("Mary", "had", "a", "little", "lamb"))
)).toDF("id", "raw")
remover.transform(dataSet).show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/StringIndexerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/StringIndexerExample.scala
index 3f0e870c8d..e01a768da9 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/StringIndexerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/StringIndexerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.StringIndexer
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object StringIndexerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("StringIndexerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("StringIndexerExample").getOrCreate()
// $example on$
- val df = sqlContext.createDataFrame(
+ val df = spark.createDataFrame(
Seq((0, "a"), (1, "b"), (2, "c"), (3, "a"), (4, "a"), (5, "c"))
).toDF("id", "category")
@@ -42,7 +39,8 @@ object StringIndexerExample {
val indexed = indexer.fit(df).transform(df)
indexed.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/TfIdfExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/TfIdfExample.scala
index 396f073e6b..910ef62a26 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/TfIdfExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/TfIdfExample.scala
@@ -18,21 +18,18 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.{HashingTF, IDF, Tokenizer}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object TfIdfExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("TfIdfExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("TfIdfExample").getOrCreate()
// $example on$
- val sentenceData = sqlContext.createDataFrame(Seq(
+ val sentenceData = spark.createDataFrame(Seq(
(0, "Hi I heard about Spark"),
(0, "I wish Java could use case classes"),
(1, "Logistic regression models are neat")
@@ -50,6 +47,8 @@ object TfIdfExample {
val rescaledData = idfModel.transform(featurizedData)
rescaledData.select("features", "label").take(3).foreach(println)
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/TokenizerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/TokenizerExample.scala
index c667728d63..4f0c47b3c8 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/TokenizerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/TokenizerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.{RegexTokenizer, Tokenizer}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object TokenizerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("TokenizerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("TokenizerExample").getOrCreate()
// $example on$
- val sentenceDataFrame = sqlContext.createDataFrame(Seq(
+ val sentenceDataFrame = spark.createDataFrame(Seq(
(0, "Hi I heard about Spark"),
(1, "I wish Java could use case classes"),
(2, "Logistic,regression,models,are,neat")
@@ -48,7 +45,8 @@ object TokenizerExample {
val regexTokenized = regexTokenizer.transform(sentenceDataFrame)
regexTokenized.select("words", "label").take(3).foreach(println)
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/VectorAssemblerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/VectorAssemblerExample.scala
index 768a8c0690..56b7263b19 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/VectorAssemblerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/VectorAssemblerExample.scala
@@ -18,21 +18,18 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.VectorAssembler
import org.apache.spark.mllib.linalg.Vectors
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object VectorAssemblerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("VectorAssemblerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("VectorAssemblerExample").getOrCreate()
// $example on$
- val dataset = sqlContext.createDataFrame(
+ val dataset = spark.createDataFrame(
Seq((0, 18, 1.0, Vectors.dense(0.0, 10.0, 0.5), 1.0))
).toDF("id", "hour", "mobile", "userFeatures", "clicked")
@@ -43,7 +40,8 @@ object VectorAssemblerExample {
val output = assembler.transform(dataset)
println(output.select("features", "clicked").first())
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/VectorIndexerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/VectorIndexerExample.scala
index 3bef37ba36..214ad91634 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/VectorIndexerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/VectorIndexerExample.scala
@@ -18,20 +18,17 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.VectorIndexer
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object VectorIndexerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("VectorIndexerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("VectorIndexerExample").getOrCreate()
// $example on$
- val data = sqlContext.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
+ val data = spark.read.format("libsvm").load("data/mllib/sample_libsvm_data.txt")
val indexer = new VectorIndexer()
.setInputCol("features")
@@ -48,7 +45,8 @@ object VectorIndexerExample {
val indexedData = indexerModel.transform(data)
indexedData.show()
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/VectorSlicerExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/VectorSlicerExample.scala
index 01377d80e7..716bf023a8 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/VectorSlicerExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/VectorSlicerExample.scala
@@ -18,31 +18,29 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
+import java.util.Arrays
+
import org.apache.spark.ml.attribute.{Attribute, AttributeGroup, NumericAttribute}
import org.apache.spark.ml.feature.VectorSlicer
import org.apache.spark.mllib.linalg.Vectors
import org.apache.spark.sql.Row
import org.apache.spark.sql.types.StructType
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object VectorSlicerExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("VectorSlicerExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("VectorSlicerExample").getOrCreate()
// $example on$
- val data = Array(Row(Vectors.dense(-2.0, 2.3, 0.0)))
+ val data = Arrays.asList(Row(Vectors.dense(-2.0, 2.3, 0.0)))
val defaultAttr = NumericAttribute.defaultAttr
val attrs = Array("f1", "f2", "f3").map(defaultAttr.withName)
val attrGroup = new AttributeGroup("userFeatures", attrs.asInstanceOf[Array[Attribute]])
- val dataRDD = sc.parallelize(data)
- val dataset = sqlContext.createDataFrame(dataRDD, StructType(Array(attrGroup.toStructField())))
+ val dataset = spark.createDataFrame(data, StructType(Array(attrGroup.toStructField())))
val slicer = new VectorSlicer().setInputCol("userFeatures").setOutputCol("features")
@@ -52,7 +50,8 @@ object VectorSlicerExample {
val output = slicer.transform(dataset)
println(output.select("userFeatures", "features").first())
// $example off$
- sc.stop()
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/ml/Word2VecExample.scala b/examples/src/main/scala/org/apache/spark/examples/ml/Word2VecExample.scala
index e77aa59ba3..292b6d9f77 100644
--- a/examples/src/main/scala/org/apache/spark/examples/ml/Word2VecExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/ml/Word2VecExample.scala
@@ -18,21 +18,18 @@
// scalastyle:off println
package org.apache.spark.examples.ml
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.ml.feature.Word2Vec
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object Word2VecExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("Word2Vec example")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("Word2Vec example").getOrCreate()
// $example on$
// Input data: Each row is a bag of words from a sentence or document.
- val documentDF = sqlContext.createDataFrame(Seq(
+ val documentDF = spark.createDataFrame(Seq(
"Hi I heard about Spark".split(" "),
"I wish Java could use case classes".split(" "),
"Logistic regression models are neat".split(" ")
@@ -48,6 +45,8 @@ object Word2VecExample {
val result = model.transform(documentDF)
result.select("result").take(3).foreach(println)
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/mllib/LDAExample.scala b/examples/src/main/scala/org/apache/spark/examples/mllib/LDAExample.scala
index e89d555884..c2bf1548b5 100644
--- a/examples/src/main/scala/org/apache/spark/examples/mllib/LDAExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/mllib/LDAExample.scala
@@ -27,7 +27,7 @@ import org.apache.spark.ml.feature.{CountVectorizer, CountVectorizerModel, Regex
import org.apache.spark.mllib.clustering.{DistributedLDAModel, EMLDAOptimizer, LDA, OnlineLDAOptimizer}
import org.apache.spark.mllib.linalg.Vector
import org.apache.spark.rdd.RDD
-import org.apache.spark.sql.{Row, SQLContext}
+import org.apache.spark.sql.{Row, SparkSession}
/**
* An example Latent Dirichlet Allocation (LDA) app. Run with
@@ -189,8 +189,8 @@ object LDAExample {
vocabSize: Int,
stopwordFile: String): (RDD[(Long, Vector)], Array[String], Long) = {
- val sqlContext = SQLContext.getOrCreate(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.getOrCreate()
+ import spark.implicits._
// Get dataset of document texts
// One document per line in each text file. If the input consists of many small files,
diff --git a/examples/src/main/scala/org/apache/spark/examples/mllib/RankingMetricsExample.scala b/examples/src/main/scala/org/apache/spark/examples/mllib/RankingMetricsExample.scala
index fdb01b86dd..cd4f0bb0de 100644
--- a/examples/src/main/scala/org/apache/spark/examples/mllib/RankingMetricsExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/mllib/RankingMetricsExample.scala
@@ -18,22 +18,19 @@
// scalastyle:off println
package org.apache.spark.examples.mllib
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.mllib.evaluation.{RankingMetrics, RegressionMetrics}
import org.apache.spark.mllib.recommendation.{ALS, Rating}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object RankingMetricsExample {
def main(args: Array[String]) {
- val conf = new SparkConf().setAppName("RankingMetricsExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
- import sqlContext.implicits._
+ val spark = SparkSession.builder.appName("RankingMetricsExample").getOrCreate()
+ import spark.implicits._
// $example on$
// Read in the ratings data
- val ratings = sc.textFile("data/mllib/sample_movielens_data.txt").map { line =>
+ val ratings = spark.read.text("data/mllib/sample_movielens_data.txt").rdd.map { line =>
val fields = line.split("::")
Rating(fields(0).toInt, fields(1).toInt, fields(2).toDouble - 2.5)
}.cache()
diff --git a/examples/src/main/scala/org/apache/spark/examples/mllib/RegressionMetricsExample.scala b/examples/src/main/scala/org/apache/spark/examples/mllib/RegressionMetricsExample.scala
index add634c957..22c47a694d 100644
--- a/examples/src/main/scala/org/apache/spark/examples/mllib/RegressionMetricsExample.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/mllib/RegressionMetricsExample.scala
@@ -18,22 +18,22 @@
package org.apache.spark.examples.mllib
-import org.apache.spark.{SparkConf, SparkContext}
// $example on$
import org.apache.spark.mllib.evaluation.RegressionMetrics
-import org.apache.spark.mllib.regression.LinearRegressionWithSGD
-import org.apache.spark.mllib.util.MLUtils
+import org.apache.spark.mllib.linalg.Vector
+import org.apache.spark.mllib.regression.{LabeledPoint, LinearRegressionWithSGD}
// $example off$
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
object RegressionMetricsExample {
def main(args: Array[String]): Unit = {
- val conf = new SparkConf().setAppName("RegressionMetricsExample")
- val sc = new SparkContext(conf)
- val sqlContext = new SQLContext(sc)
+ val spark = SparkSession.builder.appName("RegressionMetricsExample").getOrCreate()
// $example on$
// Load the data
- val data = MLUtils.loadLibSVMFile(sc, "data/mllib/sample_linear_regression_data.txt").cache()
+ val data = spark
+ .read.format("libsvm").load("data/mllib/sample_linear_regression_data.txt")
+ .rdd.map(row => LabeledPoint(row.getDouble(0), row.get(1).asInstanceOf[Vector]))
+ .cache()
// Build the model
val numIterations = 100
@@ -61,6 +61,8 @@ object RegressionMetricsExample {
// Explained variance
println(s"Explained variance = ${metrics.explainedVariance}")
// $example off$
+
+ spark.stop()
}
}
// scalastyle:on println
diff --git a/examples/src/main/scala/org/apache/spark/examples/streaming/SqlNetworkWordCount.scala b/examples/src/main/scala/org/apache/spark/examples/streaming/SqlNetworkWordCount.scala
index 918e124065..2f0fe704f7 100644
--- a/examples/src/main/scala/org/apache/spark/examples/streaming/SqlNetworkWordCount.scala
+++ b/examples/src/main/scala/org/apache/spark/examples/streaming/SqlNetworkWordCount.scala
@@ -19,9 +19,8 @@
package org.apache.spark.examples.streaming
import org.apache.spark.SparkConf
-import org.apache.spark.SparkContext
import org.apache.spark.rdd.RDD
-import org.apache.spark.sql.SQLContext
+import org.apache.spark.sql.SparkSession
import org.apache.spark.storage.StorageLevel
import org.apache.spark.streaming.{Seconds, StreamingContext, Time}
@@ -60,9 +59,9 @@ object SqlNetworkWordCount {
// Convert RDDs of the words DStream to DataFrame and run SQL query
words.foreachRDD { (rdd: RDD[String], time: Time) =>
- // Get the singleton instance of SQLContext
- val sqlContext = SQLContextSingleton.getInstance(rdd.sparkContext)
- import sqlContext.implicits._
+ // Get the singleton instance of SparkSession
+ val spark = SparkSessionSingleton.getInstance(rdd.sparkContext.getConf)
+ import spark.implicits._
// Convert RDD[String] to RDD[case class] to DataFrame
val wordsDataFrame = rdd.map(w => Record(w)).toDF()
@@ -72,7 +71,7 @@ object SqlNetworkWordCount {
// Do word count on table using SQL and print it
val wordCountsDataFrame =
- sqlContext.sql("select word, count(*) as total from words group by word")
+ spark.sql("select word, count(*) as total from words group by word")
println(s"========= $time =========")
wordCountsDataFrame.show()
}
@@ -87,14 +86,14 @@ object SqlNetworkWordCount {
case class Record(word: String)
-/** Lazily instantiated singleton instance of SQLContext */
-object SQLContextSingleton {
+/** Lazily instantiated singleton instance of SparkSession */
+object SparkSessionSingleton {
- @transient private var instance: SQLContext = _
+ @transient private var instance: SparkSession = _
- def getInstance(sparkContext: SparkContext): SQLContext = {
+ def getInstance(sparkConf: SparkConf): SparkSession = {
if (instance == null) {
- instance = new SQLContext(sparkContext)
+ instance = SparkSession.builder.config(sparkConf).getOrCreate()
}
instance
}