Skip to content
Closed
Show file tree
Hide file tree
Changes from all 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 @@ -33,7 +33,7 @@ import org.apache.hadoop.hive.common.{HiveInterruptCallback, HiveInterruptUtils,
import org.apache.hadoop.hive.conf.HiveConf
import org.apache.hadoop.hive.ql.Driver
import org.apache.hadoop.hive.ql.exec.Utilities
import org.apache.hadoop.hive.ql.processors.{SetProcessor, CommandProcessor, CommandProcessorFactory}
import org.apache.hadoop.hive.ql.processors.{AddResourceProcessor, SetProcessor, CommandProcessor, CommandProcessorFactory}
import org.apache.hadoop.hive.ql.session.SessionState
import org.apache.hadoop.hive.shims.ShimLoader
import org.apache.thrift.transport.TSocket
Expand Down Expand Up @@ -264,7 +264,8 @@ private[hive] class SparkSQLCLIDriver extends CliDriver with Logging {
val proc: CommandProcessor = HiveShim.getCommandProcessor(Array(tokens(0)), hconf)

if (proc != null) {
if (proc.isInstanceOf[Driver] || proc.isInstanceOf[SetProcessor]) {
if (proc.isInstanceOf[Driver] || proc.isInstanceOf[SetProcessor] ||
proc.isInstanceOf[AddResourceProcessor]) {
Copy link
Contributor

Choose a reason for hiding this comment

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

just need "proc.isInstanceOf[AddResourceProcessor]", then "add file" will work.

Copy link
Contributor

Choose a reason for hiding this comment

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

agree

val driver = new SparkSQLDriver

driver.init()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -35,6 +35,7 @@ import org.apache.spark.broadcast.Broadcast
import org.apache.spark.rdd.{EmptyRDD, HadoopRDD, RDD, UnionRDD}
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.types.DateUtils
import org.apache.spark.util.Utils

/**
* A trait for subclasses that handle table scans.
Expand Down Expand Up @@ -76,7 +77,9 @@ class HadoopTableReader(
override def makeRDDForTable(hiveTable: HiveTable): RDD[Row] =
makeRDDForTable(
hiveTable,
relation.tableDesc.getDeserializerClass.asInstanceOf[Class[Deserializer]],
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 @@ -79,7 +79,7 @@ case class AddJar(path: String) extends RunnableCommand {
val hiveContext = sqlContext.asInstanceOf[HiveContext]
hiveContext.runSqlHive(s"ADD JAR $path")
hiveContext.sparkContext.addJar(path)
Seq.empty[Row]
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
Binary file not shown.
Original file line number Diff line number Diff line change
Expand Up @@ -793,6 +793,21 @@ class HiveQuerySuite extends HiveComparisonTest with BeforeAndAfter {
sql("DROP TABLE alter1")
}

test("ADD JAR command 2") {
// this is a test case from mapjoin_addjar.q
val testJar = TestHive.getHiveFile("hive-hcatalog-core-0.13.1.jar").getCanonicalPath
val testData = TestHive.getHiveFile("data/files/sample.json").getCanonicalPath
if (HiveShim.version == "0.13.1") {
sql(s"ADD JAR $testJar")
sql(
"""CREATE TABLE t1(a string, b string)
|ROW FORMAT SERDE 'org.apache.hive.hcatalog.data.JsonSerDe'""".stripMargin)
sql(s"""LOAD DATA LOCAL INPATH "$testData" INTO TABLE t1""")
sql("select * from src join t1 on src.key = t1.a")
sql("DROP TABLE t1")
}
}

test("ADD FILE command") {
val testFile = TestHive.getHiveFile("data/files/v1.txt").getCanonicalFile
sql(s"ADD FILE $testFile")
Expand Down