This is an automated email from the ASF dual-hosted git repository.

richox pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/auron.git


The following commit(s) were added to refs/heads/master by this push:
     new acbb7b7e [AURON #1387] Introduce auron-flink-extension module (#1389)
acbb7b7e is described below

commit acbb7b7ecc3087a98f77bc1aad0ef87f1c11b3cc
Author: zhangmang <[email protected]>
AuthorDate: Tue Sep 30 20:18:09 2025 +0800

    [AURON #1387] Introduce auron-flink-extension module (#1389)
    
    * [AURON #1387] Introduce auron-flink-extension module
    
    * add license
---
 auron-build.sh                                     |  14 ++
 auron-flink-extension/pom.xml                      | 279 +++++++++++++++++++++
 .../auron/flink/table/AuronFlinkTableTestBase.java |  72 ++++++
 .../flink/table/runtime/AuronFlinkCalcITCase.java  |  41 +++
 pom.xml                                            |  10 +
 5 files changed, 416 insertions(+)

diff --git a/auron-build.sh b/auron-build.sh
index 6e0c343b..39e8091e 100755
--- a/auron-build.sh
+++ b/auron-build.sh
@@ -35,6 +35,7 @@ print_help() {
     echo "  --paimon <VERSION>       Specify Paimon version (e.g. 1.2)"
     echo "  --clean <true|false>     Clean before build (default: true)"
     echo "  --skiptests <true|false> Skip unit tests (default: true)"
+    echo "  --flink <VERSION>        Specify Flink version (e.g. 1.18)"
     echo "  -h, --help               Show this help message"
     echo
     echo "Examples:"
@@ -56,6 +57,7 @@ SCALA_VER=""
 CELEBORN_VER=""
 UNIFFLE_VER=""
 PAIMON_VER=""
+FLINK_VER=""
 
 # Parse command-line arguments
 while [[ $# -gt 0 ]]; do
@@ -140,6 +142,15 @@ while [[ $# -gt 0 ]]; do
                 exit 1
             fi
             ;;
+        --flink)
+            if [[ -n "$2" && "$2" != -* ]]; then
+                FLINK_VER="$2"
+                shift 2
+            else
+                echo "ERROR: --flink requires version argument" >&2
+                exit 1
+            fi
+            ;;
         -h|--help)
             print_help
             ;;
@@ -220,6 +231,9 @@ fi
 if [[ -n "$PAIMON_VER" ]]; then
     BUILD_ARGS+=("-Ppaimon,paimon-$PAIMON_VER")
 fi
+if [[ -n "$FLINK_VER" ]]; then
+    BUILD_ARGS+=("-Pflink-$FLINK_VER")
+fi
 
 MVN_ARGS=("${CLEAN_ARGS[@]}" "${BUILD_ARGS[@]}")
 
diff --git a/auron-flink-extension/pom.xml b/auron-flink-extension/pom.xml
new file mode 100644
index 00000000..c4626fd8
--- /dev/null
+++ b/auron-flink-extension/pom.xml
@@ -0,0 +1,279 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+  ~ Licensed to the Apache Software Foundation (ASF) under one or more
+  ~ contributor license agreements.  See the NOTICE file distributed with
+  ~ this work for additional information regarding copyright ownership.
+  ~ The ASF licenses this file to You under the Apache License, Version 2.0
+  ~ (the "License"); you may not use this file except in compliance with
+  ~ the License.  You may obtain a copy of the License at
+  ~
+  ~    http://www.apache.org/licenses/LICENSE-2.0
+  ~
+  ~ Unless required by applicable law or agreed to in writing, software
+  ~ distributed under the License is distributed on an "AS IS" BASIS,
+  ~ WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+  ~ See the License for the specific language governing permissions and
+  ~ limitations under the License.
+  -->
+<project xmlns="http://maven.apache.org/POM/4.0.0"; 
xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"; 
xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 
http://maven.apache.org/xsd/maven-4.0.0.xsd";>
+  <modelVersion>4.0.0</modelVersion>
+
+  <parent>
+    <groupId>org.apache.auron</groupId>
+    <artifactId>auron-parent_${scalaVersion}</artifactId>
+    <version>${project.version}</version>
+    <relativePath>../pom.xml</relativePath>
+  </parent>
+
+  <artifactId>auron-flink-extension</artifactId>
+  <packaging>jar</packaging>
+
+  <properties>
+    <flink.markBundledAsOptional>false</flink.markBundledAsOptional>
+    <janino.version>3.1.10</janino.version>
+    <guava.version>31.1-jre</guava.version>
+    <scala.binary.version>2.12</scala.binary.version>
+    <surefire.module.config>
+      <!--
+                  HashAggCodeGeneratorTest / AggTestBase
+                  -->
+      --add-opens=java.base/java.lang=ALL-UNNAMED
+      <!--
+                  chill ArraysAsListSerializer
+                  -->
+      --add-opens=java.base/java.util=ALL-UNNAMED
+      <!--
+                  kryo AtomicBoolean
+                  -->
+      --add-opens=java.base/java.util.concurrent.atomic=ALL-UNNAMED
+      <!--
+                  kryo LocalDate/ZoneOffset
+                  -->
+      --add-opens=java.base/java.time=ALL-UNNAMED
+      <!--
+                  kryo MathContext
+                  -->
+      --add-opens=java.base/java.math=ALL-UNNAMED
+      <!--
+                  kryo ByteBuffer
+                  -->
+      --add-opens=java.base/java.nio=ALL-UNNAMED
+    </surefire.module.config>
+  </properties>
+
+  <dependencies>
+    <!-- Auron dependencies -->
+    <dependency>
+      <groupId>org.apache.auron</groupId>
+      <artifactId>auron-core</artifactId>
+      <version>${project.version}</version>
+    </dependency>
+    <dependency>
+      <groupId>org.apache.auron</groupId>
+      <artifactId>proto</artifactId>
+      <version>${project.version}</version>
+    </dependency>
+
+    <dependency>
+      <groupId>org.scala-lang</groupId>
+      <artifactId>scala-library</artifactId>
+      <version>${scalaLongVersion}</version>
+      <scope>provided</scope>
+    </dependency>
+    <dependency>
+      <groupId>org.immutables</groupId>
+      <artifactId>value</artifactId>
+      <version>2.8.8</version>
+      <scope>provided</scope>
+    </dependency>
+    <dependency>
+      <groupId>org.immutables</groupId>
+      <artifactId>value-annotations</artifactId>
+      <version>2.8.8</version>
+      <scope>provided</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.codehaus.janino</groupId>
+      <artifactId>commons-compiler</artifactId>
+      <version>${janino.version}</version>
+      <scope>provided</scope>
+    </dependency>
+    <dependency>
+      <!-- Used for code generation -->
+      <groupId>org.codehaus.janino</groupId>
+      <artifactId>janino</artifactId>
+      <version>${janino.version}</version>
+      <scope>provided</scope>
+    </dependency>
+
+    <!-- Table API Java dependencies (not included in the uber) -->
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-api-java-bridge</artifactId>
+      <version>${flink.version}</version>
+      <scope>provided</scope>
+    </dependency>
+
+    <!-- Flink Scala (included only in the loader-bundle) -->
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-scala_${scala.binary.version}</artifactId>
+      <version>${flink.version}</version>
+      <scope>provided</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.checkerframework</groupId>
+      <artifactId>checker-qual</artifactId>
+      <version>3.12.0</version>
+      <scope>provided</scope>
+      <optional>${flink.markBundledAsOptional}</optional>
+    </dependency>
+
+    <!-- Table planner -->
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-planner_${scala.binary.version}</artifactId>
+      <version>${flink.version}</version>
+    </dependency>
+
+    <!-- Table Runtime (not included in the uber) -->
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-runtime</artifactId>
+      <version>${flink.version}</version>
+      <scope>provided</scope>
+    </dependency>
+
+    <!-- Table Calcite Bridge (included in the uber) -->
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-calcite-bridge</artifactId>
+      <version>${flink.version}</version>
+      <scope>provided</scope>
+      <optional>${flink.markBundledAsOptional}</optional>
+      <exclusions>
+        <exclusion>
+          <groupId>org.apache.calcite</groupId>
+          <artifactId>calcite-core</artifactId>
+        </exclusion>
+      </exclusions>
+    </dependency>
+
+    <!-- Test dependencies -->
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-planner_${scala.binary.version}</artifactId>
+      <version>${flink.version}</version>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-test-utils</artifactId>
+      <version>${flink.version}</version>
+      <scope>test</scope>
+    </dependency>
+
+    <!-- Table API Scala dependencies -->
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-api-scala_${scala.binary.version}</artifactId>
+      <version>${flink.version}</version>
+      <scope>test</scope>
+    </dependency>
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      
<artifactId>flink-table-api-scala-bridge_${scala.binary.version}</artifactId>
+      <version>${flink.version}</version>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-core</artifactId>
+      <version>${flink.version}</version>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-common</artifactId>
+      <version>${flink.version}</version>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-api-java</artifactId>
+      <version>${flink.version}</version>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-table-runtime</artifactId>
+      <version>${flink.version}</version>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-streaming-java</artifactId>
+      <version>${flink.version}</version>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-statebackend-rocksdb</artifactId>
+      <version>${flink.version}</version>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <!-- For using the filesystem connector in tests -->
+      <groupId>org.apache.flink</groupId>
+      <artifactId>flink-connector-files</artifactId>
+      <version>${flink.version}</version>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>junit</groupId>
+      <artifactId>junit</artifactId>
+      <version>4.13.2</version>
+      <scope>test</scope>
+    </dependency>
+
+    <!-- arrow dependency -->
+    <dependency>
+      <groupId>org.apache.arrow</groupId>
+      <artifactId>arrow-c-data</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.arrow</groupId>
+      <artifactId>arrow-compression</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.arrow</groupId>
+      <artifactId>arrow-memory-unsafe</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.arrow</groupId>
+      <artifactId>arrow-vector</artifactId>
+    </dependency>
+  </dependencies>
+
+</project>
diff --git 
a/auron-flink-extension/src/test/java/org/apache/auron/flink/table/AuronFlinkTableTestBase.java
 
b/auron-flink-extension/src/test/java/org/apache/auron/flink/table/AuronFlinkTableTestBase.java
new file mode 100644
index 00000000..579682d2
--- /dev/null
+++ 
b/auron-flink-extension/src/test/java/org/apache/auron/flink/table/AuronFlinkTableTestBase.java
@@ -0,0 +1,72 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.auron.flink.table;
+
+import java.math.BigDecimal;
+import java.util.Arrays;
+import org.apache.flink.api.common.RuntimeExecutionMode;
+import org.apache.flink.configuration.Configuration;
+import org.apache.flink.configuration.ExecutionOptions;
+import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
+import org.apache.flink.table.api.EnvironmentSettings;
+import org.apache.flink.table.api.bridge.java.StreamTableEnvironment;
+import org.apache.flink.table.planner.factories.TestValuesTableFactory;
+import org.apache.flink.types.Row;
+import org.junit.Before;
+
+/**
+ * Base class for Flink Table Tests.
+ */
+public class AuronFlinkTableTestBase {
+
+    protected StreamExecutionEnvironment environment;
+    protected StreamTableEnvironment tableEnvironment;
+
+    @Before
+    public void before() {
+        environment = StreamExecutionEnvironment.getExecutionEnvironment();
+        Configuration configuration = new Configuration();
+        configuration.set(ExecutionOptions.RUNTIME_MODE, 
RuntimeExecutionMode.STREAMING);
+        tableEnvironment =
+                StreamTableEnvironment.create(environment, 
EnvironmentSettings.fromConfiguration(configuration));
+        String timestampDataId = 
TestValuesTableFactory.registerData(Arrays.asList(
+                row("2020-10-10 00:00:01", 1, 1d, 1f, new BigDecimal("1.11"), 
"Hi", "a"),
+                row("2020-10-10 00:00:02", 2, 2d, 2f, new BigDecimal("2.22"), 
"Comment#1", "a"),
+                row("2020-10-10 00:00:03", 2, 2d, 2f, new BigDecimal("2.22"), 
"Comment#1", "a")));
+        tableEnvironment.executeSql(" CREATE TABLE T1 ( "
+                + "\n `ts` String, "
+                + "\n `int` INT, "
+                + "\n `double` DOUBLE, "
+                + "\n `float` FLOAT, "
+                + "\n `bigdec` DECIMAL(10, 2), "
+                + "\n `string` STRING, "
+                + "\n `name` STRING "
+                + "\n ) WITH ( "
+                + "\n 'connector' = 'values',"
+                + "\n 'data-id' = '" + timestampDataId + "',"
+                + "\n 'failing-source' = 'false' "
+                + "\n )");
+    }
+
+    protected Row row(Object... values) {
+        Row row = new Row(values.length);
+        for (int i = 0; i < values.length; i++) {
+            row.setField(i, values[i]);
+        }
+        return row;
+    }
+}
diff --git 
a/auron-flink-extension/src/test/java/org/apache/auron/flink/table/runtime/AuronFlinkCalcITCase.java
 
b/auron-flink-extension/src/test/java/org/apache/auron/flink/table/runtime/AuronFlinkCalcITCase.java
new file mode 100644
index 00000000..49bfc2be
--- /dev/null
+++ 
b/auron-flink-extension/src/test/java/org/apache/auron/flink/table/runtime/AuronFlinkCalcITCase.java
@@ -0,0 +1,41 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.auron.flink.table.runtime;
+
+import static org.assertj.core.api.Assertions.assertThat;
+
+import java.util.Arrays;
+import java.util.Comparator;
+import java.util.List;
+import org.apache.auron.flink.table.AuronFlinkTableTestBase;
+import org.apache.flink.types.Row;
+import org.apache.flink.util.CollectionUtil;
+import org.junit.Test;
+
+/**
+ * IT case for Flink Calc Operator on Auron.
+ */
+public class AuronFlinkCalcITCase extends AuronFlinkTableTestBase {
+
+    @Test
+    public void testPlus() {
+        List<Row> rows = CollectionUtil.iteratorToList(
+                tableEnvironment.executeSql("select `int` + 1 from 
T1").collect());
+        rows.sort(Comparator.comparingInt(o -> (int) o.getField(0)));
+        assertThat(rows).isEqualTo(Arrays.asList(Row.of(2), Row.of(3), 
Row.of(3)));
+    }
+}
diff --git a/pom.xml b/pom.xml
index 4e35075b..e8e8333d 100644
--- a/pom.xml
+++ b/pom.xml
@@ -715,5 +715,15 @@
         <paimonVersion>1.2.0</paimonVersion>
       </properties>
     </profile>
+
+    <profile>
+      <id>flink-1.18</id>
+      <modules>
+        <module>auron-flink-extension</module>
+      </modules>
+      <properties>
+        <flink.version>1.18.1</flink.version>
+      </properties>
+    </profile>
   </profiles>
 </project>

Reply via email to