Skip to content

Commit e506733

Browse files
committed
Merge branch 'v1.5.0_dev_order_by' into 'v1.5.0_dev'
[支持对orderby字段的解析][对orderby解析缺失][18642] 1.5只支持 order by 时间属性,不支持asc desc 1.6 以后支持 desc asc See merge request !65
2 parents f9a373d + d11eca9 commit e506733

File tree

3 files changed

+39
-0
lines changed

3 files changed

+39
-0
lines changed

core/src/main/java/com/dtstack/flink/sql/parser/InsertSqlParser.java

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -26,6 +26,7 @@
2626
import org.apache.calcite.sql.SqlJoin;
2727
import org.apache.calcite.sql.SqlKind;
2828
import org.apache.calcite.sql.SqlNode;
29+
import org.apache.calcite.sql.SqlOrderBy;
2930
import org.apache.calcite.sql.SqlSelect;
3031
import org.apache.calcite.sql.parser.SqlParseException;
3132
import org.apache.calcite.sql.parser.SqlParser;
@@ -132,6 +133,10 @@ private static void parseNode(SqlNode sqlNode, SqlParseResult sqlParseResult){
132133
parseNode(unionRight, sqlParseResult);
133134
}
134135
break;
136+
case ORDER_BY:
137+
SqlOrderBy sqlOrderBy = (SqlOrderBy) sqlNode;
138+
parseNode(sqlOrderBy.query, sqlParseResult);
139+
break;
135140
default:
136141
//do nothing
137142
break;

core/src/main/java/com/dtstack/flink/sql/side/SideSQLParser.java

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -31,6 +31,7 @@
3131
import org.apache.calcite.sql.SqlKind;
3232
import org.apache.calcite.sql.SqlNode;
3333
import org.apache.calcite.sql.SqlOperator;
34+
import org.apache.calcite.sql.SqlOrderBy;
3435
import org.apache.calcite.sql.SqlSelect;
3536
import org.apache.calcite.sql.parser.SqlParseException;
3637
import org.apache.calcite.sql.parser.SqlParser;
@@ -120,6 +121,10 @@ private Object parseSql(SqlNode sqlNode, Set<String> sideTableSet, Queue<Object>
120121
parseSql(unionRight, sideTableSet, queueInfo);
121122

122123
break;
124+
125+
case ORDER_BY:
126+
SqlOrderBy sqlOrderBy = (SqlOrderBy) sqlNode;
127+
parseSql(sqlOrderBy.query, sideTableSet, queueInfo);
123128
}
124129

125130
return "";

core/src/main/java/com/dtstack/flink/sql/side/SideSqlExec.java

Lines changed: 29 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -37,6 +37,7 @@
3737
import org.apache.calcite.sql.SqlNode;
3838
import org.apache.calcite.sql.SqlNodeList;
3939
import org.apache.calcite.sql.SqlOperator;
40+
import org.apache.calcite.sql.SqlOrderBy;
4041
import org.apache.calcite.sql.SqlSelect;
4142
import org.apache.calcite.sql.fun.SqlCase;
4243
import org.apache.calcite.sql.parser.SqlParseException;
@@ -317,11 +318,39 @@ private void replaceFieldName(SqlNode sqlNode, HashBasedTable<String, String, St
317318
replaceFieldName(unionRight, mappingTable, targetTableName, tableAlias);
318319

319320
break;
321+
case ORDER_BY:
322+
SqlOrderBy sqlOrderBy = (SqlOrderBy) sqlNode;
323+
replaceFieldName(sqlOrderBy.query, mappingTable, targetTableName, tableAlias);
324+
SqlNodeList orderFiledList = sqlOrderBy.orderList;
325+
for (int i=0 ;i<orderFiledList.size();i++) {
326+
SqlNode replaceNode = replaceOrderByTableName(orderFiledList.get(i), tableAlias);
327+
orderFiledList.set(i, replaceNode);
328+
}
329+
320330
default:
321331
break;
322332
}
323333
}
324334

335+
private SqlNode replaceOrderByTableName(SqlNode orderNode, String tableAlias) {
336+
if(orderNode.getKind() == IDENTIFIER){
337+
SqlIdentifier sqlIdentifier = (SqlIdentifier) orderNode;
338+
if (sqlIdentifier.names.size() == 1) {
339+
return orderNode;
340+
}
341+
return sqlIdentifier.setName(0, tableAlias);
342+
} else if (orderNode instanceof SqlBasicCall) {
343+
SqlBasicCall sqlBasicCall = (SqlBasicCall) orderNode;
344+
for(int i=0; i<sqlBasicCall.getOperandList().size(); i++){
345+
SqlNode sqlNode = sqlBasicCall.getOperandList().get(i);
346+
sqlBasicCall.getOperands()[i] = replaceOrderByTableName(sqlNode , tableAlias);
347+
}
348+
return sqlBasicCall;
349+
} else {
350+
return orderNode;
351+
}
352+
}
353+
325354
private SqlNode replaceNodeInfo(SqlNode groupNode, HashBasedTable<String, String, String> mappingTable, String tableAlias){
326355
if(groupNode.getKind() == IDENTIFIER){
327356
SqlIdentifier sqlIdentifier = (SqlIdentifier) groupNode;

0 commit comments

Comments
 (0)