|
34 | 34 | import com.google.common.collect.Lists; |
35 | 35 | import com.google.common.collect.Maps; |
36 | 36 | import com.google.common.collect.Sets; |
37 | | -import org.apache.calcite.sql.*; |
| 37 | +import org.apache.calcite.sql.SqlBasicCall; |
| 38 | +import org.apache.calcite.sql.SqlIdentifier; |
| 39 | +import org.apache.calcite.sql.SqlKind; |
| 40 | +import org.apache.calcite.sql.SqlNode; |
| 41 | +import org.apache.calcite.sql.SqlSelect; |
| 42 | +import org.apache.calcite.sql.SqlWithItem; |
38 | 43 | import org.apache.calcite.sql.parser.SqlParseException; |
39 | 44 | import org.apache.commons.collections.CollectionUtils; |
40 | 45 | import org.apache.commons.lang3.StringUtils; |
| 46 | +import org.apache.flink.api.common.typeinfo.BasicTypeInfo; |
41 | 47 | import org.apache.flink.api.common.typeinfo.TypeInformation; |
42 | 48 | import org.apache.flink.api.java.typeutils.RowTypeInfo; |
43 | 49 | import org.apache.flink.streaming.api.datastream.DataStream; |
|
47 | 53 | import org.apache.flink.table.api.java.StreamTableEnvironment; |
48 | 54 | import org.apache.flink.table.catalog.ObjectIdentifier; |
49 | 55 | import org.apache.flink.table.runtime.typeutils.BaseRowTypeInfo; |
| 56 | +import org.apache.flink.table.runtime.typeutils.BigDecimalTypeInfo; |
| 57 | +import org.apache.flink.table.types.logical.DecimalType; |
| 58 | +import org.apache.flink.table.types.logical.LegacyTypeInformationType; |
50 | 59 | import org.apache.flink.table.types.logical.LogicalType; |
51 | 60 | import org.apache.flink.table.typeutils.TimeIndicatorTypeInfo; |
52 | 61 | import org.apache.flink.types.Row; |
53 | 62 | import org.slf4j.Logger; |
54 | 63 | import org.slf4j.LoggerFactory; |
55 | 64 |
|
56 | 65 | import java.time.LocalDateTime; |
57 | | -import java.util.*; |
58 | | - |
59 | | -import static org.apache.calcite.sql.SqlKind.*; |
| 66 | +import java.util.Arrays; |
| 67 | +import java.util.LinkedList; |
| 68 | +import java.util.List; |
| 69 | +import java.util.Map; |
| 70 | +import java.util.Queue; |
| 71 | +import java.util.Set; |
| 72 | + |
| 73 | +import static org.apache.calcite.sql.SqlKind.AS; |
| 74 | +import static org.apache.calcite.sql.SqlKind.INSERT; |
| 75 | +import static org.apache.calcite.sql.SqlKind.SELECT; |
| 76 | +import static org.apache.calcite.sql.SqlKind.WITH_ITEM; |
60 | 77 |
|
61 | 78 | /** |
62 | 79 | * Reason: |
@@ -279,10 +296,7 @@ private Table getTableFromCache(Map<String, Table> localTableCache, String table |
279 | 296 | */ |
280 | 297 | private boolean checkJoinCondition(SqlNode conditionNode, String sideTableAlias, AbstractSideTableInfo sideTableInfo) { |
281 | 298 | List<String> conditionFields = getConditionFields(conditionNode, sideTableAlias, sideTableInfo); |
282 | | - if (CollectionUtils.isEqualCollection(conditionFields, convertPrimaryAlias(sideTableInfo))) { |
283 | | - return true; |
284 | | - } |
285 | | - return false; |
| 299 | + return CollectionUtils.isEqualCollection(conditionFields, convertPrimaryAlias(sideTableInfo)); |
286 | 300 | } |
287 | 301 |
|
288 | 302 | private List<String> convertPrimaryAlias(AbstractSideTableInfo sideTableInfo) { |
@@ -367,8 +381,12 @@ private void joinFun(Object pollObj, |
367 | 381 |
|
368 | 382 | int length = leftTable.getSchema().getFieldDataTypes().length; |
369 | 383 | LogicalType[] logicalTypes = new LogicalType[length]; |
370 | | - for(int i=0; i<length; i++){ |
| 384 | + for (int i = 0; i < length; i++) { |
371 | 385 | logicalTypes[i] = leftTable.getSchema().getFieldDataTypes()[i].getLogicalType(); |
| 386 | + if (logicalTypes[i] instanceof LegacyTypeInformationType && |
| 387 | + ((LegacyTypeInformationType<?>) logicalTypes[i]).getTypeInformation().getClass().equals(BigDecimalTypeInfo.class)) { |
| 388 | + logicalTypes[i] = new DecimalType(38, 18); |
| 389 | + } |
372 | 390 | } |
373 | 391 |
|
374 | 392 | BaseRowTypeInfo leftBaseTypeInfo = new BaseRowTypeInfo(logicalTypes, leftTable.getSchema().getFieldNames()); |
@@ -407,7 +425,14 @@ private void joinFun(Object pollObj, |
407 | 425 | targetTable = localTableCache.get(joinInfo.getLeftTableName()); |
408 | 426 | } |
409 | 427 |
|
410 | | - RowTypeInfo typeInfo = new RowTypeInfo(targetTable.getSchema().getFieldTypes(), targetTable.getSchema().getFieldNames()); |
| 428 | + TypeInformation[] fieldDataTypes = targetTable.getSchema().getFieldTypes(); |
| 429 | + for (int i = 0; i < fieldDataTypes.length; i++) { |
| 430 | + if (fieldDataTypes[i].getClass().equals(BigDecimalTypeInfo.class)) { |
| 431 | + fieldDataTypes[i] = BasicTypeInfo.BIG_DEC_TYPE_INFO; |
| 432 | + } |
| 433 | + } |
| 434 | + |
| 435 | + RowTypeInfo typeInfo = new RowTypeInfo(fieldDataTypes, targetTable.getSchema().getFieldNames()); |
411 | 436 |
|
412 | 437 | DataStream adaptStream = tableEnv.toRetractStream(targetTable, typeInfo) |
413 | 438 | .filter(f -> f.f0) |
@@ -479,9 +504,7 @@ private boolean checkFieldsInfo(CreateTmpTableParser.SqlParserResult result, Tab |
479 | 504 | String fieldType = filed[filed.length - 1].trim(); |
480 | 505 | Class fieldClass = ClassUtil.stringConvertClass(fieldType); |
481 | 506 | Class tableField = table.getSchema().getFieldType(i).get().getTypeClass(); |
482 | | - if (fieldClass == tableField) { |
483 | | - continue; |
484 | | - } else { |
| 507 | + if (fieldClass != tableField) { |
485 | 508 | return false; |
486 | 509 | } |
487 | 510 | } |
|
0 commit comments