Skip to content
Closed
Show file tree
Hide file tree
Changes from 1 commit
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
Prev Previous commit
Next Next commit
use sessionstate classloader in makeRDDforTable
  • Loading branch information
adrian-wang committed Apr 8, 2015
commit 9957d870063c70e12c850c07192ada2eeb11a9e7
11 changes: 0 additions & 11 deletions core/src/main/scala/org/apache/spark/SparkContext.scala
Original file line number Diff line number Diff line change
Expand Up @@ -103,17 +103,6 @@ class SparkContext(config: SparkConf) extends Logging with ExecutorAllocationCli
}
}

private var classLoader = Thread.currentThread().getContextClassLoader();
if (classLoader == null) {
classLoader = this.getClass.getClassLoader()
}

def getClassLoader: ClassLoader = classLoader

def setClassLoader(cl: ClassLoader) = {
classLoader = cl
}

/**
* Create a SparkContext that loads settings from system properties (for instance, when
* launching with ./bin/spark-submit).
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -620,7 +620,6 @@ class HiveCompatibilitySuite extends HiveQueryFileTest with BeforeAndAfter {
"load_file_with_space_in_the_name",
"loadpart1",
"louter_join_ppr",
"mapjoin_addjar",
"mapjoin_distinct",
"mapjoin_filter_on_outerjoin",
"mapjoin_mapjoin",
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -77,7 +77,7 @@ class HadoopTableReader(
override def makeRDDForTable(hiveTable: HiveTable): RDD[Row] =
makeRDDForTable(
hiveTable,
Class.forName(relation.tableDesc.getSerdeClassName, true, sc.sparkContext.getClassLoader)
Class.forName(relation.tableDesc.getSerdeClassName, true, sc.sessionState.getConf.getClassLoader)
.asInstanceOf[Class[Deserializer]],
filterOpt = None)

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,10 +17,6 @@

package org.apache.spark.sql.hive.execution

import org.apache.commons.lang.StringUtils
import org.apache.hadoop.hive.ql.exec.Utilities
import org.apache.hadoop.hive.ql.parse.VariableSubstitution

import org.apache.spark.sql.AnalysisException
import org.apache.spark.sql.catalyst.analysis.EliminateSubQueries
import org.apache.spark.sql.catalyst.util._
Expand Down Expand Up @@ -83,20 +79,7 @@ case class AddJar(path: String) extends RunnableCommand {
val hiveContext = sqlContext.asInstanceOf[HiveContext]
hiveContext.runSqlHive(s"ADD JAR $path")
hiveContext.sparkContext.addJar(path)
try {
val loader = Thread.currentThread().getContextClassLoader()
val newLoader = Utilities.addToClassPath(loader, StringUtils.split(path, ","))
Thread.currentThread().setContextClassLoader(newLoader)
sqlContext.sparkContext.setClassLoader(newLoader)
logInfo("Added " + path + " to class path")
Seq(Row(0))
} catch {
case e: Throwable =>
logError("Unable to register " + path + "\nException: "
+ e.getMessage() + "\n"
+ org.apache.hadoop.util.StringUtils.stringifyException(e), e)
Seq(Row(1))
}
Seq(Row(0))
Copy link
Contributor

Choose a reason for hiding this comment

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

@adrian-wang Why we need a Row with a value of 0 at here?

Copy link
Contributor Author

Choose a reason for hiding this comment

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

Hive would return a 0 for add jar command.

Copy link
Contributor

Choose a reason for hiding this comment

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

(I thought it may be better to comment at the original pr).

OK, I see. In future, let's make sure we also update the output if the result of a command is not an empty Seq (#5350 will change the schema for AddJar).

}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -66,10 +66,6 @@ class TestHiveContext(sc: SparkContext) extends HiveContext(sc) {
// without restarting the JVM.
System.clearProperty("spark.hostPort")
CommandProcessorFactory.clean(hiveconf)
System.setProperty("hive.version", HiveShim.version)
System.setProperty(
"maven.local.repository",
System.getProperty("user.home") + File.separator + ".m2" + File.separator + "repository")

hiveconf.set("hive.plan.serialization.format", "javaXML")

Expand Down

This file was deleted.

This file was deleted.

This file was deleted.

This file was deleted.

This file was deleted.