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 @@ -29,29 +29,27 @@ class EquivalentExpressions {
* Wrapper around an Expression that provides semantic equality.
*/
case class Expr(e: Expression) {
val hash = e.semanticHash()
override def equals(o: Any): Boolean = o match {
case other: Expr => e.semanticEquals(other.e)
case _ => false
}
override def hashCode: Int = hash
override val hashCode: Int = e.semanticHash()
}

// For each expression, the set of equivalent expressions.
private val equivalenceMap: mutable.HashMap[Expr, mutable.MutableList[Expression]] =
new mutable.HashMap[Expr, mutable.MutableList[Expression]]
private val equivalenceMap = mutable.HashMap.empty[Expr, mutable.MutableList[Expression]]

/**
* Adds each expression to this data structure, grouping them with existing equivalent
* expressions. Non-recursive.
* Returns if there was already a matching expression.
* Returns true if there was already a matching expression.
*/
def addExpr(expr: Expression): Boolean = {
if (expr.deterministic) {
val e: Expr = Expr(expr)
val f = equivalenceMap.get(e)
if (f.isDefined) {
f.get.+= (expr)
f.get += expr
true
} else {
equivalenceMap.put(e, mutable.MutableList(expr))
Expand All @@ -63,19 +61,19 @@ class EquivalentExpressions {
}

/**
* Adds the expression to this datastructure recursively. Stops if a matching expression
* Adds the expression to this data structure recursively. Stops if a matching expression
* is found. That is, if `expr` has already been added, its children are not added.
* If ignoreLeaf is true, leaf nodes are ignored.
*/
def addExprTree(root: Expression, ignoreLeaf: Boolean = true): Unit = {
val skip = root.isInstanceOf[LeafExpression] && ignoreLeaf
if (!skip && root.deterministic && !addExpr(root)) {
if (!skip && !addExpr(root)) {
root.children.foreach(addExprTree(_, ignoreLeaf))
Copy link
Contributor

Choose a reason for hiding this comment

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

indent one more space? I will check this when I merge it.

}
}

/**
* Returns all fo the expression trees that are equivalent to `e`. Returns
* Returns all of the expression trees that are equivalent to `e`. Returns
* an empty collection if there are none.
*/
def getEquivalentExprs(e: Expression): Seq[Expression] = {
Expand All @@ -90,8 +88,8 @@ class EquivalentExpressions {
}

/**
* Returns the state of the datastructure as a string. If all is false, skips sets of equivalent
* expressions with cardinality 1.
* Returns the state of the data structure as a string. If `all` is false, skips sets of
* equivalent expressions with cardinality 1.
*/
def debugString(all: Boolean = false): String = {
val sb: mutable.StringBuilder = new StringBuilder()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -92,17 +92,16 @@ abstract class Expression extends TreeNode[Expression] {
* @return [[GeneratedExpressionCode]]
*/
def gen(ctx: CodeGenContext): GeneratedExpressionCode = {
val subExprState = ctx.subExprEliminationExprs.get(this)
if (subExprState.isDefined) {
ctx.subExprEliminationExprs.get(this).map { subExprState =>
// This expression is repeated meaning the code to evaluated has already been added
// as a function, `subExprState.fnName`. Just call that.
val code =
s"""
|/* $this */
|${subExprState.get.fnName}(${ctx.INPUT_ROW});
|""".stripMargin.trim
GeneratedExpressionCode(code, subExprState.get.code.isNull, subExprState.get.code.value)
} else {
|${subExprState.fnName}(${ctx.INPUT_ROW});
""".stripMargin.trim
GeneratedExpressionCode(code, subExprState.code.isNull, subExprState.code.value)
}.getOrElse {
val isNull = ctx.freshName("isNull")
val primitive = ctx.freshName("primitive")
val ve = GeneratedExpressionCode("", isNull, primitive)
Expand Down Expand Up @@ -157,7 +156,7 @@ abstract class Expression extends TreeNode[Expression] {
case (i1, i2) => i1 == i2
}
}
// Non-determinstic expressions cannot be equal
// Non-deterministic expressions cannot be semantic equal
if (!deterministic || !other.deterministic) return false
val elements1 = this.productIterator.toSeq
val elements2 = other.asInstanceOf[Product].productIterator.toSeq
Expand All @@ -174,11 +173,11 @@ abstract class Expression extends TreeNode[Expression] {
var hash: Int = 17
e.foreach(i => {
val h: Int = i match {
case (e: Expression) => e.semanticHash()
case (Some(e: Expression)) => e.semanticHash()
case (t: Traversable[_]) => computeHash(t.toSeq)
case e: Expression => e.semanticHash()
case Some(e: Expression) => e.semanticHash()
case t: Traversable[_] => computeHash(t.toSeq)
case null => 0
case (o) => o.hashCode()
case other => other.hashCode()
}
hash = hash * 37 + h
})
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -109,15 +109,15 @@ class CodeGenContext {

// State used for subexpression elimination.
case class SubExprEliminationState(
val isLoaded: String, code: GeneratedExpressionCode, val fnName: String)
isLoaded: String,
code: GeneratedExpressionCode,
fnName: String)

// Foreach expression that is participating in subexpression elimination, the state to use.
val subExprEliminationExprs: mutable.HashMap[Expression, SubExprEliminationState] =
mutable.HashMap[Expression, SubExprEliminationState]()
val subExprEliminationExprs = mutable.HashMap.empty[Expression, SubExprEliminationState]

// The collection of isLoaded variables that need to be reset on each row.
val subExprIsLoadedVariables: mutable.ArrayBuffer[String] =
mutable.ArrayBuffer.empty[String]
val subExprIsLoadedVariables = mutable.ArrayBuffer.empty[String]

final val JAVA_BOOLEAN = "boolean"
final val JAVA_BYTE = "byte"
Expand Down Expand Up @@ -361,7 +361,7 @@ class CodeGenContext {
val expr = e.head
val isLoaded = freshName("isLoaded")
val isNull = freshName("isNull")
val primitive = freshName("primitive")
val value = freshName("value")
val fnName = freshName("evalExpr")

// Generate the code for this expression tree and wrap it in a function.
Expand All @@ -373,13 +373,13 @@ class CodeGenContext {
| ${code.code.trim}
| $isLoaded = true;
| $isNull = ${code.isNull};
| $primitive = ${code.value};
| $value = ${code.value};
| }
|}
""".stripMargin
code.code = fn
code.isNull = isNull
code.value = primitive
code.value = value

addNewFunction(fnName, fn)

Expand All @@ -406,8 +406,8 @@ class CodeGenContext {
// optimizations.
addMutableState("boolean", isLoaded, s"$isLoaded = false;")
addMutableState("boolean", isNull, s"$isNull = false;")
addMutableState(javaType(expr.dataType), primitive,
s"$primitive = ${defaultValue(expr.dataType)};")
addMutableState(javaType(expr.dataType), value,
s"$value = ${defaultValue(expr.dataType)};")
subExprIsLoadedVariables += isLoaded

val state = SubExprEliminationState(isLoaded, code, fnName)
Expand Down