You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@flink.apache.org by "Dawid Wysakowicz (Jira)" <ji...@apache.org> on 2020/04/14 16:33:00 UTC

[jira] [Created] (FLINK-17144) Input type generalization does not work correctly for ARRAY and MAP in sql

Dawid Wysakowicz created FLINK-17144:
----------------------------------------

             Summary: Input type generalization does not work correctly for ARRAY and MAP in sql
                 Key: FLINK-17144
                 URL: https://issues.apache.org/jira/browse/FLINK-17144
             Project: Flink
          Issue Type: Bug
          Components: Table SQL / Planner
            Reporter: Dawid Wysakowicz


Query: {{"SELECT * FROM (VALUES (ARRAY[1, 2.0])) T(a)"}}
fails with:
{code}
/* 1 */
/* 2 */      public class StreamExecCalc$6 extends org.apache.flink.table.runtime.operators.AbstractProcessStreamOperator
/* 3 */          implements org.apache.flink.streaming.api.operators.OneInputStreamOperator {
/* 4 */
/* 5 */        private final Object[] references;
/* 6 */        
/* 7 */        org.apache.flink.table.dataformat.Decimal decimal$3 =
/* 8 */            org.apache.flink.table.dataformat.Decimal.castFrom("2.0", 2, 1);
/* 9 */        
/* 10 */        
/* 11 */        org.apache.flink.table.dataformat.BinaryArray array$4 = new org.apache.flink.table.dataformat.BinaryArray();
/* 12 */        org.apache.flink.table.dataformat.BinaryArrayWriter writer$5 = new org.apache.flink.table.dataformat.BinaryArrayWriter(array$4, 2, 8);
/* 13 */               
/* 14 */        final org.apache.flink.table.dataformat.BoxedWrapperRow out = new org.apache.flink.table.dataformat.BoxedWrapperRow(1);
/* 15 */        private final org.apache.flink.streaming.runtime.streamrecord.StreamRecord outElement = new org.apache.flink.streaming.runtime.streamrecord.StreamRecord(null);
/* 16 */
/* 17 */        public StreamExecCalc$6(
/* 18 */            Object[] references,
/* 19 */            org.apache.flink.streaming.runtime.tasks.StreamTask task,
/* 20 */            org.apache.flink.streaming.api.graph.StreamConfig config,
/* 21 */            org.apache.flink.streaming.api.operators.Output output,
/* 22 */            org.apache.flink.streaming.runtime.tasks.ProcessingTimeService processingTimeService) throws Exception {
/* 23 */          this.references = references;
/* 24 */          
/* 25 */          writer$5.reset();
/* 26 */          
/* 27 */          
/* 28 */          if (false) {
/* 29 */            writer$5.setNullLong(0);
/* 30 */          } else {
/* 31 */            writer$5.writeDecimal(0, ((int) 1), 11);
/* 32 */          }
/* 33 */                    
/* 34 */          
/* 35 */          
/* 36 */          if (false) {
/* 37 */            writer$5.setNullLong(1);
/* 38 */          } else {
/* 39 */            writer$5.writeDecimal(1, ((org.apache.flink.table.dataformat.Decimal) decimal$3), 11);
/* 40 */          }
/* 41 */                    
/* 42 */          writer$5.complete();
/* 43 */                   
/* 44 */          this.setup(task, config, output);
/* 45 */          if (this instanceof org.apache.flink.streaming.api.operators.AbstractStreamOperator) {
/* 46 */            ((org.apache.flink.streaming.api.operators.AbstractStreamOperator) this)
/* 47 */              .setProcessingTimeService(processingTimeService);
/* 48 */          }
/* 49 */        }
/* 50 */
/* 51 */        @Override
/* 52 */        public void open() throws Exception {
/* 53 */          super.open();
/* 54 */          
/* 55 */        }
/* 56 */
/* 57 */        @Override
/* 58 */        public void processElement(org.apache.flink.streaming.runtime.streamrecord.StreamRecord element) throws Exception {
/* 59 */          org.apache.flink.table.dataformat.BaseRow in1 = (org.apache.flink.table.dataformat.BaseRow) element.getValue();
/* 60 */          
/* 61 */          
/* 62 */          
/* 63 */          
/* 64 */          
/* 65 */          
/* 66 */          out.setHeader(in1.getHeader());
/* 67 */          
/* 68 */          
/* 69 */          
/* 70 */          
/* 71 */          if (false) {
/* 72 */            out.setNullAt(0);
/* 73 */          } else {
/* 74 */            out.setNonPrimitiveValue(0, array$4);
/* 75 */          }
/* 76 */                    
/* 77 */                  
/* 78 */          output.collect(outElement.replace(out));
/* 79 */          
/* 80 */          
/* 81 */        }
/* 82 */
/* 83 */        
/* 84 */
/* 85 */        @Override
/* 86 */        public void close() throws Exception {
/* 87 */           super.close();
/* 88 */          
/* 89 */        }
/* 90 */
/* 91 */        
/* 92 */      }
/* 93 */    


org.apache.flink.runtime.client.JobExecutionException: Job execution failed.

	at org.apache.flink.runtime.jobmaster.JobResult.toJobExecutionResult(JobResult.java:147)
	at org.apache.flink.runtime.minicluster.MiniCluster.executeJobBlocking(MiniCluster.java:659)
	at org.apache.flink.streaming.util.TestStreamEnvironment.execute(TestStreamEnvironment.java:77)
	at org.apache.flink.streaming.api.environment.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.java:1643)
	at org.apache.flink.streaming.api.environment.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.java:1625)
	at org.apache.flink.streaming.api.scala.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.scala:673)
	at org.apache.flink.table.planner.runtime.stream.sql.ValuesITCase.testValues2(ValuesITCase.scala:63)
	at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
	at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
	at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
	at java.lang.reflect.Method.invoke(Method.java:498)
	at org.junit.runners.model.FrameworkMethod$1.runReflectiveCall(FrameworkMethod.java:50)
	at org.junit.internal.runners.model.ReflectiveCallable.run(ReflectiveCallable.java:12)
	at org.junit.runners.model.FrameworkMethod.invokeExplosively(FrameworkMethod.java:47)
	at org.junit.internal.runners.statements.InvokeMethod.evaluate(InvokeMethod.java:17)
	at org.junit.internal.runners.statements.RunBefores.evaluate(RunBefores.java:26)
	at org.junit.rules.ExpectedException$ExpectedExceptionStatement.evaluate(ExpectedException.java:239)
	at org.junit.rules.ExternalResource$1.evaluate(ExternalResource.java:48)
	at org.junit.rules.TestWatcher$1.evaluate(TestWatcher.java:55)
	at org.junit.rules.RunRules.evaluate(RunRules.java:20)
	at org.junit.runners.ParentRunner.runLeaf(ParentRunner.java:325)
	at org.junit.runners.BlockJUnit4ClassRunner.runChild(BlockJUnit4ClassRunner.java:78)
	at org.junit.runners.BlockJUnit4ClassRunner.runChild(BlockJUnit4ClassRunner.java:57)
	at org.junit.runners.ParentRunner$3.run(ParentRunner.java:290)
	at org.junit.runners.ParentRunner$1.schedule(ParentRunner.java:71)
	at org.junit.runners.ParentRunner.runChildren(ParentRunner.java:288)
	at org.junit.runners.ParentRunner.access$000(ParentRunner.java:58)
	at org.junit.runners.ParentRunner$2.evaluate(ParentRunner.java:268)
	at org.junit.rules.ExternalResource$1.evaluate(ExternalResource.java:48)
	at org.junit.rules.ExternalResource$1.evaluate(ExternalResource.java:48)
	at org.junit.rules.RunRules.evaluate(RunRules.java:20)
	at org.junit.runners.ParentRunner.run(ParentRunner.java:363)
	at org.junit.runner.JUnitCore.run(JUnitCore.java:137)
	at com.intellij.junit4.JUnit4IdeaTestRunner.startRunnerWithArgs(JUnit4IdeaTestRunner.java:68)
	at com.intellij.rt.junit.IdeaTestRunner$Repeater.startRunnerWithArgs(IdeaTestRunner.java:33)
	at com.intellij.rt.junit.JUnitStarter.prepareStreamsAndStart(JUnitStarter.java:230)
	at com.intellij.rt.junit.JUnitStarter.main(JUnitStarter.java:58)
	at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
	at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
	at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
	at java.lang.reflect.Method.invoke(Method.java:498)
	at com.intellij.rt.execution.application.AppMainV2.main(AppMainV2.java:128)
Caused by: org.apache.flink.runtime.JobException: Recovery is suppressed by NoRestartBackoffTimeStrategy
	at org.apache.flink.runtime.executiongraph.failover.flip1.ExecutionFailureHandler.handleFailure(ExecutionFailureHandler.java:112)
	at org.apache.flink.runtime.executiongraph.failover.flip1.ExecutionFailureHandler.getFailureHandlingResult(ExecutionFailureHandler.java:78)
	at org.apache.flink.runtime.scheduler.DefaultScheduler.handleTaskFailure(DefaultScheduler.java:189)
	at org.apache.flink.runtime.scheduler.DefaultScheduler.maybeHandleTaskFailure(DefaultScheduler.java:183)
	at org.apache.flink.runtime.scheduler.DefaultScheduler.updateTaskExecutionStateInternal(DefaultScheduler.java:177)
	at org.apache.flink.runtime.scheduler.SchedulerBase.updateTaskExecutionState(SchedulerBase.java:497)
	at org.apache.flink.runtime.jobmaster.JobMaster.updateTaskExecutionState(JobMaster.java:384)
	at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
	at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
	at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
	at java.lang.reflect.Method.invoke(Method.java:498)
	at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleRpcInvocation(AkkaRpcActor.java:284)
	at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleRpcMessage(AkkaRpcActor.java:199)
	at org.apache.flink.runtime.rpc.akka.FencedAkkaRpcActor.handleRpcMessage(FencedAkkaRpcActor.java:74)
	at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleMessage(AkkaRpcActor.java:152)
	at akka.japi.pf.UnitCaseStatement.apply(CaseStatements.scala:26)
	at akka.japi.pf.UnitCaseStatement.apply(CaseStatements.scala:21)
	at scala.PartialFunction$class.applyOrElse(PartialFunction.scala:123)
	at akka.japi.pf.UnitCaseStatement.applyOrElse(CaseStatements.scala:21)
	at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:170)
	at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:171)
	at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:171)
	at akka.actor.Actor$class.aroundReceive(Actor.scala:517)
	at akka.actor.AbstractActor.aroundReceive(AbstractActor.scala:225)
	at akka.actor.ActorCell.receiveMessage(ActorCell.scala:592)
	at akka.actor.ActorCell.invoke(ActorCell.scala:561)
	at akka.dispatch.Mailbox.processMailbox(Mailbox.scala:258)
	at akka.dispatch.Mailbox.run(Mailbox.scala:225)
	at akka.dispatch.Mailbox.exec(Mailbox.scala:235)
	at akka.dispatch.forkjoin.ForkJoinTask.doExec(ForkJoinTask.java:260)
	at akka.dispatch.forkjoin.ForkJoinPool$WorkQueue.runTask(ForkJoinPool.java:1339)
	at akka.dispatch.forkjoin.ForkJoinPool.runWorker(ForkJoinPool.java:1979)
	at akka.dispatch.forkjoin.ForkJoinWorkerThread.run(ForkJoinWorkerThread.java:107)
Caused by: java.lang.RuntimeException: Could not instantiate generated class 'StreamExecCalc$6'
	at org.apache.flink.table.runtime.generated.GeneratedClass.newInstance(GeneratedClass.java:67)
	at org.apache.flink.table.runtime.operators.CodeGenOperatorFactory.createStreamOperator(CodeGenOperatorFactory.java:40)
	at org.apache.flink.streaming.api.operators.StreamOperatorFactoryUtil.createOperator(StreamOperatorFactoryUtil.java:61)
	at org.apache.flink.streaming.runtime.tasks.OperatorChain.createChainedOperator(OperatorChain.java:473)
	at org.apache.flink.streaming.runtime.tasks.OperatorChain.createOutputCollector(OperatorChain.java:396)
	at org.apache.flink.streaming.runtime.tasks.OperatorChain.<init>(OperatorChain.java:157)
	at org.apache.flink.streaming.runtime.tasks.StreamTask.beforeInvoke(StreamTask.java:425)
	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:462)
	at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:718)
	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:542)
	at java.lang.Thread.run(Thread.java:748)
Caused by: org.apache.flink.util.FlinkRuntimeException: org.apache.flink.api.common.InvalidProgramException: Table program cannot be compiled. This is a bug. Please file an issue.
	at org.apache.flink.table.runtime.generated.CompileUtils.compile(CompileUtils.java:68)
	at org.apache.flink.table.runtime.generated.GeneratedClass.compile(GeneratedClass.java:78)
	at org.apache.flink.table.runtime.generated.GeneratedClass.newInstance(GeneratedClass.java:65)
	... 10 more
Caused by: org.apache.flink.shaded.guava18.com.google.common.util.concurrent.UncheckedExecutionException: org.apache.flink.api.common.InvalidProgramException: Table program cannot be compiled. This is a bug. Please file an issue.
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.get(LocalCache.java:2203)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache.get(LocalCache.java:3937)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$LocalManualCache.get(LocalCache.java:4739)
	at org.apache.flink.table.runtime.generated.CompileUtils.compile(CompileUtils.java:66)
	... 12 more
Caused by: org.apache.flink.api.common.InvalidProgramException: Table program cannot be compiled. This is a bug. Please file an issue.
	at org.apache.flink.table.runtime.generated.CompileUtils.doCompile(CompileUtils.java:81)
	at org.apache.flink.table.runtime.generated.CompileUtils.lambda$compile$1(CompileUtils.java:66)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$LocalManualCache$1.load(LocalCache.java:4742)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$LoadingValueReference.loadFuture(LocalCache.java:3527)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.loadSync(LocalCache.java:2319)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.lockedGetOrLoad(LocalCache.java:2282)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.get(LocalCache.java:2197)
	... 15 more
Caused by: org.codehaus.commons.compiler.CompileException: Line 31, Column 35: No applicable constructor/method found for actual parameters "int, int, int"; candidates are: "public void org.apache.flink.table.dataformat.AbstractBinaryWriter.writeDecimal(int, org.apache.flink.table.dataformat.Decimal, int)", "public abstract void org.apache.flink.table.dataformat.BinaryWriter.writeDecimal(int, org.apache.flink.table.dataformat.Decimal, int)"
	at org.codehaus.janino.UnitCompiler.compileError(UnitCompiler.java:12124)
	at org.codehaus.janino.UnitCompiler.findMostSpecificIInvocable(UnitCompiler.java:9176)
	at org.codehaus.janino.UnitCompiler.findIMethod(UnitCompiler.java:9036)
	at org.codehaus.janino.UnitCompiler.findIMethod(UnitCompiler.java:8938)
	at org.codehaus.janino.UnitCompiler.compileGet2(UnitCompiler.java:5060)
	at org.codehaus.janino.UnitCompiler.access$9100(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$16.visitMethodInvocation(UnitCompiler.java:4421)
	at org.codehaus.janino.UnitCompiler$16.visitMethodInvocation(UnitCompiler.java:4394)
	at org.codehaus.janino.Java$MethodInvocation.accept(Java.java:5062)
	at org.codehaus.janino.UnitCompiler.compileGet(UnitCompiler.java:4394)
	at org.codehaus.janino.UnitCompiler.compileGetValue(UnitCompiler.java:5575)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:3781)
	at org.codehaus.janino.UnitCompiler.access$5900(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$13.visitMethodInvocation(UnitCompiler.java:3760)
	at org.codehaus.janino.UnitCompiler$13.visitMethodInvocation(UnitCompiler.java:3732)
	at org.codehaus.janino.Java$MethodInvocation.accept(Java.java:5062)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:3732)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:2360)
	at org.codehaus.janino.UnitCompiler.access$1800(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$6.visitExpressionStatement(UnitCompiler.java:1494)
	at org.codehaus.janino.UnitCompiler$6.visitExpressionStatement(UnitCompiler.java:1487)
	at org.codehaus.janino.Java$ExpressionStatement.accept(Java.java:2871)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:1487)
	at org.codehaus.janino.UnitCompiler.compileStatements(UnitCompiler.java:1567)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:1553)
	at org.codehaus.janino.UnitCompiler.access$1700(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$6.visitBlock(UnitCompiler.java:1493)
	at org.codehaus.janino.UnitCompiler$6.visitBlock(UnitCompiler.java:1487)
	at org.codehaus.janino.Java$Block.accept(Java.java:2776)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:1487)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:2433)
	at org.codehaus.janino.UnitCompiler.access$1900(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$6.visitIfStatement(UnitCompiler.java:1495)
	at org.codehaus.janino.UnitCompiler$6.visitIfStatement(UnitCompiler.java:1487)
	at org.codehaus.janino.Java$IfStatement.accept(Java.java:2947)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:1487)
	at org.codehaus.janino.UnitCompiler.compileStatements(UnitCompiler.java:1567)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:3388)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:834)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:432)
	at org.codehaus.janino.UnitCompiler.access$400(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$2.visitPackageMemberClassDeclaration(UnitCompiler.java:411)
	at org.codehaus.janino.UnitCompiler$2.visitPackageMemberClassDeclaration(UnitCompiler.java:406)
	at org.codehaus.janino.Java$PackageMemberClassDeclaration.accept(Java.java:1414)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:406)
	at org.codehaus.janino.UnitCompiler.compileUnit(UnitCompiler.java:378)
	at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:237)
	at org.codehaus.janino.SimpleCompiler.compileToClassLoader(SimpleCompiler.java:465)
	at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:216)
	at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:207)
	at org.codehaus.commons.compiler.Cookable.cook(Cookable.java:80)
	at org.codehaus.commons.compiler.Cookable.cook(Cookable.java:75)
	at org.apache.flink.table.runtime.generated.CompileUtils.doCompile(CompileUtils.java:78)
	... 21 more
{code}

Query: {{SELECT * FROM (VALUES (MAP[1, 'ABC', 2.0, 'DEF'])) T(a)}}
fails with:
{code}
/* 1 */
/* 2 */      public class StreamExecCalc$11 extends org.apache.flink.table.runtime.operators.AbstractProcessStreamOperator
/* 3 */          implements org.apache.flink.streaming.api.operators.OneInputStreamOperator {
/* 4 */
/* 5 */        private final Object[] references;
/* 6 */        
/* 7 */        private final org.apache.flink.table.dataformat.BinaryString str$3 = org.apache.flink.table.dataformat.BinaryString.fromString("ABC");
/* 8 */                   
/* 9 */        
/* 10 */        org.apache.flink.table.dataformat.Decimal decimal$4 =
/* 11 */            org.apache.flink.table.dataformat.Decimal.castFrom("2.0", 2, 1);
/* 12 */        
/* 13 */        
/* 14 */        private final org.apache.flink.table.dataformat.BinaryString str$5 = org.apache.flink.table.dataformat.BinaryString.fromString("DEF");
/* 15 */                   
/* 16 */        
/* 17 */        org.apache.flink.table.dataformat.BinaryArray array$7 = new org.apache.flink.table.dataformat.BinaryArray();
/* 18 */        org.apache.flink.table.dataformat.BinaryArrayWriter writer$8 = new org.apache.flink.table.dataformat.BinaryArrayWriter(array$7, 2, 8);
/* 19 */               
/* 20 */        
/* 21 */        org.apache.flink.table.dataformat.BinaryArray array$9 = new org.apache.flink.table.dataformat.BinaryArray();
/* 22 */        org.apache.flink.table.dataformat.BinaryArrayWriter writer$10 = new org.apache.flink.table.dataformat.BinaryArrayWriter(array$9, 2, 8);
/* 23 */               
/* 24 */        org.apache.flink.table.dataformat.BaseMap map$6 = null;
/* 25 */        final org.apache.flink.table.dataformat.BoxedWrapperRow out = new org.apache.flink.table.dataformat.BoxedWrapperRow(1);
/* 26 */        private final org.apache.flink.streaming.runtime.streamrecord.StreamRecord outElement = new org.apache.flink.streaming.runtime.streamrecord.StreamRecord(null);
/* 27 */
/* 28 */        public StreamExecCalc$11(
/* 29 */            Object[] references,
/* 30 */            org.apache.flink.streaming.runtime.tasks.StreamTask task,
/* 31 */            org.apache.flink.streaming.api.graph.StreamConfig config,
/* 32 */            org.apache.flink.streaming.api.operators.Output output,
/* 33 */            org.apache.flink.streaming.runtime.tasks.ProcessingTimeService processingTimeService) throws Exception {
/* 34 */          this.references = references;
/* 35 */          
/* 36 */          writer$8.reset();
/* 37 */          
/* 38 */          
/* 39 */          if (false) {
/* 40 */            writer$8.setNullLong(0);
/* 41 */          } else {
/* 42 */            writer$8.writeDecimal(0, ((int) 1), 11);
/* 43 */          }
/* 44 */                    
/* 45 */          
/* 46 */          
/* 47 */          if (false) {
/* 48 */            writer$8.setNullLong(1);
/* 49 */          } else {
/* 50 */            writer$8.writeDecimal(1, ((org.apache.flink.table.dataformat.Decimal) decimal$4), 11);
/* 51 */          }
/* 52 */                    
/* 53 */          writer$8.complete();
/* 54 */                   
/* 55 */          
/* 56 */          writer$10.reset();
/* 57 */          
/* 58 */          
/* 59 */          if (false) {
/* 60 */            writer$10.setNullLong(0);
/* 61 */          } else {
/* 62 */            writer$10.writeString(0, ((org.apache.flink.table.dataformat.BinaryString) str$3));
/* 63 */          }
/* 64 */                    
/* 65 */          
/* 66 */          
/* 67 */          if (false) {
/* 68 */            writer$10.setNullLong(1);
/* 69 */          } else {
/* 70 */            writer$10.writeString(1, ((org.apache.flink.table.dataformat.BinaryString) str$5));
/* 71 */          }
/* 72 */                    
/* 73 */          writer$10.complete();
/* 74 */                   
/* 75 */          this.setup(task, config, output);
/* 76 */          if (this instanceof org.apache.flink.streaming.api.operators.AbstractStreamOperator) {
/* 77 */            ((org.apache.flink.streaming.api.operators.AbstractStreamOperator) this)
/* 78 */              .setProcessingTimeService(processingTimeService);
/* 79 */          }
/* 80 */        }
/* 81 */
/* 82 */        @Override
/* 83 */        public void open() throws Exception {
/* 84 */          super.open();
/* 85 */          
/* 86 */        }
/* 87 */
/* 88 */        @Override
/* 89 */        public void processElement(org.apache.flink.streaming.runtime.streamrecord.StreamRecord element) throws Exception {
/* 90 */          org.apache.flink.table.dataformat.BaseRow in1 = (org.apache.flink.table.dataformat.BaseRow) element.getValue();
/* 91 */          
/* 92 */          
/* 93 */          
/* 94 */          
/* 95 */          
/* 96 */          
/* 97 */          out.setHeader(in1.getHeader());
/* 98 */          
/* 99 */          
/* 100 */          
/* 101 */          
/* 102 */          
/* 103 */          
/* 104 */          map$6 = org.apache.flink.table.dataformat.BinaryMap.valueOf(array$7, array$9);
/* 105 */                 
/* 106 */          if (false) {
/* 107 */            out.setNullAt(0);
/* 108 */          } else {
/* 109 */            out.setNonPrimitiveValue(0, map$6);
/* 110 */          }
/* 111 */                    
/* 112 */                  
/* 113 */          output.collect(outElement.replace(out));
/* 114 */          
/* 115 */          
/* 116 */        }
/* 117 */
/* 118 */        
/* 119 */
/* 120 */        @Override
/* 121 */        public void close() throws Exception {
/* 122 */           super.close();
/* 123 */          
/* 124 */        }
/* 125 */
/* 126 */        
/* 127 */      }
/* 128 */    


org.apache.flink.runtime.client.JobExecutionException: Job execution failed.

	at org.apache.flink.runtime.jobmaster.JobResult.toJobExecutionResult(JobResult.java:147)
	at org.apache.flink.runtime.minicluster.MiniCluster.executeJobBlocking(MiniCluster.java:659)
	at org.apache.flink.streaming.util.TestStreamEnvironment.execute(TestStreamEnvironment.java:77)
	at org.apache.flink.streaming.api.environment.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.java:1643)
	at org.apache.flink.streaming.api.environment.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.java:1625)
	at org.apache.flink.streaming.api.scala.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.scala:673)
	at org.apache.flink.table.planner.runtime.stream.sql.ValuesITCase.testValues2(ValuesITCase.scala:63)
	at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
	at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
	at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
	at java.lang.reflect.Method.invoke(Method.java:498)
	at org.junit.runners.model.FrameworkMethod$1.runReflectiveCall(FrameworkMethod.java:50)
	at org.junit.internal.runners.model.ReflectiveCallable.run(ReflectiveCallable.java:12)
	at org.junit.runners.model.FrameworkMethod.invokeExplosively(FrameworkMethod.java:47)
	at org.junit.internal.runners.statements.InvokeMethod.evaluate(InvokeMethod.java:17)
	at org.junit.internal.runners.statements.RunBefores.evaluate(RunBefores.java:26)
	at org.junit.rules.ExpectedException$ExpectedExceptionStatement.evaluate(ExpectedException.java:239)
	at org.junit.rules.ExternalResource$1.evaluate(ExternalResource.java:48)
	at org.junit.rules.TestWatcher$1.evaluate(TestWatcher.java:55)
	at org.junit.rules.RunRules.evaluate(RunRules.java:20)
	at org.junit.runners.ParentRunner.runLeaf(ParentRunner.java:325)
	at org.junit.runners.BlockJUnit4ClassRunner.runChild(BlockJUnit4ClassRunner.java:78)
	at org.junit.runners.BlockJUnit4ClassRunner.runChild(BlockJUnit4ClassRunner.java:57)
	at org.junit.runners.ParentRunner$3.run(ParentRunner.java:290)
	at org.junit.runners.ParentRunner$1.schedule(ParentRunner.java:71)
	at org.junit.runners.ParentRunner.runChildren(ParentRunner.java:288)
	at org.junit.runners.ParentRunner.access$000(ParentRunner.java:58)
	at org.junit.runners.ParentRunner$2.evaluate(ParentRunner.java:268)
	at org.junit.rules.ExternalResource$1.evaluate(ExternalResource.java:48)
	at org.junit.rules.ExternalResource$1.evaluate(ExternalResource.java:48)
	at org.junit.rules.RunRules.evaluate(RunRules.java:20)
	at org.junit.runners.ParentRunner.run(ParentRunner.java:363)
	at org.junit.runner.JUnitCore.run(JUnitCore.java:137)
	at com.intellij.junit4.JUnit4IdeaTestRunner.startRunnerWithArgs(JUnit4IdeaTestRunner.java:68)
	at com.intellij.rt.junit.IdeaTestRunner$Repeater.startRunnerWithArgs(IdeaTestRunner.java:33)
	at com.intellij.rt.junit.JUnitStarter.prepareStreamsAndStart(JUnitStarter.java:230)
	at com.intellij.rt.junit.JUnitStarter.main(JUnitStarter.java:58)
	at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
	at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
	at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
	at java.lang.reflect.Method.invoke(Method.java:498)
	at com.intellij.rt.execution.application.AppMainV2.main(AppMainV2.java:128)
Caused by: org.apache.flink.runtime.JobException: Recovery is suppressed by NoRestartBackoffTimeStrategy
	at org.apache.flink.runtime.executiongraph.failover.flip1.ExecutionFailureHandler.handleFailure(ExecutionFailureHandler.java:112)
	at org.apache.flink.runtime.executiongraph.failover.flip1.ExecutionFailureHandler.getFailureHandlingResult(ExecutionFailureHandler.java:78)
	at org.apache.flink.runtime.scheduler.DefaultScheduler.handleTaskFailure(DefaultScheduler.java:189)
	at org.apache.flink.runtime.scheduler.DefaultScheduler.maybeHandleTaskFailure(DefaultScheduler.java:183)
	at org.apache.flink.runtime.scheduler.DefaultScheduler.updateTaskExecutionStateInternal(DefaultScheduler.java:177)
	at org.apache.flink.runtime.scheduler.SchedulerBase.updateTaskExecutionState(SchedulerBase.java:497)
	at org.apache.flink.runtime.jobmaster.JobMaster.updateTaskExecutionState(JobMaster.java:384)
	at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
	at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
	at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
	at java.lang.reflect.Method.invoke(Method.java:498)
	at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleRpcInvocation(AkkaRpcActor.java:284)
	at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleRpcMessage(AkkaRpcActor.java:199)
	at org.apache.flink.runtime.rpc.akka.FencedAkkaRpcActor.handleRpcMessage(FencedAkkaRpcActor.java:74)
	at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleMessage(AkkaRpcActor.java:152)
	at akka.japi.pf.UnitCaseStatement.apply(CaseStatements.scala:26)
	at akka.japi.pf.UnitCaseStatement.apply(CaseStatements.scala:21)
	at scala.PartialFunction$class.applyOrElse(PartialFunction.scala:123)
	at akka.japi.pf.UnitCaseStatement.applyOrElse(CaseStatements.scala:21)
	at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:170)
	at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:171)
	at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:171)
	at akka.actor.Actor$class.aroundReceive(Actor.scala:517)
	at akka.actor.AbstractActor.aroundReceive(AbstractActor.scala:225)
	at akka.actor.ActorCell.receiveMessage(ActorCell.scala:592)
	at akka.actor.ActorCell.invoke(ActorCell.scala:561)
	at akka.dispatch.Mailbox.processMailbox(Mailbox.scala:258)
	at akka.dispatch.Mailbox.run(Mailbox.scala:225)
	at akka.dispatch.Mailbox.exec(Mailbox.scala:235)
	at akka.dispatch.forkjoin.ForkJoinTask.doExec(ForkJoinTask.java:260)
	at akka.dispatch.forkjoin.ForkJoinPool$WorkQueue.runTask(ForkJoinPool.java:1339)
	at akka.dispatch.forkjoin.ForkJoinPool.runWorker(ForkJoinPool.java:1979)
	at akka.dispatch.forkjoin.ForkJoinWorkerThread.run(ForkJoinWorkerThread.java:107)
Caused by: java.lang.RuntimeException: Could not instantiate generated class 'StreamExecCalc$11'
	at org.apache.flink.table.runtime.generated.GeneratedClass.newInstance(GeneratedClass.java:67)
	at org.apache.flink.table.runtime.operators.CodeGenOperatorFactory.createStreamOperator(CodeGenOperatorFactory.java:40)
	at org.apache.flink.streaming.api.operators.StreamOperatorFactoryUtil.createOperator(StreamOperatorFactoryUtil.java:61)
	at org.apache.flink.streaming.runtime.tasks.OperatorChain.createChainedOperator(OperatorChain.java:473)
	at org.apache.flink.streaming.runtime.tasks.OperatorChain.createOutputCollector(OperatorChain.java:396)
	at org.apache.flink.streaming.runtime.tasks.OperatorChain.<init>(OperatorChain.java:157)
	at org.apache.flink.streaming.runtime.tasks.StreamTask.beforeInvoke(StreamTask.java:425)
	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:462)
	at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:718)
	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:542)
	at java.lang.Thread.run(Thread.java:748)
Caused by: org.apache.flink.util.FlinkRuntimeException: org.apache.flink.api.common.InvalidProgramException: Table program cannot be compiled. This is a bug. Please file an issue.
	at org.apache.flink.table.runtime.generated.CompileUtils.compile(CompileUtils.java:68)
	at org.apache.flink.table.runtime.generated.GeneratedClass.compile(GeneratedClass.java:78)
	at org.apache.flink.table.runtime.generated.GeneratedClass.newInstance(GeneratedClass.java:65)
	... 10 more
Caused by: org.apache.flink.shaded.guava18.com.google.common.util.concurrent.UncheckedExecutionException: org.apache.flink.api.common.InvalidProgramException: Table program cannot be compiled. This is a bug. Please file an issue.
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.get(LocalCache.java:2203)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache.get(LocalCache.java:3937)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$LocalManualCache.get(LocalCache.java:4739)
	at org.apache.flink.table.runtime.generated.CompileUtils.compile(CompileUtils.java:66)
	... 12 more
Caused by: org.apache.flink.api.common.InvalidProgramException: Table program cannot be compiled. This is a bug. Please file an issue.
	at org.apache.flink.table.runtime.generated.CompileUtils.doCompile(CompileUtils.java:81)
	at org.apache.flink.table.runtime.generated.CompileUtils.lambda$compile$1(CompileUtils.java:66)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$LocalManualCache$1.load(LocalCache.java:4742)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$LoadingValueReference.loadFuture(LocalCache.java:3527)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.loadSync(LocalCache.java:2319)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.lockedGetOrLoad(LocalCache.java:2282)
	at org.apache.flink.shaded.guava18.com.google.common.cache.LocalCache$Segment.get(LocalCache.java:2197)
	... 15 more
Caused by: org.codehaus.commons.compiler.CompileException: Line 42, Column 35: No applicable constructor/method found for actual parameters "int, int, int"; candidates are: "public void org.apache.flink.table.dataformat.AbstractBinaryWriter.writeDecimal(int, org.apache.flink.table.dataformat.Decimal, int)", "public abstract void org.apache.flink.table.dataformat.BinaryWriter.writeDecimal(int, org.apache.flink.table.dataformat.Decimal, int)"
	at org.codehaus.janino.UnitCompiler.compileError(UnitCompiler.java:12124)
	at org.codehaus.janino.UnitCompiler.findMostSpecificIInvocable(UnitCompiler.java:9176)
	at org.codehaus.janino.UnitCompiler.findIMethod(UnitCompiler.java:9036)
	at org.codehaus.janino.UnitCompiler.findIMethod(UnitCompiler.java:8938)
	at org.codehaus.janino.UnitCompiler.compileGet2(UnitCompiler.java:5060)
	at org.codehaus.janino.UnitCompiler.access$9100(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$16.visitMethodInvocation(UnitCompiler.java:4421)
	at org.codehaus.janino.UnitCompiler$16.visitMethodInvocation(UnitCompiler.java:4394)
	at org.codehaus.janino.Java$MethodInvocation.accept(Java.java:5062)
	at org.codehaus.janino.UnitCompiler.compileGet(UnitCompiler.java:4394)
	at org.codehaus.janino.UnitCompiler.compileGetValue(UnitCompiler.java:5575)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:3781)
	at org.codehaus.janino.UnitCompiler.access$5900(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$13.visitMethodInvocation(UnitCompiler.java:3760)
	at org.codehaus.janino.UnitCompiler$13.visitMethodInvocation(UnitCompiler.java:3732)
	at org.codehaus.janino.Java$MethodInvocation.accept(Java.java:5062)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:3732)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:2360)
	at org.codehaus.janino.UnitCompiler.access$1800(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$6.visitExpressionStatement(UnitCompiler.java:1494)
	at org.codehaus.janino.UnitCompiler$6.visitExpressionStatement(UnitCompiler.java:1487)
	at org.codehaus.janino.Java$ExpressionStatement.accept(Java.java:2871)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:1487)
	at org.codehaus.janino.UnitCompiler.compileStatements(UnitCompiler.java:1567)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:1553)
	at org.codehaus.janino.UnitCompiler.access$1700(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$6.visitBlock(UnitCompiler.java:1493)
	at org.codehaus.janino.UnitCompiler$6.visitBlock(UnitCompiler.java:1487)
	at org.codehaus.janino.Java$Block.accept(Java.java:2776)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:1487)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:2433)
	at org.codehaus.janino.UnitCompiler.access$1900(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$6.visitIfStatement(UnitCompiler.java:1495)
	at org.codehaus.janino.UnitCompiler$6.visitIfStatement(UnitCompiler.java:1487)
	at org.codehaus.janino.Java$IfStatement.accept(Java.java:2947)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:1487)
	at org.codehaus.janino.UnitCompiler.compileStatements(UnitCompiler.java:1567)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:3388)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:834)
	at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:432)
	at org.codehaus.janino.UnitCompiler.access$400(UnitCompiler.java:215)
	at org.codehaus.janino.UnitCompiler$2.visitPackageMemberClassDeclaration(UnitCompiler.java:411)
	at org.codehaus.janino.UnitCompiler$2.visitPackageMemberClassDeclaration(UnitCompiler.java:406)
	at org.codehaus.janino.Java$PackageMemberClassDeclaration.accept(Java.java:1414)
	at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:406)
	at org.codehaus.janino.UnitCompiler.compileUnit(UnitCompiler.java:378)
	at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:237)
	at org.codehaus.janino.SimpleCompiler.compileToClassLoader(SimpleCompiler.java:465)
	at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:216)
	at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:207)
	at org.codehaus.commons.compiler.Cookable.cook(Cookable.java:80)
	at org.codehaus.commons.compiler.Cookable.cook(Cookable.java:75)
	at org.apache.flink.table.runtime.generated.CompileUtils.doCompile(CompileUtils.java:78)
	... 21 more
{code}



--
This message was sent by Atlassian Jira
(v8.3.4#803005)