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

ASF GitHub Bot commented on FLINK-6196:
---------------------------------------

Github user KurtYoung commented on a diff in the pull request:

    https://github.com/apache/flink/pull/3623#discussion_r108320419
  
    --- Diff: 
flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/schema/FlinkTableFunctionImpl.scala
 ---
    @@ -27,52 +27,73 @@ import 
org.apache.flink.api.common.typeinfo.{AtomicType, TypeInformation}
     import org.apache.flink.api.common.typeutils.CompositeType
     import org.apache.flink.table.api.TableException
     import org.apache.flink.table.calcite.FlinkTypeFactory
    +import org.apache.flink.table.functions.utils.UserDefinedFunctionUtils
    +import org.apache.flink.table.functions.{TableFunction => FlinkUDTF}
     
     /**
       * This is heavily inspired by Calcite's 
[[org.apache.calcite.schema.impl.TableFunctionImpl]].
       * We need it in order to create a 
[[org.apache.flink.table.functions.utils.TableSqlFunction]].
       * The main difference is that we override the [[getRowType()]] and 
[[getElementType()]].
    +  *
    +  * @param tableFunction The Table Function instance
    +  * @param implicitResultType The implicit result type.
    +  * @param evalMethod The eval() method of the [[tableFunction]]
       */
    -class FlinkTableFunctionImpl[T](
    -    val typeInfo: TypeInformation[T],
    -    val fieldIndexes: Array[Int],
    -    val fieldNames: Array[String],
    -    val evalMethod: Method)
    +class FlinkTableFunctionImpl[T](val tableFunction: FlinkUDTF[_],
    --- End diff --
    
    Can we just separate this implementation into 2 classes, one for which we 
already know the type, and one for whose type has not been decided yet. 
    Why i suggesting this is that i noticed `tableFunction.getResultType` is 
called multiple times when evaluating the plan, even we had already determine 
the row type long time ago, i.e. when `apply` method from `TableFunction` has 
been called. 



> Support dynamic schema in Table Function
> ----------------------------------------
>
>                 Key: FLINK-6196
>                 URL: https://issues.apache.org/jira/browse/FLINK-6196
>             Project: Flink
>          Issue Type: Improvement
>          Components: Table API & SQL
>            Reporter: Zhuoluo Yang
>            Assignee: Zhuoluo Yang
>
> In many of our use cases. We have to decide the schema of a UDTF at the run 
> time. For example. udtf('c1, c2, c3') will generate three columns for a 
> lateral view. 
> Most systems such as calcite and hive support this feature. However, the 
> current implementation of flink didn't implement the feature correctly.



--
This message was sent by Atlassian JIRA
(v6.3.15#6346)

Reply via email to