k-raina commented on code in PR #20144:
URL: https://github.com/apache/kafka/pull/20144#discussion_r2212810251


##########
clients/src/main/java/org/apache/kafka/common/telemetry/internals/ClientTelemetryUtils.java:
##########
@@ -181,13 +182,22 @@ public static boolean 
validateRequiredResourceLabels(Map<String, String> metadat
         return validateResourceLabel(metadata, MetricsContext.NAMESPACE);
     }
 
-    public static CompressionType 
preferredCompressionType(List<CompressionType> acceptedCompressionTypes) {
-        if (acceptedCompressionTypes != null && 
!acceptedCompressionTypes.isEmpty()) {
-            // Broker is providing the compression types in order of 
preference. Grab the
-            // first one.
+    public static CompressionType 
preferredCompressionType(List<CompressionType> acceptedCompressionTypes, 
Set<CompressionType> unsupportedCompressionTypes) {
+        if (acceptedCompressionTypes == null || 
acceptedCompressionTypes.isEmpty()) {
+            return CompressionType.NONE;
+        }
+
+        // If no unsupported types, return the first accepted type (broker's 
preference)
+        if (unsupportedCompressionTypes == null || 
unsupportedCompressionTypes.isEmpty()) {
             return acceptedCompressionTypes.get(0);
         }

Review Comment:
   Addressed in commit 
https://github.com/apache/kafka/pull/20144/commits/fe88db0c50668c23509d75d8d439e8a19501cd4d



##########
clients/src/main/java/org/apache/kafka/common/telemetry/internals/ClientTelemetryReporter.java:
##########
@@ -713,14 +716,23 @@ private Optional<Builder<?>> 
createPushRequest(ClientTelemetrySubscription local
                 return Optional.empty();
             }
 
-            CompressionType compressionType = 
ClientTelemetryUtils.preferredCompressionType(localSubscription.acceptedCompressionTypes());
+            CompressionType compressionType = 
ClientTelemetryUtils.preferredCompressionType(localSubscription.acceptedCompressionTypes(),
 unsupportedCompressionTypes);
             ByteBuffer compressedPayload;
             try {
                 compressedPayload = ClientTelemetryUtils.compress(payload, 
compressionType);
-            } catch (Throwable e) {
-                log.debug("Failed to compress telemetry payload for 
compression: {}, sending uncompressed data", compressionType);
+            } catch (IOException e) {
+                log.debug("Failed to compress telemetry payload for 
compression: {}, sending uncompressed data", compressionType, e);
                 compressedPayload = ByteBuffer.wrap(payload.toByteArray());
                 compressionType = CompressionType.NONE;
+            } catch (Throwable e) { 
+                if (e instanceof NoClassDefFoundError || e.getCause() 
instanceof NoClassDefFoundError) {
+                    log.debug("Compression library {} not found, sending 
uncompressed data", compressionType, e);
+                    unsupportedCompressionTypes.add(compressionType);

Review Comment:
   Addressed in commit 
https://github.com/apache/kafka/pull/20144/commits/fe88db0c50668c23509d75d8d439e8a19501cd4d



-- 
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: jira-unsubscr...@kafka.apache.org

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

Reply via email to