pnowojski commented on a change in pull request #6787: [FLINK-8577][table] Implement proctime DataStream to Table upsert conversion URL: https://github.com/apache/flink/pull/6787#discussion_r253546583
########## File path: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/nodes/datastream/StreamScan.scala ########## @@ -19,22 +19,138 @@ package org.apache.flink.table.plan.nodes.datastream import org.apache.calcite.rex.RexNode -import org.apache.flink.api.common.functions.{MapFunction, RichMapFunction} +import org.apache.flink.api.common.functions.RichMapFunction import org.apache.flink.api.common.typeinfo.TypeInformation +import org.apache.flink.api.java.tuple.{Tuple2 => JTuple2} import org.apache.flink.configuration.Configuration import org.apache.flink.streaming.api.datastream.DataStream import org.apache.flink.streaming.api.functions.ProcessFunction -import org.apache.flink.table.api.TableConfig +import org.apache.flink.table.api.{TableConfig, Types} import org.apache.flink.table.codegen.{FunctionCodeGenerator, GeneratedFunction} import org.apache.flink.table.plan.nodes.CommonScan import org.apache.flink.table.plan.schema.RowSchema import org.apache.flink.types.Row -import org.apache.flink.table.runtime.CRowOutputProcessRunner import org.apache.flink.table.runtime.types.{CRow, CRowTypeInfo} import org.apache.flink.table.typeutils.TimeIndicatorTypeInfo +import java.lang.{Boolean => JBool} + +import org.apache.flink.api.java.typeutils.TupleTypeInfo +import org.apache.flink.api.scala.typeutils.CaseClassTypeInfo +import org.apache.flink.table.runtime.conversion.{ExternalTypeToCRowProcessRunner, JavaTupleToCRowProcessRunner, ScalaTupleToCRowProcessRunner} trait StreamScan extends CommonScan[CRow] with DataStreamRel { + protected def convertUpsertToInternalRow( Review comment: Couple of remarks regarding this method: 1. Why does it have this `switch/case` for scala/java tuples while the `convertToInternalRow` doesn't? 2. I'm not sure, but It looks like it's duplicating logic from `convertToInternalRow`, maybe they should be deduplicated into one method and passing the `upsert/append` mode as a parameter for a couple of `if` statements? 3. Please split it into smaller methods and you could do the same for `convertToInternalRow`. More or less where ever there is a comment like `// Scala tuple` or `// input is already of correct type. Only need to wrap it as CRow` extract the matching piece of code to a named method like `convertScalaTuple....(...)` or `wrapAsCRow(...)` 4. are all of the branches covered by the tests? Like the both branches of `if` under the java tuple? ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services