nastra commented on code in PR #9452:
URL: https://github.com/apache/iceberg/pull/9452#discussion_r1453732220


##########
flink/v1.16/flink/src/main/java/org/apache/iceberg/flink/util/FlinkPackage.java:
##########
@@ -19,15 +19,31 @@
 package org.apache.iceberg.flink.util;
 
 import org.apache.flink.streaming.api.datastream.DataStream;
+import 
org.apache.iceberg.relocated.com.google.common.annotations.VisibleForTesting;
 
 public class FlinkPackage {
-  /** Choose {@link DataStream} class because it is one of the core Flink API. 
*/
-  private static final String VERSION = 
DataStream.class.getPackage().getImplementationVersion();
+
+  public static final String FLINK_UNKNOWN_VERSION = "Flink-UNKNOWN";
 
   private FlinkPackage() {}
 
   /** Returns Flink version string like x.y.z */
   public static String version() {
-    return VERSION;
+    try {
+      String version = getVersionFromJar();
+      /* If we can't detect the exact implementation version from the jar 
(this can happen if the DataStream class
+       appears multiple times in the same classpath such as with shading), 
then the best we can do is say it's
+       unknown
+      */
+      return version != null ? version : FLINK_UNKNOWN_VERSION;

Review Comment:
   I think we might want to slightly update the implementation to 
   ```
   public class FlinkPackage {
   
     private static final AtomicReference<String> VERSION = new 
AtomicReference<>();
     private static final String UNKNOWN_VERSION = "UNKNOWN-VERSION";
   
     private FlinkPackage() {}
   
     /** Returns Flink version string like x.y.z */
     public static String version() {
       if (null == VERSION.get()) {
         VERSION.set(initFlinkVersion());
       }
   
       return VERSION.get();
     }
   
     private static String initFlinkVersion() {
       try {
         String version = versionFromJar();
         // use unknown version in case exact implementation version can't be 
found from the jar (this
         // can happen if the DataStream class appears multiple times in the 
same classpath such as
         // with shading)
         return version != null ? version : UNKNOWN_VERSION;
       } catch (Exception e) {
         return UNKNOWN_VERSION;
       }
     }
   
     @VisibleForTesting
     static String versionFromJar() {
       // Choose {@link DataStream} class because it is one of the core Flink 
API
       return DataStream.class.getPackage().getImplementationVersion();
     }
   }
   ```
   Notice that I also removed the `get` prefix from `versionFromJar()` as 
Iceberg typically doesn't use `get` methods



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscr...@iceberg.apache.org
For additional commands, e-mail: issues-h...@iceberg.apache.org

Reply via email to