Skip to content

Commit

Permalink
child first classpath first has no regular parent
Browse files Browse the repository at this point in the history
  • Loading branch information
yaooqinn committed Jan 4, 2018
1 parent a6fc300 commit 7cb0c44
Show file tree
Hide file tree
Showing 2 changed files with 35 additions and 1 deletion.
Original file line number Diff line number Diff line change
Expand Up @@ -47,7 +47,7 @@ import org.apache.spark.sql.internal.SQLConf
import org.apache.spark.sql.internal.SQLConf._
import org.apache.spark.sql.internal.StaticSQLConf.{CATALOG_IMPLEMENTATION, WAREHOUSE_PATH}
import org.apache.spark.sql.types._
import org.apache.spark.util.Utils
import org.apache.spark.util.{ChildFirstURLClassLoader, Utils}


private[spark] object HiveUtils extends Logging {
Expand Down Expand Up @@ -312,6 +312,8 @@ private[spark] object HiveUtils extends Logging {
// starting from the given classLoader.
def allJars(classLoader: ClassLoader): Array[URL] = classLoader match {
case null => Array.empty[URL]
case childFirst: ChildFirstURLClassLoader =>
childFirst.getURLs() ++ allJars(Utils.getSparkClassLoader)
case urlClassLoader: URLClassLoader =>
urlClassLoader.getURLs ++ allJars(urlClassLoader.getParent)
case other => allJars(other.getParent)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,11 +17,16 @@

package org.apache.spark.sql.hive

import java.net.URL

import org.apache.hadoop.hive.conf.HiveConf.ConfVars

import org.apache.spark.SparkConf
import org.apache.spark.deploy.SparkHadoopUtil
import org.apache.spark.sql.QueryTest
import org.apache.spark.sql.hive.test.TestHiveSingleton
import org.apache.spark.sql.test.SQLTestUtils
import org.apache.spark.util.{ChildFirstURLClassLoader, MutableURLClassLoader}

class HiveUtilsSuite extends QueryTest with SQLTestUtils with TestHiveSingleton {

Expand All @@ -42,4 +47,31 @@ class HiveUtilsSuite extends QueryTest with SQLTestUtils with TestHiveSingleton
assert(hiveConf("foo") === "bar")
}
}

test("ChildFirstURLClassLoader's parent is null") {
val conf = new SparkConf
val contextClassLoader = Thread.currentThread().getContextClassLoader
val loader = new FakeChildFirstURLClassLoader(Array(), contextClassLoader)
Thread.currentThread().setContextClassLoader(loader)
intercept[IllegalArgumentException](
HiveUtils.newClientForMetadata(conf, SparkHadoopUtil.newConfiguration(conf)))
Thread.currentThread().setContextClassLoader(contextClassLoader)
}

test("ChildFirstURLClassLoader's parent is null, get spark classloader instead") {
val conf = new SparkConf
val contextClassLoader = Thread.currentThread().getContextClassLoader
val loader = new ChildFirstURLClassLoader(Array(), contextClassLoader)
Thread.currentThread().setContextClassLoader(loader)
HiveUtils.newClientForMetadata(conf, SparkHadoopUtil.newConfiguration(conf))
Thread.currentThread().setContextClassLoader(contextClassLoader)
}
}

/**
* A Fake [[ChildFirstURLClassLoader]] used for test
* @param urls
* @param parent
*/
class FakeChildFirstURLClassLoader(urls: Array[URL], parent: ClassLoader)
extends MutableURLClassLoader(urls, null)

0 comments on commit 7cb0c44

Please sign in to comment.