Skip to content
Closed
Show file tree
Hide file tree
Changes from 2 commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -36,13 +36,14 @@ object HiveFromSpark {

def main(args: Array[String]) {
val sparkConf = new SparkConf().setAppName("HiveFromSpark")
val sc = new SparkContext(sparkConf)

// A hive context adds support for finding tables in the MetaStore and writing queries
// using HiveQL. Users who do not have an existing Hive deployment can still create a
// HiveContext. When not configured by the hive-site.xml, the context automatically
// creates metastore_db and warehouse in the current directory.
val sparkSession = SparkSession.withHiveSupport(sc)
val sparkSession = SparkSession.builder.config(sparkConf).enableHiveSupport().getOrCreate()
val sc = sparkSession.sparkContext
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

github won't let me comment down there, but in L78 please replace sc.stop with spark.stop


import sparkSession.implicits._
import sparkSession.sql

Expand Down
13 changes: 0 additions & 13 deletions sql/core/src/main/scala/org/apache/spark/sql/SparkSession.scala
Original file line number Diff line number Diff line change
Expand Up @@ -816,17 +816,4 @@ object SparkSession {
}
}

/**
* Create a new [[SparkSession]] with a catalog backed by Hive.
*/
def withHiveSupport(sc: SparkContext): SparkSession = {
if (hiveClassesArePresent) {
sc.conf.set(CATALOG_IMPLEMENTATION.key, "hive")
new SparkSession(sc)
} else {
throw new IllegalArgumentException(
"Unable to instantiate SparkSession with Hive support because Hive classes are not found.")
}
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -54,13 +54,15 @@ private[hive] object SparkSQLEnv extends Logging {
"spark.kryo.referenceTracking",
maybeKryoReferenceTracking.getOrElse("false"))

sparkContext = new SparkContext(sparkConf)
sqlContext = SparkSession.withHiveSupport(sparkContext).wrapped
val sessionState = sqlContext.sessionState.asInstanceOf[HiveSessionState]
val sparkSession = SparkSession.builder.config(sparkConf).enableHiveSupport().getOrCreate()
sparkContext = sparkSession.sparkContext
sqlContext = sparkSession.wrapped

val sessionState = sparkSession.sessionState.asInstanceOf[HiveSessionState]
sessionState.metadataHive.setOut(new PrintStream(System.out, true, "UTF-8"))
sessionState.metadataHive.setInfo(new PrintStream(System.err, true, "UTF-8"))
sessionState.metadataHive.setError(new PrintStream(System.err, true, "UTF-8"))
sqlContext.setConf("spark.sql.hive.version", HiveUtils.hiveExecutionVersion)
sparkSession.conf.set("spark.sql.hive.version", HiveUtils.hiveExecutionVersion)
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,7 +15,7 @@
* limitations under the License.
*/

import org.apache.spark.SparkContext
import org.apache.spark.{SparkContext, SparkConf}
import org.apache.spark.sql.SparkSession

/**
Expand All @@ -33,8 +33,14 @@ object Main {
def main(args: Array[String]) {
// scalastyle:off println
println("Running regression test for SPARK-8489.")
val sc = new SparkContext("local", "testing")
val sparkSession = SparkSession.withHiveSupport(sc)

val conf = new SparkConf()
.setMaster("local")
.setAppName("testing")

val sparkSession = SparkSession.builder.enableHiveSupport().getOrCreate()
Copy link
Contributor

@andrewor14 andrewor14 May 5, 2016

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

this should use the builder's conf method. By the way for this file I would revert the changes here and fix it in #12924 instead, since @dilipbiswal is actually rebuilding the jar there.

val sc = sparkSession.sparkContext

// This line should not throw scala.reflect.internal.MissingRequirementError.
// See SPARK-8470 for more detail.
val df = sparkSession.createDataFrame(Seq(MyCoolClass("1", "2", "3")))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -290,8 +290,9 @@ object SetWarehouseLocationTest extends Logging {
conf.set("spark.sql.warehouse.dir", warehouseLocation.toString)
conf.set("hive.metastore.warehouse.dir", hiveWarehouseLocation.toString)

val sc = new SparkContext(conf)
val sparkSession = SparkSession.withHiveSupport(sc)
val sparkSession = SparkSession.builder.config(conf).enableHiveSupport().getOrCreate()
val sc = sparkSession.sparkContext
Copy link
Contributor

@andrewor14 andrewor14 May 5, 2016

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

this isn't actually used anywhere


val catalog = sparkSession.sessionState.catalog

sparkSession.sql("drop table if exists testLocation")
Expand Down