Skip to content

Commit

Permalink
[SPARK-15622] [SQL] Wrap the parent classloader of Janino's classload…
Browse files Browse the repository at this point in the history
…er in the ParentClassLoader.
  • Loading branch information
yhuai committed May 27, 2016
1 parent 4b88067 commit a539fae
Show file tree
Hide file tree
Showing 2 changed files with 14 additions and 10 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -79,13 +79,7 @@ class ExecutorClassLoader(
case e: ClassNotFoundException =>
val classOption = findClassLocally(name)
classOption match {
case None =>
// If this class has a cause, it will break the internal assumption of Janino
// (the compiler used for Spark SQL code-gen).
// See org.codehaus.janino.ClassLoaderIClassLoader's findIClass, you will see
// its behavior will be changed if there is a cause and the compilation
// of generated class will fail.
throw new ClassNotFoundException(name)
case None => throw new ClassNotFoundException(name, e)
case Some(a) => a
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,10 +19,10 @@ package org.apache.spark.sql.catalyst.expressions.codegen

import scala.collection.mutable
import scala.collection.mutable.ArrayBuffer
import scala.language.existentials

import com.google.common.cache.{CacheBuilder, CacheLoader}
import org.codehaus.janino.ClassBodyEvaluator
import scala.language.existentials

import org.apache.spark.internal.Logging
import org.apache.spark.sql.catalyst.InternalRow
Expand All @@ -31,7 +31,7 @@ import org.apache.spark.sql.catalyst.util.{ArrayData, MapData}
import org.apache.spark.sql.types._
import org.apache.spark.unsafe.Platform
import org.apache.spark.unsafe.types._
import org.apache.spark.util.Utils
import org.apache.spark.util.{ParentClassLoader, Utils}

/**
* Java source for evaluating an [[Expression]] given a [[InternalRow]] of input.
Expand Down Expand Up @@ -806,7 +806,17 @@ object CodeGenerator extends Logging {
*/
private[this] def doCompile(code: CodeAndComment): GeneratedClass = {
val evaluator = new ClassBodyEvaluator()
evaluator.setParentClassLoader(Utils.getContextOrSparkClassLoader)

// A special classloader used to wrap the actual parent classloader of
// [[org.codehaus.janino.ClassBodyEvaluator]] (see CodeGenerator.doCompile). This classloader
// does not throw a ClassNotFoundException with a cause set (i.e. exception.getCause returns
// a null). This classloader is needed because janino will throw the exception directly if
// the parent classloader throws a ClassNotFoundException with cause set instead of trying to
// find other possible classes (see org.codehaus.janinoClassLoaderIClassLoader's
// findIClass method). Please also see https://issues.apache.org/jira/browse/SPARK-15622 and
// https://issues.apache.org/jira/browse/SPARK-11636.
val parentClassLoader = new ParentClassLoader(Utils.getContextOrSparkClassLoader)
evaluator.setParentClassLoader(parentClassLoader)
// Cannot be under package codegen, or fail with java.lang.InstantiationException
evaluator.setClassName("org.apache.spark.sql.catalyst.expressions.GeneratedClass")
evaluator.setDefaultImports(Array(
Expand Down

0 comments on commit a539fae

Please sign in to comment.