[ 
https://issues.apache.org/jira/browse/FLINK-8173?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16272512#comment-16272512
 ] 

Fabian Hueske commented on FLINK-8173:
--------------------------------------

Yes, you should change {{toAppendStream[org.apache.avro.util.Utf8]}} to 
{{toAppendStream\[java.lang.CharSequence\]}} or perform a manual casting with a 
scalar UDF. 
The support for Avro objects should be improved by converting `CharSequence` to 
`String` to be able to use it as a VARCHAR (String) field, but then you'd get 
the same error message but with String instead of CharSequence (see FLINK-8175)

Btw. I tried to reproduce the error with Flink 1.4.0 and got a much better 
error message.

> InvalidProgramException: Table program cannot be compiled. This is a bug. 
> Please file an issue.
> -----------------------------------------------------------------------------------------------
>
>                 Key: FLINK-8173
>                 URL: https://issues.apache.org/jira/browse/FLINK-8173
>             Project: Flink
>          Issue Type: Bug
>            Reporter: Tao Xia
>
> It is a stream of Avro objects, simply select a String field and trying to 
> print out
>     val query = "SELECT nd_key FROM table1"
>     val result = tableEnv.sql(query)
>     tableEnv.toAppendStream[org.apache.avro.util.Utf8](result).print()
> 11/29/2017 16:07:36   Source: Custom Source -> from: (accepted_cohort_id, 
> admin_id, after_submission, amount_paid, anonymous_id, application_id, 
> atom_key, bd_group_key, biz_geo, braavos_purchase_id, category, cohort_id, 
> concept_key, concept_rank, context, context_campaign, context_experiment, 
> coupon_code, course_key, course_rank, cta_destination, cta_location, 
> cta_message, cta_type, currency, decision_group_id, device_browser, 
> device_os, device_os_version, device_type, duration, evaluation_id, 
> event_type, fin_geo, in_collaboration_with, lab_id, lab_rank, label, 
> lesson_key, lesson_rank, locale, max_pause_duration, message, message_id, 
> module_key, module_rank, nd_key, nd_unit_id, nd_unit_rank, new_cohort_id, 
> notification_id, num_concepts_completed, num_interactions, 
> num_lessons_completed, old_cohort_id, part_key, part_rank, pause_duration, 
> pause_reason, payment_plan, payment_provider, points_earned, points_possible, 
> price, price_sheet, product_key, product_type, provider_charge_id, 
> provider_refund_id, quiz_type, referrer, refund_amount, requested_cohort_id, 
> results, scholarship_group_key, search_term, skill_level, subscription_id, 
> suspension_length, suspension_reason, technology, timestamp, total_concepts, 
> total_lessons, total_time_sec, type, unenroll_reason, user_id, user_locale, 
> user_response, variant, version, workspace_id, workspace_session, 
> workspace_type) -> select: (nd_key) -> to: Utf8 -> Sink: Unnamed(5/8) 
> switched to FAILED 
> org.apache.flink.api.common.InvalidProgramException: Table program cannot be 
> compiled. This is a bug. Please file an issue.
>       at 
> org.apache.flink.table.codegen.Compiler$class.compile(Compiler.scala:36)
>       at 
> org.apache.flink.table.runtime.CRowOutputMapRunner.compile(CRowOutputMapRunner.scala:33)
>       at 
> org.apache.flink.table.runtime.CRowOutputMapRunner.open(CRowOutputMapRunner.scala:48)
>       at 
> org.apache.flink.api.common.functions.util.FunctionUtils.openFunction(FunctionUtils.java:36)
>       at 
> org.apache.flink.streaming.api.operators.AbstractUdfStreamOperator.open(AbstractUdfStreamOperator.java:111)
>       at 
> org.apache.flink.streaming.runtime.tasks.StreamTask.openAllOperators(StreamTask.java:376)
>       at 
> org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:253)
>       at org.apache.flink.runtime.taskmanager.Task.run(Task.java:702)
>       at java.lang.Thread.run(Thread.java:748)
> Caused by: org.codehaus.commons.compiler.CompileException: Line 790, Column 
> 15: Assignment conversion not possible from type "java.lang.CharSequence" to 
> type "org.apache.avro.util.Utf8"
>       at 
> org.codehaus.janino.UnitCompiler.compileError(UnitCompiler.java:11672)
>       at 
> org.codehaus.janino.UnitCompiler.assignmentConversion(UnitCompiler.java:10528)
>       at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:2534)
>       at org.codehaus.janino.UnitCompiler.access$2600(UnitCompiler.java:212)
>       at 
> org.codehaus.janino.UnitCompiler$6.visitLocalVariableDeclarationStatement(UnitCompiler.java:1459)
>       at 
> org.codehaus.janino.UnitCompiler$6.visitLocalVariableDeclarationStatement(UnitCompiler.java:1443)
>       at 
> org.codehaus.janino.Java$LocalVariableDeclarationStatement.accept(Java.java:3348)
>       at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:1443)
>       at 
> org.codehaus.janino.UnitCompiler.compileStatements(UnitCompiler.java:1523)
>       at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:3052)
>       at 
> org.codehaus.janino.UnitCompiler.compileDeclaredMethods(UnitCompiler.java:1313)
>       at 
> org.codehaus.janino.UnitCompiler.compileDeclaredMethods(UnitCompiler.java:1286)
>       at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:785)
>       at org.codehaus.janino.UnitCompiler.compile2(UnitCompiler.java:436)
>       at org.codehaus.janino.UnitCompiler.access$400(UnitCompiler.java:212)
>       at 
> org.codehaus.janino.UnitCompiler$2.visitPackageMemberClassDeclaration(UnitCompiler.java:390)
>       at 
> org.codehaus.janino.UnitCompiler$2.visitPackageMemberClassDeclaration(UnitCompiler.java:385)
>       at 
> org.codehaus.janino.Java$PackageMemberClassDeclaration.accept(Java.java:1405)
>       at org.codehaus.janino.UnitCompiler.compile(UnitCompiler.java:385)
>       at org.codehaus.janino.UnitCompiler.compileUnit(UnitCompiler.java:357)
>       at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:234)
>       at 
> org.codehaus.janino.SimpleCompiler.compileToClassLoader(SimpleCompiler.java:446)
>       at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:213)
>       at org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:204)
>       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.codegen.Compiler$class.compile(Compiler.scala:33)
>       ... 8 more



--
This message was sent by Atlassian JIRA
(v6.4.14#64029)

Reply via email to