[ 
https://issues.apache.org/jira/browse/FLINK-10107?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16666873#comment-16666873
 ] 

ASF GitHub Bot commented on FLINK-10107:
----------------------------------------

twalthr closed pull request #6534: [FLINK-10107] [sql-client] Relocate Flink 
Kafka connectors for SQL JARs
URL: https://github.com/apache/flink/pull/6534
 
 
   

This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:

As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):

diff --git a/flink-connectors/flink-connector-kafka-0.10/pom.xml 
b/flink-connectors/flink-connector-kafka-0.10/pom.xml
index 135dc59b655..c7a840caafb 100644
--- a/flink-connectors/flink-connector-kafka-0.10/pom.xml
+++ b/flink-connectors/flink-connector-kafka-0.10/pom.xml
@@ -225,6 +225,7 @@ under the License.
                                                                <configuration>
                                                                        
<shadedArtifactAttached>true</shadedArtifactAttached>
                                                                        
<shadedClassifierName>sql-jar</shadedClassifierName>
+                                                                       <!-- 
Flink's Kafka 0.10 connector depends on Flink's Kafka 0.9 connector. -->
                                                                        
<artifactSet>
                                                                                
<includes combine.children="append">
                                                                                
        <include>org.apache.kafka:*</include>
@@ -232,6 +233,11 @@ under the License.
                                                                                
        
<include>org.apache.flink:flink-connector-kafka-0.9_${scala.binary.version}</include>
                                                                                
</includes>
                                                                        
</artifactSet>
+                                                                       <!-- 
Update service files for table factories. -->
+                                                                       
<transformers>
+                                                                               
<transformer 
implementation="org.apache.maven.plugins.shade.resource.ServicesResourceTransformer"
 />
+                                                                       
</transformers>
+                                                                       <!-- 
Remove version-specific resources not needed by Flink's Kafka 0.10 connector. 
-->
                                                                        
<filters>
                                                                                
<filter>
                                                                                
        <artifact>*:*</artifact>
@@ -239,12 +245,23 @@ under the License.
                                                                                
                <exclude>kafka/kafka-version.properties</exclude>
                                                                                
        </excludes>
                                                                                
</filter>
+                                                                               
<filter>
+                                                                               
        
<artifact>org.apache.flink:flink-connector-kafka-0.9_${scala.binary.version}</artifact>
+                                                                               
        <excludes>
+                                                                               
                
<exclude>META-INF/services/org.apache.flink.table.factories.TableFactory</exclude>
+                                                                               
        </excludes>
+                                                                               
</filter>
                                                                        
</filters>
+                                                                       <!-- 
Relocate all version-specific classes. -->
                                                                        
<relocations>
                                                                                
<relocation>
                                                                                
        <pattern>org.apache.kafka</pattern>
                                                                                
        
<shadedPattern>org.apache.flink.kafka010.shaded.org.apache.kafka</shadedPattern>
                                                                                
</relocation>
+                                                                               
<relocation>
+                                                                               
        <pattern>org.apache.flink.streaming.connectors.kafka</pattern>
+                                                                               
        
<shadedPattern>org.apache.flink.kafka010.shaded.org.apache.flink.streaming.connectors.kafka</shadedPattern>
+                                                                               
</relocation>
                                                                        
</relocations>
                                                                </configuration>
                                                        </execution>
diff --git a/flink-connectors/flink-connector-kafka-0.11/pom.xml 
b/flink-connectors/flink-connector-kafka-0.11/pom.xml
index bf04aebb390..c6bf71e8d5a 100644
--- a/flink-connectors/flink-connector-kafka-0.11/pom.xml
+++ b/flink-connectors/flink-connector-kafka-0.11/pom.xml
@@ -234,6 +234,7 @@ under the License.
                                                                <configuration>
                                                                        
<shadedArtifactAttached>true</shadedArtifactAttached>
                                                                        
<shadedClassifierName>sql-jar</shadedClassifierName>
+                                                                       <!-- 
Flink's Kafka 0.11 connector depends on Flink's Kafka 0.9 and 0.10 connector. 
-->
                                                                        
<artifactSet>
                                                                                
<includes combine.children="append">
                                                                                
        <include>org.apache.kafka:*</include>
@@ -242,6 +243,11 @@ under the License.
                                                                                
        
<include>org.apache.flink:flink-connector-kafka-0.10_${scala.binary.version}</include>
                                                                                
</includes>
                                                                        
</artifactSet>
+                                                                       <!-- 
Update service files for table factories. -->
+                                                                       
<transformers>
+                                                                               
<transformer 
implementation="org.apache.maven.plugins.shade.resource.ServicesResourceTransformer"
 />
+                                                                       
</transformers>
+                                                                       <!-- 
Remove version-specific resources not needed by Flink's Kafka 0.11 connector. 
-->
                                                                        
<filters>
                                                                                
<filter>
                                                                                
        <artifact>*:*</artifact>
@@ -249,12 +255,29 @@ under the License.
                                                                                
                <exclude>kafka/kafka-version.properties</exclude>
                                                                                
        </excludes>
                                                                                
</filter>
+                                                                               
<filter>
+                                                                               
        
<artifact>org.apache.flink:flink-connector-kafka-0.9_${scala.binary.version}</artifact>
+                                                                               
        <excludes>
+                                                                               
                
<exclude>META-INF/services/org.apache.flink.table.factories.TableFactory</exclude>
+                                                                               
        </excludes>
+                                                                               
</filter>
+                                                                               
<filter>
+                                                                               
        
<artifact>org.apache.flink:flink-connector-kafka-0.10_${scala.binary.version}</artifact>
+                                                                               
        <excludes>
+                                                                               
                
<exclude>META-INF/services/org.apache.flink.table.factories.TableFactory</exclude>
+                                                                               
        </excludes>
+                                                                               
</filter>
                                                                        
</filters>
+                                                                       <!-- 
Relocate all version-specific classes. -->
                                                                        
<relocations>
                                                                                
<relocation>
                                                                                
        <pattern>org.apache.kafka</pattern>
                                                                                
        
<shadedPattern>org.apache.flink.kafka011.shaded.org.apache.kafka</shadedPattern>
                                                                                
</relocation>
+                                                                               
<relocation>
+                                                                               
        <pattern>org.apache.flink.streaming.connectors.kafka</pattern>
+                                                                               
        
<shadedPattern>org.apache.flink.kafka011.shaded.org.apache.flink.streaming.connectors.kafka</shadedPattern>
+                                                                               
</relocation>
                                                                        
</relocations>
                                                                </configuration>
                                                        </execution>
diff --git a/flink-connectors/flink-connector-kafka-0.9/pom.xml 
b/flink-connectors/flink-connector-kafka-0.9/pom.xml
index 3e0aa46e095..99f40c574af 100644
--- a/flink-connectors/flink-connector-kafka-0.9/pom.xml
+++ b/flink-connectors/flink-connector-kafka-0.9/pom.xml
@@ -219,6 +219,10 @@ under the License.
                                                                                
        
<include>org.apache.flink:flink-connector-kafka-base_${scala.binary.version}</include>
                                                                                
</includes>
                                                                        
</artifactSet>
+                                                                       <!-- 
Update service files for table factories. -->
+                                                                       
<transformers>
+                                                                               
<transformer 
implementation="org.apache.maven.plugins.shade.resource.ServicesResourceTransformer"
 />
+                                                                       
</transformers>
                                                                        
<filters>
                                                                                
<filter>
                                                                                
        <artifact>*:*</artifact>
@@ -227,11 +231,16 @@ under the License.
                                                                                
        </excludes>
                                                                                
</filter>
                                                                        
</filters>
+                                                                       <!-- 
Relocate all version-specific classes. -->
                                                                        
<relocations>
                                                                                
<relocation>
                                                                                
        <pattern>org.apache.kafka</pattern>
                                                                                
        
<shadedPattern>org.apache.flink.kafka09.shaded.org.apache.kafka</shadedPattern>
                                                                                
</relocation>
+                                                                               
<relocation>
+                                                                               
        <pattern>org.apache.flink.streaming.connectors.kafka</pattern>
+                                                                               
        
<shadedPattern>org.apache.flink.kafka09.shaded.org.apache.flink.streaming.connectors.kafka</shadedPattern>
+                                                                               
</relocation>
                                                                        
</relocations>
                                                                </configuration>
                                                        </execution>
diff --git a/flink-end-to-end-tests/flink-sql-client-test/pom.xml 
b/flink-end-to-end-tests/flink-sql-client-test/pom.xml
index 6e69568939f..ec5a0e10360 100644
--- a/flink-end-to-end-tests/flink-sql-client-test/pom.xml
+++ b/flink-end-to-end-tests/flink-sql-client-test/pom.xml
@@ -153,14 +153,13 @@ under the License.
                                                                        
<classifier>sql-jar</classifier>
                                                                        
<type>jar</type>
                                                                </artifactItem>
-                                                               <!-- This SQL 
JAR is not used for now to avoid dependency conflicts; see FLINK-10107.
                                                                <artifactItem>
                                                                        
<groupId>org.apache.flink</groupId>
                                                                        
<artifactId>flink-connector-kafka-0.9_${scala.binary.version}</artifactId>
                                                                        
<version>${project.version}</version>
                                                                        
<classifier>sql-jar</classifier>
                                                                        
<type>jar</type>
-                                                               
</artifactItem>-->
+                                                               </artifactItem>
                                                                <artifactItem>
                                                                        
<groupId>org.apache.flink</groupId>
                                                                        
<artifactId>flink-connector-kafka-0.10_${scala.binary.version}</artifactId>
@@ -168,14 +167,13 @@ under the License.
                                                                        
<classifier>sql-jar</classifier>
                                                                        
<type>jar</type>
                                                                </artifactItem>
-                                                               <!-- This SQL 
JAR is not used for now to avoid dependency conflicts; see FLINK-10107.
                                                                <artifactItem>
                                                                        
<groupId>org.apache.flink</groupId>
                                                                        
<artifactId>flink-connector-kafka-0.11_${scala.binary.version}</artifactId>
                                                                        
<version>${project.version}</version>
                                                                        
<classifier>sql-jar</classifier>
                                                                        
<type>jar</type>
-                                                               
</artifactItem>-->
+                                                               </artifactItem>
                                                        </artifactItems>
                                                </configuration>
                                        </execution>
diff --git a/flink-end-to-end-tests/test-scripts/test_sql_client.sh 
b/flink-end-to-end-tests/test-scripts/test_sql_client.sh
index 934f7d43ed4..ff364a0b6c9 100755
--- a/flink-end-to-end-tests/test-scripts/test_sql_client.sh
+++ b/flink-end-to-end-tests/test-scripts/test_sql_client.sh
@@ -57,6 +57,10 @@ for SQL_JAR in $SQL_JARS_DIR/*.jar; do
   rm -r $EXTRACTED_JAR/*
 done
 
+# randomize JAR list for detecting classloading issues due to JAR order
+SQL_JARS_PARAMETERS=$(find $SQL_JARS_DIR -type f | sort -R | awk '{ print 
"--jar \""$0"\""}' | tr '\n' ' ')
+echo "Using randomized SQL JAR list: $SQL_JARS_PARAMETERS"
+
 
################################################################################
 # Run a SQL statement
 
################################################################################
@@ -254,7 +258,7 @@ echo "Executing SQL: Kafka JSON -> Kafka Avro"
 echo "$SQL_STATEMENT_1"
 
 $FLINK_DIR/bin/sql-client.sh embedded \
-  --library $SQL_JARS_DIR \
+  $SQL_JARS_PARAMETERS \
   --jar $SQL_TOOLBOX_JAR \
   --environment $SQL_CONF \
   --update "$SQL_STATEMENT_1"
@@ -269,7 +273,7 @@ echo "Executing SQL: Kafka Avro -> Filesystem CSV"
 echo "$SQL_STATEMENT_2"
 
 $FLINK_DIR/bin/sql-client.sh embedded \
-  --library $SQL_JARS_DIR \
+  $SQL_JARS_PARAMETERS \
   --jar $SQL_TOOLBOX_JAR \
   --environment $SQL_CONF \
   --update "$SQL_STATEMENT_2"


 

----------------------------------------------------------------
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


> SQL Client end-to-end test fails for releases
> ---------------------------------------------
>
>                 Key: FLINK-10107
>                 URL: https://issues.apache.org/jira/browse/FLINK-10107
>             Project: Flink
>          Issue Type: Bug
>          Components: Table API &amp; SQL
>            Reporter: Timo Walther
>            Assignee: Timo Walther
>            Priority: Major
>              Labels: pull-request-available
>
> It seems that SQL JARs for Kafka 0.10 and Kafka 0.9 have conflicts that only 
> occur for releases and not SNAPSHOT builds. This might be due to their file 
> name. Depending on the file name either 0.9 is loaded before 0.10 and vice 
> versa.
> One of the following errors occured:
> {code}
> 2018-08-08 18:28:51,636 ERROR 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.ClientUtils  - 
> Failed to close coordinator
> java.lang.NoClassDefFoundError: 
> org/apache/flink/kafka09/shaded/org/apache/kafka/common/requests/OffsetCommitResponse
>     at 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.sendOffsetCommitRequest(ConsumerCoordinator.java:473)
>     at 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.commitOffsetsSync(ConsumerCoordinator.java:357)
>     at 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.maybeAutoCommitOffsetsSync(ConsumerCoordinator.java:439)
>     at 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.close(ConsumerCoordinator.java:319)
>     at 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.ClientUtils.closeQuietly(ClientUtils.java:63)
>     at 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.consumer.KafkaConsumer.close(KafkaConsumer.java:1277)
>     at 
> org.apache.flink.kafka09.shaded.org.apache.kafka.clients.consumer.KafkaConsumer.close(KafkaConsumer.java:1258)
>     at 
> org.apache.flink.streaming.connectors.kafka.internal.KafkaConsumerThread.run(KafkaConsumerThread.java:286)
> Caused by: java.lang.ClassNotFoundException: 
> org.apache.flink.kafka09.shaded.org.apache.kafka.common.requests.OffsetCommitResponse
>     at java.net.URLClassLoader.findClass(URLClassLoader.java:381)
>     at java.lang.ClassLoader.loadClass(ClassLoader.java:424)
>     at 
> org.apache.flink.runtime.execution.librarycache.FlinkUserCodeClassLoaders$ChildFirstClassLoader.loadClass(FlinkUserCodeClassLoaders.java:120)
>     at java.lang.ClassLoader.loadClass(ClassLoader.java:357)
>     ... 8 more
> {code}
> {code}
> java.lang.NoSuchFieldError: producer
>     at 
> org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducer010.invoke(FlinkKafkaProducer010.java:369)
>     at 
> org.apache.flink.streaming.api.operators.StreamSink.processElement(StreamSink.java:56)
>     at 
> org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.pushToOperator(OperatorChain.java:579)
>     at 
> org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:554)
>     at 
> org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:534)
>     at 
> org.apache.flink.streaming.api.operators.AbstractStreamOperator$CountingOutput.collect(AbstractStreamOperator.java:689)
>     at 
> org.apache.flink.streaming.api.operators.AbstractStreamOperator$CountingOutput.collect(AbstractStreamOperator.java:667)
>     at 
> org.apache.flink.streaming.api.operators.StreamMap.processElement(StreamMap.java:41)
>     at 
> org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.pushToOperator(OperatorChain.java:579)
>     at 
> org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:554)
>     at 
> org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:534)
>     at 
> org.apache.flink.streaming.api.operators.AbstractStreamOperator$CountingOutput.collect(AbstractStreamOperator.java:689)
>     at 
> org.apache.flink.streaming.api.operators.AbstractStreamOperator$CountingOutput.collect(AbstractStreamOperator.java:667)
>     at 
> org.apache.flink.streaming.api.operators.TimestampedCollector.collect(TimestampedCollector.java:51)
>     at 
> org.apache.flink.table.runtime.CRowWrappingCollector.collect(CRowWrappingCollector.scala:37)
>     at 
> org.apache.flink.table.runtime.CRowWrappingCollector.collect(CRowWrappingCollector.scala:28)
> {code}



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to