Skip to content
Snippets Groups Projects
Commit 875f5079 authored by Davies Liu's avatar Davies Liu Committed by Davies Liu
Browse files

[SPARK-13215] [SQL] remove fallback in codegen

Since we remove the configuration for codegen, we are heavily reply on codegen (also TungstenAggregate require the generated MutableProjection to update UnsafeRow), should remove the fallback, which could make user confusing, see the discussion in SPARK-13116.

Author: Davies Liu <davies@databricks.com>

Closes #11097 from davies/remove_fallback.
parent 0bb5b733
No related branches found
No related tags found
No related merge requests found
...@@ -200,47 +200,17 @@ abstract class SparkPlan extends QueryPlan[SparkPlan] with Logging with Serializ ...@@ -200,47 +200,17 @@ abstract class SparkPlan extends QueryPlan[SparkPlan] with Logging with Serializ
inputSchema: Seq[Attribute], inputSchema: Seq[Attribute],
useSubexprElimination: Boolean = false): () => MutableProjection = { useSubexprElimination: Boolean = false): () => MutableProjection = {
log.debug(s"Creating MutableProj: $expressions, inputSchema: $inputSchema") log.debug(s"Creating MutableProj: $expressions, inputSchema: $inputSchema")
try { GenerateMutableProjection.generate(expressions, inputSchema, useSubexprElimination)
GenerateMutableProjection.generate(expressions, inputSchema, useSubexprElimination)
} catch {
case e: Exception =>
if (isTesting) {
throw e
} else {
log.error("Failed to generate mutable projection, fallback to interpreted", e)
() => new InterpretedMutableProjection(expressions, inputSchema)
}
}
} }
protected def newPredicate( protected def newPredicate(
expression: Expression, inputSchema: Seq[Attribute]): (InternalRow) => Boolean = { expression: Expression, inputSchema: Seq[Attribute]): (InternalRow) => Boolean = {
try { GeneratePredicate.generate(expression, inputSchema)
GeneratePredicate.generate(expression, inputSchema)
} catch {
case e: Exception =>
if (isTesting) {
throw e
} else {
log.error("Failed to generate predicate, fallback to interpreted", e)
InterpretedPredicate.create(expression, inputSchema)
}
}
} }
protected def newOrdering( protected def newOrdering(
order: Seq[SortOrder], inputSchema: Seq[Attribute]): Ordering[InternalRow] = { order: Seq[SortOrder], inputSchema: Seq[Attribute]): Ordering[InternalRow] = {
try { GenerateOrdering.generate(order, inputSchema)
GenerateOrdering.generate(order, inputSchema)
} catch {
case e: Exception =>
if (isTesting) {
throw e
} else {
log.error("Failed to generate ordering, fallback to interpreted", e)
new InterpretedOrdering(order, inputSchema)
}
}
} }
/** /**
......
...@@ -20,8 +20,8 @@ package org.apache.spark.sql.execution.aggregate ...@@ -20,8 +20,8 @@ package org.apache.spark.sql.execution.aggregate
import org.apache.spark.Logging import org.apache.spark.Logging
import org.apache.spark.sql.Row import org.apache.spark.sql.Row
import org.apache.spark.sql.catalyst.{CatalystTypeConverters, InternalRow} import org.apache.spark.sql.catalyst.{CatalystTypeConverters, InternalRow}
import org.apache.spark.sql.catalyst.expressions.{AttributeReference, Expression, InterpretedMutableProjection, MutableRow} import org.apache.spark.sql.catalyst.expressions.{AttributeReference, Expression, MutableRow}
import org.apache.spark.sql.catalyst.expressions.aggregate.{AggregateFunction, ImperativeAggregate} import org.apache.spark.sql.catalyst.expressions.aggregate.ImperativeAggregate
import org.apache.spark.sql.catalyst.expressions.codegen.GenerateMutableProjection import org.apache.spark.sql.catalyst.expressions.codegen.GenerateMutableProjection
import org.apache.spark.sql.expressions.{MutableAggregationBuffer, UserDefinedAggregateFunction} import org.apache.spark.sql.expressions.{MutableAggregationBuffer, UserDefinedAggregateFunction}
import org.apache.spark.sql.types._ import org.apache.spark.sql.types._
...@@ -361,13 +361,7 @@ private[sql] case class ScalaUDAF( ...@@ -361,13 +361,7 @@ private[sql] case class ScalaUDAF(
val inputAttributes = childrenSchema.toAttributes val inputAttributes = childrenSchema.toAttributes
log.debug( log.debug(
s"Creating MutableProj: $children, inputSchema: $inputAttributes.") s"Creating MutableProj: $children, inputSchema: $inputAttributes.")
try { GenerateMutableProjection.generate(children, inputAttributes)()
GenerateMutableProjection.generate(children, inputAttributes)()
} catch {
case e: Exception =>
log.error("Failed to generate mutable projection, fallback to interpreted", e)
new InterpretedMutableProjection(children, inputAttributes)
}
} }
private[this] lazy val inputToScalaConverters: Any => Any = private[this] lazy val inputToScalaConverters: Any => Any =
......
...@@ -17,8 +17,6 @@ ...@@ -17,8 +17,6 @@
package org.apache.spark.sql.execution.local package org.apache.spark.sql.execution.local
import scala.util.control.NonFatal
import org.apache.spark.Logging import org.apache.spark.Logging
import org.apache.spark.sql.{Row, SQLConf} import org.apache.spark.sql.{Row, SQLConf}
import org.apache.spark.sql.catalyst.{CatalystTypeConverters, InternalRow} import org.apache.spark.sql.catalyst.{CatalystTypeConverters, InternalRow}
...@@ -96,33 +94,13 @@ abstract class LocalNode(conf: SQLConf) extends QueryPlan[LocalNode] with Loggin ...@@ -96,33 +94,13 @@ abstract class LocalNode(conf: SQLConf) extends QueryPlan[LocalNode] with Loggin
inputSchema: Seq[Attribute]): () => MutableProjection = { inputSchema: Seq[Attribute]): () => MutableProjection = {
log.debug( log.debug(
s"Creating MutableProj: $expressions, inputSchema: $inputSchema") s"Creating MutableProj: $expressions, inputSchema: $inputSchema")
try { GenerateMutableProjection.generate(expressions, inputSchema)
GenerateMutableProjection.generate(expressions, inputSchema)
} catch {
case NonFatal(e) =>
if (isTesting) {
throw e
} else {
log.error("Failed to generate mutable projection, fallback to interpreted", e)
() => new InterpretedMutableProjection(expressions, inputSchema)
}
}
} }
protected def newPredicate( protected def newPredicate(
expression: Expression, expression: Expression,
inputSchema: Seq[Attribute]): (InternalRow) => Boolean = { inputSchema: Seq[Attribute]): (InternalRow) => Boolean = {
try { GeneratePredicate.generate(expression, inputSchema)
GeneratePredicate.generate(expression, inputSchema)
} catch {
case NonFatal(e) =>
if (isTesting) {
throw e
} else {
log.error("Failed to generate predicate, fallback to interpreted", e)
InterpretedPredicate.create(expression, inputSchema)
}
}
} }
} }
......
0% Loading or .
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment