|
| 1 | +/* |
| 2 | + * Licensed to the Apache Software Foundation (ASF) under one |
| 3 | + * or more contributor license agreements. See the NOTICE file |
| 4 | + * distributed with this work for additional information |
| 5 | + * regarding copyright ownership. The ASF licenses this file |
| 6 | + * to you under the Apache License, Version 2.0 (the |
| 7 | + * "License"); you may not use this file except in compliance |
| 8 | + * with the License. You may obtain a copy of the License at |
| 9 | + * |
| 10 | + * http://www.apache.org/licenses/LICENSE-2.0 |
| 11 | + * |
| 12 | + * Unless required by applicable law or agreed to in writing, software |
| 13 | + * distributed under the License is distributed on an "AS IS" BASIS, |
| 14 | + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. |
| 15 | + * See the License for the specific language governing permissions and |
| 16 | + * limitations under the License. |
| 17 | + */ |
| 18 | + |
| 19 | +package org.apache.inlong.sort.parser; |
| 20 | + |
| 21 | +import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; |
| 22 | +import org.apache.flink.table.api.EnvironmentSettings; |
| 23 | +import org.apache.flink.table.api.bridge.java.StreamTableEnvironment; |
| 24 | +import org.apache.flink.test.util.AbstractTestBase; |
| 25 | +import org.apache.inlong.sort.formats.common.IntFormatInfo; |
| 26 | +import org.apache.inlong.sort.formats.common.LongFormatInfo; |
| 27 | +import org.apache.inlong.sort.formats.common.StringFormatInfo; |
| 28 | +import org.apache.inlong.sort.parser.impl.FlinkSqlParser; |
| 29 | +import org.apache.inlong.sort.parser.result.ParseResult; |
| 30 | +import org.apache.inlong.sort.protocol.FieldInfo; |
| 31 | +import org.apache.inlong.sort.protocol.GroupInfo; |
| 32 | +import org.apache.inlong.sort.protocol.StreamInfo; |
| 33 | +import org.apache.inlong.sort.protocol.enums.KafkaScanStartupMode; |
| 34 | +import org.apache.inlong.sort.protocol.node.Node; |
| 35 | +import org.apache.inlong.sort.protocol.node.extract.KafkaExtractNode; |
| 36 | +import org.apache.inlong.sort.protocol.node.format.JsonFormat; |
| 37 | +import org.apache.inlong.sort.protocol.node.load.MySqlLoadNode; |
| 38 | +import org.apache.inlong.sort.protocol.transformation.FieldRelation; |
| 39 | +import org.apache.inlong.sort.protocol.transformation.relation.NodeRelation; |
| 40 | +import org.junit.Assert; |
| 41 | +import org.junit.Test; |
| 42 | + |
| 43 | +import java.util.Arrays; |
| 44 | +import java.util.Collections; |
| 45 | +import java.util.List; |
| 46 | +import java.util.stream.Collectors; |
| 47 | + |
| 48 | +/** |
| 49 | + * Test for kafka sql parse |
| 50 | + */ |
| 51 | +public class KafkaSqlParseTest extends AbstractTestBase { |
| 52 | + |
| 53 | + /** |
| 54 | + * Test flink sql task for extract is kafka {@link KafkaExtractNode} and load is mysql {@link MySqlLoadNode} |
| 55 | + * |
| 56 | + * @throws Exception The exception may be thrown when executing |
| 57 | + */ |
| 58 | + @Test |
| 59 | + public void testKafkaSourceSqlParse() throws Exception { |
| 60 | + StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment(); |
| 61 | + env.setParallelism(1); |
| 62 | + env.enableCheckpointing(10000); |
| 63 | + env.disableOperatorChaining(); |
| 64 | + EnvironmentSettings settings = EnvironmentSettings |
| 65 | + .newInstance() |
| 66 | + .useBlinkPlanner() |
| 67 | + .inStreamingMode() |
| 68 | + .build(); |
| 69 | + StreamTableEnvironment tableEnv = StreamTableEnvironment.create(env, settings); |
| 70 | + Node inputNode = buildKafkaExtractSpecificOffset(); |
| 71 | + Node outputNode = buildMysqlLoadNode(); |
| 72 | + StreamInfo streamInfo = new StreamInfo("1", Arrays.asList(inputNode, outputNode), |
| 73 | + Collections.singletonList(buildNodeRelation(Collections.singletonList(inputNode), |
| 74 | + Collections.singletonList(outputNode)))); |
| 75 | + GroupInfo groupInfo = new GroupInfo("1", Collections.singletonList(streamInfo)); |
| 76 | + FlinkSqlParser parser = FlinkSqlParser.getInstance(tableEnv, groupInfo); |
| 77 | + ParseResult result = parser.parse(); |
| 78 | + Assert.assertTrue(result.tryExecute()); |
| 79 | + } |
| 80 | + |
| 81 | + private KafkaExtractNode buildKafkaExtractSpecificOffset() { |
| 82 | + List<FieldInfo> fields = Arrays.asList(new FieldInfo("id", new LongFormatInfo()), |
| 83 | + new FieldInfo("name", new StringFormatInfo()), |
| 84 | + new FieldInfo("age", new IntFormatInfo())); |
| 85 | + return new KafkaExtractNode("1", "kafka_input", fields, null, |
| 86 | + null, "topic_input", "localhost:9092", |
| 87 | + new JsonFormat(), KafkaScanStartupMode.SPECIFIC_OFFSETS, null, "groupId", |
| 88 | + "partition:0,offset:42;partition:1,offset:300"); |
| 89 | + } |
| 90 | + |
| 91 | + private Node buildMysqlLoadNode() { |
| 92 | + List<FieldInfo> fields = Arrays.asList(new FieldInfo("id", new LongFormatInfo()), |
| 93 | + new FieldInfo("name", new StringFormatInfo()), |
| 94 | + new FieldInfo("age", new IntFormatInfo()) |
| 95 | + ); |
| 96 | + List<FieldRelation> relations = Arrays |
| 97 | + .asList(new FieldRelation(new FieldInfo("id", new LongFormatInfo()), |
| 98 | + new FieldInfo("id", new LongFormatInfo())), |
| 99 | + new FieldRelation(new FieldInfo("name", new StringFormatInfo()), |
| 100 | + new FieldInfo("name", new StringFormatInfo())), |
| 101 | + new FieldRelation(new FieldInfo("age", new IntFormatInfo()), |
| 102 | + new FieldInfo("age", new IntFormatInfo())) |
| 103 | + ); |
| 104 | + return new MySqlLoadNode("2", "mysql_output", fields, relations, null, |
| 105 | + null, null, null, "jdbc:mysql://localhost:3306/inlong", |
| 106 | + "inlong", "inlong", "table_output", "id"); |
| 107 | + } |
| 108 | + |
| 109 | + /** |
| 110 | + * build node relation |
| 111 | + * |
| 112 | + * @param inputs extract node |
| 113 | + * @param outputs load node |
| 114 | + * @return node relation |
| 115 | + */ |
| 116 | + private NodeRelation buildNodeRelation(List<Node> inputs, List<Node> outputs) { |
| 117 | + List<String> inputIds = inputs.stream().map(Node::getId).collect(Collectors.toList()); |
| 118 | + List<String> outputIds = outputs.stream().map(Node::getId).collect(Collectors.toList()); |
| 119 | + return new NodeRelation(inputIds, outputIds); |
| 120 | + } |
| 121 | +} |
0 commit comments