Skip to content
Closed
Prev Previous commit
Next Next commit
address review comments
  • Loading branch information
kiszk committed Nov 30, 2017
commit a455ac3bf5aa49a15ef80fc7807078eb41bbfe74
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@ import java.util.{Calendar, Locale, TimeZone}
import org.apache.spark.SparkFunSuite
import org.apache.spark.sql.Row
import org.apache.spark.sql.catalyst.InternalRow
import org.apache.spark.sql.catalyst.expressions.codegen.CodegenContext
import org.apache.spark.sql.catalyst.util.DateTimeTestUtils._
import org.apache.spark.sql.catalyst.util.DateTimeUtils
import org.apache.spark.sql.catalyst.util.DateTimeUtils.TimeZoneGMT
Expand Down Expand Up @@ -846,23 +847,10 @@ class CastSuite extends SparkFunSuite with ExpressionEvalHelper {
checkEvaluation(cast(Literal.create(inputOuter, fromOuter), toOuter), outputOuter)
}

test("SPARK-22570: should not create a lot of instance variables") {
val N = 30000

val from1 = new StructType(
(1 to N).map(i => StructField(s"s$i", StringType)).toArray)
val to1 = new StructType(
(1 to N).map(i => StructField(s"i$i", IntegerType)).toArray)
val input1 = Row.fromSeq((1 to N).map(i => i.toString))
val output1 = Row.fromSeq((1 to N))
checkEvaluation(cast(Literal.create(input1, from1), to1), output1)

val from2 = new StructType(
(1 to N).map(i => StructField(s"s$i", StringType)).toArray)
val to2 = new StructType(
(1 to N).map(i => StructField(s"i$i", LongType)).toArray)
val input2 = Row.fromSeq((1 to N).map(i => i.toString))
val output2 = Row.fromSeq((1 to N).map(i => i.toLong))
checkEvaluation(cast(Literal.create(input2, from2), to2), output2)
test("SPARK-22570: Cast should not create a lot of instance variables") {
Copy link
Member

Choose a reason for hiding this comment

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

nit: instance variables -> global variables. To match with other two added tests.

val ctx = new CodegenContext
cast("1", IntegerType).genCode(ctx).code
cast("2", LongType).genCode(ctx).code
Copy link
Contributor

Choose a reason for hiding this comment

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

nit: no need to call code

assert(ctx.mutableStates.length == 0)
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -179,14 +179,12 @@ class RegexpExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper {
checkEvaluation(nonNullExpr, "num-num", row1)
}

test("SPARK-22570: should not create a lot of instance variables") {
val N = 16000
val expr = RegExpReplace(Literal("100"), Literal("(\\d+)"), Literal("num"))
test("SPARK-22570: RegExpReplace should not create a lot of global variables") {
val ctx = new CodegenContext
(1 to N).map(_ => expr.genCode(ctx).code)
RegExpReplace(Literal("100"), Literal("(\\d+)"), Literal("num")).genCode(ctx)
// four global variables (lastRegex, pattern, lastReplacement, and lastReplacementInUTF8)
// are always required
assert(ctx.mutableStates.length == 4 * N)
assert(ctx.mutableStates.length == 4)
}

test("RegexExtract") {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -165,9 +165,9 @@ class ComplexTypesSuite extends PlanTest{
comparePlans(Optimizer execute query, expected)
}

test("SPARK-22570: should not create a lot of instance variables") {
test("SPARK-22570: CreateArray should not create a lot of global variables") {
val ctx = new CodegenContext
Copy link
Contributor

Choose a reason for hiding this comment

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

something like this, we have a codegen context, and do codegen. After that, instead of compiling the code to make sure it doesn't fail, we can just check the size of ctx.mutableStates to confirm that we don't add global variables during codegen

(1 to 60000).map(i => CreateArray(Seq(Literal(s"$i"))).genCode(ctx).code)
CreateArray(Seq(Literal(1))).genCode(ctx).code
Copy link
Contributor

Choose a reason for hiding this comment

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

ditto

assert(ctx.mutableStates.length == 0)
}

Expand Down