Skip to content

Commit b45059d

Browse files
adrian-wangmarmbrus
authored andcommitted
[SPARK-5794] [SQL] fix add jar
Author: Daoyuan Wang <[email protected]> Closes #4586 from adrian-wang/addjar and squashes the following commits: efdd602 [Daoyuan Wang] move jar to another place 6c707e8 [Daoyuan Wang] restrict hive version for test 32c4fb8 [Daoyuan Wang] fix style and add a test 9957d87 [Daoyuan Wang] use sessionstate classloader in makeRDDforTable 0810e71 [Daoyuan Wang] remove variable substitution 1898309 [Daoyuan Wang] fix classnotfound 95a40da [Daoyuan Wang] support env argus in add jar, and set add jar ret to 0
1 parent 3782e1f commit b45059d

File tree

5 files changed

+23
-4
lines changed

5 files changed

+23
-4
lines changed

sql/hive-thriftserver/src/main/scala/org/apache/spark/sql/hive/thriftserver/SparkSQLCLIDriver.scala

Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -33,7 +33,7 @@ import org.apache.hadoop.hive.common.{HiveInterruptCallback, HiveInterruptUtils,
3333
import org.apache.hadoop.hive.conf.HiveConf
3434
import org.apache.hadoop.hive.ql.Driver
3535
import org.apache.hadoop.hive.ql.exec.Utilities
36-
import org.apache.hadoop.hive.ql.processors.{SetProcessor, CommandProcessor, CommandProcessorFactory}
36+
import org.apache.hadoop.hive.ql.processors.{AddResourceProcessor, SetProcessor, CommandProcessor, CommandProcessorFactory}
3737
import org.apache.hadoop.hive.ql.session.SessionState
3838
import org.apache.hadoop.hive.shims.ShimLoader
3939
import org.apache.thrift.transport.TSocket
@@ -264,7 +264,8 @@ private[hive] class SparkSQLCLIDriver extends CliDriver with Logging {
264264
val proc: CommandProcessor = HiveShim.getCommandProcessor(Array(tokens(0)), hconf)
265265

266266
if (proc != null) {
267-
if (proc.isInstanceOf[Driver] || proc.isInstanceOf[SetProcessor]) {
267+
if (proc.isInstanceOf[Driver] || proc.isInstanceOf[SetProcessor] ||
268+
proc.isInstanceOf[AddResourceProcessor]) {
268269
val driver = new SparkSQLDriver
269270

270271
driver.init()

sql/hive/src/main/scala/org/apache/spark/sql/hive/TableReader.scala

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -35,6 +35,7 @@ import org.apache.spark.broadcast.Broadcast
3535
import org.apache.spark.rdd.{EmptyRDD, HadoopRDD, RDD, UnionRDD}
3636
import org.apache.spark.sql.catalyst.expressions._
3737
import org.apache.spark.sql.types.DateUtils
38+
import org.apache.spark.util.Utils
3839

3940
/**
4041
* A trait for subclasses that handle table scans.
@@ -76,7 +77,9 @@ class HadoopTableReader(
7677
override def makeRDDForTable(hiveTable: HiveTable): RDD[Row] =
7778
makeRDDForTable(
7879
hiveTable,
79-
relation.tableDesc.getDeserializerClass.asInstanceOf[Class[Deserializer]],
80+
Class.forName(
81+
relation.tableDesc.getSerdeClassName, true, sc.sessionState.getConf.getClassLoader)
82+
.asInstanceOf[Class[Deserializer]],
8083
filterOpt = None)
8184

8285
/**

sql/hive/src/main/scala/org/apache/spark/sql/hive/execution/commands.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -80,7 +80,7 @@ case class AddJar(path: String) extends RunnableCommand {
8080
val hiveContext = sqlContext.asInstanceOf[HiveContext]
8181
hiveContext.runSqlHive(s"ADD JAR $path")
8282
hiveContext.sparkContext.addJar(path)
83-
Seq.empty[Row]
83+
Seq(Row(0))
8484
}
8585
}
8686

458 KB
Binary file not shown.

sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveQuerySuite.scala

Lines changed: 15 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -813,6 +813,21 @@ class HiveQuerySuite extends HiveComparisonTest with BeforeAndAfter {
813813
sql("DROP TABLE alter1")
814814
}
815815

816+
test("ADD JAR command 2") {
817+
// this is a test case from mapjoin_addjar.q
818+
val testJar = TestHive.getHiveFile("hive-hcatalog-core-0.13.1.jar").getCanonicalPath
819+
val testData = TestHive.getHiveFile("data/files/sample.json").getCanonicalPath
820+
if (HiveShim.version == "0.13.1") {
821+
sql(s"ADD JAR $testJar")
822+
sql(
823+
"""CREATE TABLE t1(a string, b string)
824+
|ROW FORMAT SERDE 'org.apache.hive.hcatalog.data.JsonSerDe'""".stripMargin)
825+
sql(s"""LOAD DATA LOCAL INPATH "$testData" INTO TABLE t1""")
826+
sql("select * from src join t1 on src.key = t1.a")
827+
sql("DROP TABLE t1")
828+
}
829+
}
830+
816831
test("ADD FILE command") {
817832
val testFile = TestHive.getHiveFile("data/files/v1.txt").getCanonicalFile
818833
sql(s"ADD FILE $testFile")

0 commit comments

Comments
 (0)