kafka-commits mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From ewe...@apache.org
Subject kafka git commit: KAFKA-5535: Handle null values in ExtractField
Date Fri, 04 Aug 2017 17:25:26 GMT
Repository: kafka
Updated Branches:
  refs/heads/trunk 5704d8f05 -> 22611aca9


KAFKA-5535: Handle null values in ExtractField

Author: Ewen Cheslack-Postava <me@ewencp.org>

Reviewers: Randall Hauch <rhauch@gmail.com>, Konstantine Karantasis <konstantine@confluent.io>,
Jason Gustafson <jason@confluent.io>

Closes #3559 from ewencp/kafka-5535-extract-field-null


Project: http://git-wip-us.apache.org/repos/asf/kafka/repo
Commit: http://git-wip-us.apache.org/repos/asf/kafka/commit/22611aca
Tree: http://git-wip-us.apache.org/repos/asf/kafka/tree/22611aca
Diff: http://git-wip-us.apache.org/repos/asf/kafka/diff/22611aca

Branch: refs/heads/trunk
Commit: 22611aca9b33cc3db9362add785e44c7a3698ba3
Parents: 5704d8f
Author: Ewen Cheslack-Postava <me@ewencp.org>
Authored: Fri Aug 4 10:25:21 2017 -0700
Committer: Ewen Cheslack-Postava <me@ewencp.org>
Committed: Fri Aug 4 10:25:21 2017 -0700

----------------------------------------------------------------------
 .../kafka/connect/transforms/ExtractField.java  | 15 +++++------
 .../connect/transforms/util/Requirements.java   | 14 +++++++++++
 .../connect/transforms/ExtractFieldTest.java    | 26 ++++++++++++++++++++
 3 files changed, 48 insertions(+), 7 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/kafka/blob/22611aca/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/ExtractField.java
----------------------------------------------------------------------
diff --git a/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/ExtractField.java
b/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/ExtractField.java
index bb4b539..eb8c357 100644
--- a/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/ExtractField.java
+++ b/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/ExtractField.java
@@ -24,13 +24,14 @@ import org.apache.kafka.connect.transforms.util.SimpleConfig;
 
 import java.util.Map;
 
-import static org.apache.kafka.connect.transforms.util.Requirements.requireMap;
-import static org.apache.kafka.connect.transforms.util.Requirements.requireStruct;
+import static org.apache.kafka.connect.transforms.util.Requirements.requireMapOrNull;
+import static org.apache.kafka.connect.transforms.util.Requirements.requireStructOrNull;
 
 public abstract class ExtractField<R extends ConnectRecord<R>> implements Transformation<R>
{
 
     public static final String OVERVIEW_DOC =
-            "Extract the specified field from a Struct when schema present, or a Map in the
case of schemaless data."
+            "Extract the specified field from a Struct when schema present, or a Map in the
case of schemaless data. "
+                    + "Any null values are passed through unmodified."
                     + "<p/>Use the concrete transformation type designed for the record
key (<code>" + Key.class.getName() + "</code>) "
                     + "or value (<code>" + Value.class.getName() + "</code>).";
 
@@ -53,11 +54,11 @@ public abstract class ExtractField<R extends ConnectRecord<R>>
implements Transf
     public R apply(R record) {
         final Schema schema = operatingSchema(record);
         if (schema == null) {
-            final Map<String, Object> value = requireMap(operatingValue(record), PURPOSE);
-            return newRecord(record, null, value.get(fieldName));
+            final Map<String, Object> value = requireMapOrNull(operatingValue(record),
PURPOSE);
+            return newRecord(record, null, value == null ? null : value.get(fieldName));
         } else {
-            final Struct value = requireStruct(operatingValue(record), PURPOSE);
-            return newRecord(record, schema.field(fieldName).schema(), value.get(fieldName));
+            final Struct value = requireStructOrNull(operatingValue(record), PURPOSE);
+            return newRecord(record, schema.field(fieldName).schema(), value == null ? null
: value.get(fieldName));
         }
     }
 

http://git-wip-us.apache.org/repos/asf/kafka/blob/22611aca/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/util/Requirements.java
----------------------------------------------------------------------
diff --git a/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/util/Requirements.java
b/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/util/Requirements.java
index 6f1be19..6d1cd78 100644
--- a/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/util/Requirements.java
+++ b/connect/transforms/src/main/java/org/apache/kafka/connect/transforms/util/Requirements.java
@@ -40,6 +40,13 @@ public class Requirements {
         return (Map<String, Object>) value;
     }
 
+    public static Map<String, Object> requireMapOrNull(Object value, String purpose)
{
+        if (value == null) {
+            return null;
+        }
+        return requireMap(value, purpose);
+    }
+
     public static Struct requireStruct(Object value, String purpose) {
         if (!(value instanceof Struct)) {
             throw new DataException("Only Struct objects supported for [" + purpose + "],
found: " + nullSafeClassName(value));
@@ -47,6 +54,13 @@ public class Requirements {
         return (Struct) value;
     }
 
+    public static Struct requireStructOrNull(Object value, String purpose) {
+        if (value == null) {
+            return null;
+        }
+        return requireStruct(value, purpose);
+    }
+
     public static SinkRecord requireSinkRecord(ConnectRecord<?> record, String purpose)
{
         if (!(record instanceof SinkRecord)) {
             throw new DataException("Only SinkRecord supported for [" + purpose + "], found:
" + nullSafeClassName(record));

http://git-wip-us.apache.org/repos/asf/kafka/blob/22611aca/connect/transforms/src/test/java/org/apache/kafka/connect/transforms/ExtractFieldTest.java
----------------------------------------------------------------------
diff --git a/connect/transforms/src/test/java/org/apache/kafka/connect/transforms/ExtractFieldTest.java
b/connect/transforms/src/test/java/org/apache/kafka/connect/transforms/ExtractFieldTest.java
index 0b7ce96..acb0beb 100644
--- a/connect/transforms/src/test/java/org/apache/kafka/connect/transforms/ExtractFieldTest.java
+++ b/connect/transforms/src/test/java/org/apache/kafka/connect/transforms/ExtractFieldTest.java
@@ -24,6 +24,7 @@ import org.junit.After;
 import org.junit.Test;
 
 import java.util.Collections;
+import java.util.Map;
 
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertNull;
@@ -48,6 +49,18 @@ public class ExtractFieldTest {
     }
 
     @Test
+    public void testNullSchemaless() {
+        xform.configure(Collections.singletonMap("field", "magic"));
+
+        final Map<String, Object> key = null;
+        final SinkRecord record = new SinkRecord("test", 0, null, key, null, null, 0);
+        final SinkRecord transformedRecord = xform.apply(record);
+
+        assertNull(transformedRecord.keySchema());
+        assertNull(transformedRecord.key());
+    }
+
+    @Test
     public void withSchema() {
         xform.configure(Collections.singletonMap("field", "magic"));
 
@@ -60,4 +73,17 @@ public class ExtractFieldTest {
         assertEquals(42, transformedRecord.key());
     }
 
+    @Test
+    public void testNullWithSchema() {
+        xform.configure(Collections.singletonMap("field", "magic"));
+
+        final Schema keySchema = SchemaBuilder.struct().field("magic", Schema.INT32_SCHEMA).optional().build();
+        final Struct key = null;
+        final SinkRecord record = new SinkRecord("test", 0, keySchema, key, null, null, 0);
+        final SinkRecord transformedRecord = xform.apply(record);
+
+        assertEquals(Schema.INT32_SCHEMA, transformedRecord.keySchema());
+        assertNull(transformedRecord.key());
+    }
+
 }


Mime
View raw message