Github user JoshRosen commented on the pull request: https://github.com/apache/spark/pull/9093#issuecomment-147807925 Digging into the test failures here, it looks like a bunch of them are caused by the BindReferences call inside of `AggregationIterator`: ```scala [info] Caused by: org.apache.spark.sql.catalyst.errors.package$TreeNodeException: makeCopy, tree: hyperloglogplusplus(a#4,0.04) [info] at org.apache.spark.sql.catalyst.errors.package$.attachTree(package.scala:49) [info] at org.apache.spark.sql.catalyst.trees.TreeNode.makeCopy(TreeNode.scala:315) [info] at org.apache.spark.sql.catalyst.trees.TreeNode.transformChildren(TreeNode.scala:280) [info] at org.apache.spark.sql.catalyst.trees.TreeNode.transformDown(TreeNode.scala:232) [info] at org.apache.spark.sql.catalyst.trees.TreeNode.transform(TreeNode.scala:217) [info] at org.apache.spark.sql.catalyst.expressions.BindReferences$.bindReference(BoundAttribute.scala:85) [info] at org.apache.spark.sql.execution.aggregate.AggregationIterator.<init>(AggregationIterator.scala:93) [info] at org.apache.spark.sql.execution.aggregate.SortBasedAggregationIterator.<init>(SortBasedAggregationIterator.scala:29) [info] at org.apache.spark.sql.execution.aggregate.SortBasedAggregate$$anonfun$doExecute$1$$anonfun$2.apply(SortBasedAggregate.scala:86) [info] at org.apache.spark.sql.execution.aggregate.SortBasedAggregate$$anonfun$doExecute$1$$anonfun$2.apply(SortBasedAggregate.scala:72) [info] at org.apache.spark.rdd.RDD$$anonfun$mapPartitions$1$$anonfun$apply$20.apply(RDD.scala:700) [info] at org.apache.spark.rdd.RDD$$anonfun$mapPartitions$1$$anonfun$apply$20.apply(RDD.scala:700) [info] at org.apache.spark.rdd.MapPartitionsRDD.compute(MapPartitionsRDD.scala:38) [info] at org.apache.spark.rdd.RDD.computeOrReadCheckpoint(RDD.scala:297) [info] at org.apache.spark.rdd.RDD.iterator(RDD.scala:264) [info] at org.apache.spark.rdd.MapPartitionsRDD.compute(MapPartitionsRDD.scala:38) [info] at org.apache.spark.rdd.RDD.computeOrReadCheckpoint(RDD.scala:297) [info] at org.apache.spark.rdd.RDD.iterator(RDD.scala:264) [info] at org.apache.spark.rdd.MapPartitionsRDD.compute(MapPartitionsRDD.scala:38) [info] at org.apache.spark.rdd.RDD.computeOrReadCheckpoint(RDD.scala:297) [info] at org.apache.spark.rdd.RDD.iterator(RDD.scala:264) [info] at org.apache.spark.scheduler.ShuffleMapTask.runTask(ShuffleMapTask.scala:73) [info] at org.apache.spark.scheduler.ShuffleMapTask.runTask(ShuffleMapTask.scala:41) [info] at org.apache.spark.scheduler.Task.run(Task.scala:88) [info] at org.apache.spark.executor.Executor$TaskRunner.run(Executor.scala:214) [info] ... 3 more [info] Caused by: java.lang.reflect.InvocationTargetException [info] at sun.reflect.NativeConstructorAccessorImpl.newInstance0(Native Method) [info] at sun.reflect.NativeConstructorAccessorImpl.newInstance(NativeConstructorAccessorImpl.java:57) [info] at sun.reflect.DelegatingConstructorAccessorImpl.newInstance(DelegatingConstructorAccessorImpl.java:45) [info] at java.lang.reflect.Constructor.newInstance(Constructor.java:526) [info] at org.apache.spark.sql.catalyst.trees.TreeNode$$anonfun$makeCopy$1$$anonfun$apply$10.apply(TreeNode.scala:326) [info] at org.apache.spark.sql.catalyst.trees.TreeNode$$anonfun$makeCopy$1$$anonfun$apply$10.apply(TreeNode.scala:325) [info] at org.apache.spark.sql.catalyst.trees.CurrentOrigin$.withOrigin(TreeNode.scala:51) [info] at org.apache.spark.sql.catalyst.trees.TreeNode$$anonfun$makeCopy$1.apply(TreeNode.scala:323) [info] at org.apache.spark.sql.catalyst.trees.TreeNode$$anonfun$makeCopy$1.apply(TreeNode.scala:315) [info] at org.apache.spark.sql.catalyst.errors.package$.attachTree(package.scala:48) [info] ... 27 more [info] Caused by: java.lang.IllegalStateException: Expression ids should not be allocated inside of tasks [info] at org.apache.spark.sql.catalyst.expressions.NamedExpression$.newExprId(namedExpressions.scala:30) [info] at org.apache.spark.sql.catalyst.expressions.AttributeReference$.apply$default$5(namedExpressions.scala:184) [info] at org.apache.spark.sql.catalyst.expressions.aggregate.HyperLogLogPlusPlus$$anonfun$2.apply(functions.scala:546) [info] at org.apache.spark.sql.catalyst.expressions.aggregate.HyperLogLogPlusPlus$$anonfun$2.apply(functions.scala:545) [info] at scala.collection.generic.GenTraversableFactory.tabulate(GenTraversableFactory.scala:149) [info] at org.apache.spark.sql.catalyst.expressions.aggregate.HyperLogLogPlusPlus.<init>(functions.scala:545) [info] ... 37 more (QueryTest.scala:78) ``` Here, it looks like the copying of `HyperLogLogPlus` is causing problems because that ends up changing the expressionIds of its `aggBufferAttributes`.
--- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. --- --------------------------------------------------------------------- To unsubscribe, e-mail: reviews-unsubscr...@spark.apache.org For additional commands, e-mail: reviews-h...@spark.apache.org