From issues-return-394804-archive-asf-public=cust-asf.ponee.io@flink.apache.org Tue Sep 1 16:57:04 2020 Return-Path: X-Original-To: archive-asf-public@cust-asf.ponee.io Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mxout1-he-de.apache.org (mxout1-he-de.apache.org [95.216.194.37]) by mx-eu-01.ponee.io (Postfix) with ESMTPS id 280B4180660 for ; Tue, 1 Sep 2020 18:57:04 +0200 (CEST) Received: from mail.apache.org (mailroute1-lw-us.apache.org [207.244.88.153]) by mxout1-he-de.apache.org (ASF Mail Server at mxout1-he-de.apache.org) with SMTP id 81033635FE for ; Tue, 1 Sep 2020 16:57:03 +0000 (UTC) Received: (qmail 43419 invoked by uid 500); 1 Sep 2020 16:57:02 -0000 Mailing-List: contact issues-help@flink.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@flink.apache.org Delivered-To: mailing list issues@flink.apache.org Received: (qmail 43369 invoked by uid 99); 1 Sep 2020 16:57:02 -0000 Received: from mailrelay1-us-west.apache.org (HELO mailrelay1-us-west.apache.org) (209.188.14.139) by apache.org (qpsmtpd/0.29) with ESMTP; Tue, 01 Sep 2020 16:57:02 +0000 Received: from jira-he-de.apache.org (static.172.67.40.188.clients.your-server.de [188.40.67.172]) by mailrelay1-us-west.apache.org (ASF Mail Server at mailrelay1-us-west.apache.org) with ESMTP id 6F4BD45C58 for ; Tue, 1 Sep 2020 16:57:01 +0000 (UTC) Received: from jira-he-de.apache.org (localhost.localdomain [127.0.0.1]) by jira-he-de.apache.org (ASF Mail Server at jira-he-de.apache.org) with ESMTP id 404AC782274 for ; Tue, 1 Sep 2020 16:57:00 +0000 (UTC) Date: Tue, 1 Sep 2020 16:57:00 +0000 (UTC) From: "Timo Walther (Jira)" To: issues@flink.apache.org Message-ID: In-Reply-To: References: Subject: [jira] [Assigned] (FLINK-19112) No access to metric group in ScalarFunction when optimizing MIME-Version: 1.0 Content-Type: text/plain; charset=utf-8 Content-Transfer-Encoding: 7bit X-JIRA-FingerPrint: 30527f35849b9dde25b450d4833f0394 [ https://issues.apache.org/jira/browse/FLINK-19112?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Timo Walther reassigned FLINK-19112: ------------------------------------ Assignee: Timo Walther > No access to metric group in ScalarFunction when optimizing > ----------------------------------------------------------- > > Key: FLINK-19112 > URL: https://issues.apache.org/jira/browse/FLINK-19112 > Project: Flink > Issue Type: Bug > Components: Table SQL / Runtime > Affects Versions: 1.11.1 > Reporter: Nico Kruber > Assignee: Timo Walther > Priority: Major > Attachments: MetricsGroupBug.java > > > Under some circumstances, I cannot access {{context.getMetricGroup()}} in a {{ScalarFunction}} like this (full job attached): > {code:java} > public static class MyUDF extends ScalarFunction { > @Override > public void open(FunctionContext context) throws Exception { > super.open(context); > context.getMetricGroup(); > } > public Integer eval(Integer id) { > return id; > } > } > {code} > which leads to this exception: > {code:java} > Exception in thread "main" java.lang.UnsupportedOperationException: getMetricGroup is not supported when optimizing > at org.apache.flink.table.planner.codegen.ConstantFunctionContext.getMetricGroup(ExpressionReducer.scala:249) > at com.ververica.MetricsGroupBug$MyUDF.open(MetricsGroupBug.java:57) > at ExpressionReducer$2.open(Unknown Source) > at org.apache.flink.table.planner.codegen.ExpressionReducer.reduce(ExpressionReducer.scala:118) > at org.apache.calcite.rel.rules.ReduceExpressionsRule.reduceExpressionsInternal(ReduceExpressionsRule.java:696) > at org.apache.calcite.rel.rules.ReduceExpressionsRule.reduceExpressions(ReduceExpressionsRule.java:618) > at org.apache.calcite.rel.rules.ReduceExpressionsRule$ProjectReduceExpressionsRule.onMatch(ReduceExpressionsRule.java:303) > at org.apache.calcite.plan.AbstractRelOptPlanner.fireRule(AbstractRelOptPlanner.java:328) > at org.apache.calcite.plan.hep.HepPlanner.applyRule(HepPlanner.java:562) > at org.apache.calcite.plan.hep.HepPlanner.applyRules(HepPlanner.java:427) > at org.apache.calcite.plan.hep.HepPlanner.executeInstruction(HepPlanner.java:264) > at org.apache.calcite.plan.hep.HepInstruction$RuleInstance.execute(HepInstruction.java:127) > at org.apache.calcite.plan.hep.HepPlanner.executeProgram(HepPlanner.java:223) > at org.apache.calcite.plan.hep.HepPlanner.findBestExp(HepPlanner.java:210) > at org.apache.flink.table.planner.plan.optimize.program.FlinkHepProgram.optimize(FlinkHepProgram.scala:69) > at org.apache.flink.table.planner.plan.optimize.program.FlinkHepRuleSetProgram.optimize(FlinkHepRuleSetProgram.scala:87) > at org.apache.flink.table.planner.plan.optimize.program.FlinkChainedProgram.$anonfun$optimize$1(FlinkChainedProgram.scala:62) > at scala.collection.TraversableOnce.$anonfun$foldLeft$1(TraversableOnce.scala:156) > at scala.collection.TraversableOnce.$anonfun$foldLeft$1$adapted(TraversableOnce.scala:156) > at scala.collection.Iterator.foreach(Iterator.scala:937) > at scala.collection.Iterator.foreach$(Iterator.scala:937) > at scala.collection.AbstractIterator.foreach(Iterator.scala:1425) > at scala.collection.IterableLike.foreach(IterableLike.scala:70) > at scala.collection.IterableLike.foreach$(IterableLike.scala:69) > at scala.collection.AbstractIterable.foreach(Iterable.scala:54) > at scala.collection.TraversableOnce.foldLeft(TraversableOnce.scala:156) > at scala.collection.TraversableOnce.foldLeft$(TraversableOnce.scala:154) > at scala.collection.AbstractTraversable.foldLeft(Traversable.scala:104) > at org.apache.flink.table.planner.plan.optimize.program.FlinkChainedProgram.optimize(FlinkChainedProgram.scala:58) > at org.apache.flink.table.planner.plan.optimize.StreamCommonSubGraphBasedOptimizer.optimizeTree(StreamCommonSubGraphBasedOptimizer.scala:164) > at org.apache.flink.table.planner.plan.optimize.StreamCommonSubGraphBasedOptimizer.doOptimize(StreamCommonSubGraphBasedOptimizer.scala:84) > at org.apache.flink.table.planner.plan.optimize.CommonSubGraphBasedOptimizer.optimize(CommonSubGraphBasedOptimizer.scala:77) > at org.apache.flink.table.planner.delegation.PlannerBase.optimize(PlannerBase.scala:279) > at org.apache.flink.table.planner.delegation.PlannerBase.translate(PlannerBase.scala:164) > at org.apache.flink.table.api.internal.TableEnvironmentImpl.translate(TableEnvironmentImpl.java:1264) > at org.apache.flink.table.api.internal.TableEnvironmentImpl.executeInternal(TableEnvironmentImpl.java:700) > at org.apache.flink.table.api.internal.TableImpl.executeInsert(TableImpl.java:565) > at org.apache.flink.table.api.internal.TableImpl.executeInsert(TableImpl.java:549) > at com.ververica.MetricsGroupBug.main(MetricsGroupBug.java:50) > {code} > I also tried to work around this with a try-catch, assuming that this method is called once during optimisation and another time at runtime. However, it seems as if {{open()}} is actually only called once (during optimization) thus giving me no choice to access the metrics group. > It seems that removing the where condition before my UDF call also fixes it but it shouldn't be that way... -- This message was sent by Atlassian Jira (v8.3.4#803005)