Selaa lähdekoodia

HADOOP-19315. Upgrade Apache Avro to 1.11.4 (#7128) (#7615)

* All field access is now via setter/getter methods
* To use Avro to marshal Serializable objects,
  the packages they are in must be declared in the system property
  "org.apache.avro.SERIALIZABLE_PACKAGES"

This is required to address
- CVE-2024-47561
- CVE-2023-39410

This change is not backwards compatible.

Contributed by Dominik Diedrich
Steve Loughran 1 viikko sitten
vanhempi
commit
c239f617a7

+ 1 - 2
LICENSE-binary

@@ -296,8 +296,7 @@ io.swagger:swagger-annotations:1.5.4
 javax.inject:javax.inject:1
 net.java.dev.jna:jna:5.2.0
 net.minidev:accessors-smart:1.2
-org.apache.avro:avro:1.9.2
-org.apache.avro:avro:1.11.3
+org.apache.avro:avro:1.11.4
 org.apache.commons:commons-compress:1.26.1
 org.apache.commons:commons-configuration2:2.10.1
 org.apache.commons:commons-csv:1.9.0

+ 35 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/constants/ConfigConstants.java

@@ -0,0 +1,35 @@
+/**
+ * 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.hadoop.constants;
+
+/**
+ * Evolving config constants class used in various hadoop tests.
+ */
+public final class ConfigConstants {
+
+  private ConfigConstants() {}
+
+ /**
+  * System property name for the avro dependency.
+  * This property is used to configure trusted packages,
+  * which the avro dependency can use for serialization.
+  */
+  public static final String CONFIG_AVRO_SERIALIZABLE_PACKAGES =
+      "org.apache.avro.SERIALIZABLE_PACKAGES";
+}

+ 22 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/constants/package-info.java

@@ -0,0 +1,22 @@
+/**
+ * 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.
+ */
+
+/**
+ * Evolving config constants class used in various hadoop tests.
+ */
+package org.apache.hadoop.constants;

+ 4 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/fs/TestPath.java

@@ -17,6 +17,7 @@
  */
 
 package org.apache.hadoop.fs;
+
 import org.junit.Assert;
 import org.junit.Test;
 
@@ -30,6 +31,7 @@ import java.net.URISyntaxException;
 import java.util.Arrays;
 
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.constants.ConfigConstants;
 import org.apache.hadoop.io.AvroTestUtil;
 import org.apache.hadoop.test.GenericTestUtils;
 import org.apache.hadoop.util.Shell;
@@ -404,6 +406,8 @@ public class TestPath {
   
   @Test (timeout = 30000)
   public void testAvroReflect() throws Exception {
+    // Avro expects explicitely stated, trusted packages used for (de-)serialization
+    System.setProperty(ConfigConstants.CONFIG_AVRO_SERIALIZABLE_PACKAGES, "org.apache.hadoop.fs");
     AvroTestUtil.testReflect
       (new Path("foo"),
        "{\"type\":\"string\",\"java-class\":\"org.apache.hadoop.fs.Path\"}");

+ 4 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/TestText.java

@@ -24,6 +24,8 @@ import java.nio.ByteBuffer;
 import java.nio.charset.CharacterCodingException;
 import java.nio.charset.StandardCharsets;
 import java.util.Random;
+
+import org.apache.hadoop.constants.ConfigConstants;
 import org.apache.hadoop.thirdparty.com.google.common.primitives.Bytes;
 import org.junit.Test;
 
@@ -344,6 +346,8 @@ public class TestText {
 
   @Test
   public void testAvroReflect() throws Exception {
+    // Avro expects explicitely stated, trusted packages used for (de-)serialization
+    System.setProperty(ConfigConstants.CONFIG_AVRO_SERIALIZABLE_PACKAGES, "org.apache.hadoop.io");
     AvroTestUtil.testReflect
             (new Text("foo"),
                     "{\"type\":\"string\",\"java-class\":\"org.apache.hadoop.io.Text\"}");

+ 1 - 1
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/serializer/avro/TestAvroSerialization.java

@@ -33,7 +33,7 @@ public class TestAvroSerialization {
   @Test
   public void testSpecific() throws Exception {
     AvroRecord before = new AvroRecord();
-    before.intField = 5;
+    before.setIntField(5);
     AvroRecord after = SerializationTestUtil.testSerialization(conf, before);
     assertEquals(before, after);
   }

+ 6 - 6
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapreduce/jobhistory/JobQueueChangeEvent.java

@@ -20,7 +20,6 @@ package org.apache.hadoop.mapreduce.jobhistory;
 
 import java.util.Set;
 
-import org.apache.avro.util.Utf8;
 import org.apache.hadoop.mapreduce.JobID;
 import org.apache.hadoop.util.StringUtils;
 import org.apache.hadoop.yarn.api.records.timelineservice.TimelineEvent;
@@ -31,8 +30,8 @@ public class JobQueueChangeEvent implements HistoryEvent {
   private JobQueueChange datum = new JobQueueChange();
   
   public JobQueueChangeEvent(JobID id, String queueName) {
-    datum.jobid = new Utf8(id.toString());
-    datum.jobQueueName = new Utf8(queueName);
+    datum.setJobid(id.toString());
+    datum.setJobQueueName(queueName);
   }
   
   JobQueueChangeEvent() { }
@@ -54,13 +53,14 @@ public class JobQueueChangeEvent implements HistoryEvent {
   
   /** Get the Job ID */
   public JobID getJobId() {
-    return JobID.forName(datum.jobid.toString());
+    return JobID.forName(datum.getJobid().toString());
   }
   
   /** Get the new Job queue name */
   public String getJobQueueName() {
-    if (datum.jobQueueName != null) {
-      return datum.jobQueueName.toString();
+    java.lang.CharSequence jobQueueName = datum.getJobQueueName();
+    if (jobQueueName != null) {
+      return jobQueueName.toString();
     }
     return null;
   }

+ 1 - 1
hadoop-project/pom.xml

@@ -63,7 +63,7 @@
     <java.security.egd>file:///dev/urandom</java.security.egd>
 
     <!-- avro version -->
-    <avro.version>1.9.2</avro.version>
+    <avro.version>1.11.4</avro.version>
 
     <!-- jersey version -->
     <jersey.version>1.19.4</jersey.version>

+ 8 - 8
hadoop-tools/hadoop-rumen/src/main/java/org/apache/hadoop/tools/rumen/JobBuilder.java

@@ -460,7 +460,7 @@ public class JobBuilder {
     }
     task.setFinishTime(event.getFinishTime());
     task.setTaskStatus(getPre21Value(event.getTaskStatus()));
-    task.incorporateCounters(((TaskFinished) event.getDatum()).counters);
+    task.incorporateCounters(((TaskFinished) event.getDatum()).getCounters());
   }
 
   private void processTaskFailedEvent(TaskFailedEvent event) {
@@ -472,7 +472,7 @@ public class JobBuilder {
     task.setFinishTime(event.getFinishTime());
     task.setTaskStatus(getPre21Value(event.getTaskStatus()));
     TaskFailed t = (TaskFailed)(event.getDatum());
-    task.putDiagnosticInfo(t.error.toString());
+    task.putDiagnosticInfo(t.getError().toString());
     // killed task wouldn't have failed attempt.
     if (t.getFailedDueToAttempt() != null) {
       task.putFailedDueToAttemptId(t.getFailedDueToAttempt().toString());
@@ -542,7 +542,7 @@ public class JobBuilder {
     }
     attempt.setFinishTime(event.getFinishTime());
     attempt
-        .incorporateCounters(((TaskAttemptFinished) event.getDatum()).counters);
+        .incorporateCounters(((TaskAttemptFinished) event.getDatum()).getCounters());
   }
 
   private void processReduceAttemptFinishedEvent(
@@ -568,7 +568,7 @@ public class JobBuilder {
     attempt.setShuffleFinished(event.getShuffleFinishTime());
     attempt.setSortFinished(event.getSortFinishTime());
     attempt
-        .incorporateCounters(((ReduceAttemptFinished) event.getDatum()).counters);
+        .incorporateCounters(((ReduceAttemptFinished) event.getDatum()).getCounters());
     attempt.arraySetClockSplits(event.getClockSplits());
     attempt.arraySetCpuUsages(event.getCpuUsages());
     attempt.arraySetVMemKbytes(event.getVMemKbytes());
@@ -596,7 +596,7 @@ public class JobBuilder {
     // is redundant, but making this will add future-proofing.
     attempt.setFinishTime(event.getFinishTime());
     attempt
-      .incorporateCounters(((MapAttemptFinished) event.getDatum()).counters);
+      .incorporateCounters(((MapAttemptFinished) event.getDatum()).getCounters());
     attempt.arraySetClockSplits(event.getClockSplits());
     attempt.arraySetCpuUsages(event.getCpuUsages());
     attempt.arraySetVMemKbytes(event.getVMemKbytes());
@@ -661,11 +661,11 @@ public class JobBuilder {
 
     JobFinished job = (JobFinished)event.getDatum();
     Map<String, Long> countersMap =
-        JobHistoryUtils.extractCounters(job.totalCounters);
+        JobHistoryUtils.extractCounters(job.getTotalCounters());
     result.putTotalCounters(countersMap);
-    countersMap = JobHistoryUtils.extractCounters(job.mapCounters);
+    countersMap = JobHistoryUtils.extractCounters(job.getMapCounters());
     result.putMapCounters(countersMap);
-    countersMap = JobHistoryUtils.extractCounters(job.reduceCounters);
+    countersMap = JobHistoryUtils.extractCounters(job.getReduceCounters());
     result.putReduceCounters(countersMap);
   }
 

+ 3 - 3
hadoop-tools/hadoop-rumen/src/main/java/org/apache/hadoop/tools/rumen/JobHistoryUtils.java

@@ -157,9 +157,9 @@ public class JobHistoryUtils {
   static Map<String, Long> extractCounters(JhCounters counters) {
     Map<String, Long> countersMap = new HashMap<String, Long>();
     if (counters != null) {
-      for (JhCounterGroup group : counters.groups) {
-        for (JhCounter counter : group.counts) {
-          countersMap.put(counter.name.toString(), counter.value);
+      for (JhCounterGroup group : counters.getGroups()) {
+        for (JhCounter counter : group.getCounts()) {
+          countersMap.put(counter.getName().toString(), counter.getValue());
         }
       }
     }

+ 4 - 4
hadoop-tools/hadoop-rumen/src/main/java/org/apache/hadoop/tools/rumen/LoggedTask.java

@@ -268,11 +268,11 @@ public class LoggedTask implements DeepCompare {
       String counterName) {
     counterName = canonicalizeCounterName(counterName);
 
-    for (JhCounterGroup group : counters.groups) {
-      for (JhCounter counter : group.counts) {
+    for (JhCounterGroup group : counters.getGroups()) {
+      for (JhCounter counter : group.getCounts()) {
         if (counterName
-            .equals(canonicalizeCounterName(counter.name.toString()))) {
-          thunk.set(counter.value);
+            .equals(canonicalizeCounterName(counter.getName().toString()))) {
+          thunk.set(counter.getValue());
           return;
         }
       }

+ 4 - 4
hadoop-tools/hadoop-rumen/src/main/java/org/apache/hadoop/tools/rumen/LoggedTaskAttempt.java

@@ -636,11 +636,11 @@ public class LoggedTaskAttempt implements DeepCompare {
       String counterName) {
     counterName = canonicalizeCounterName(counterName);
 
-    for (JhCounterGroup group : counters.groups) {
-      for (JhCounter counter : group.counts) {
+    for (JhCounterGroup group : counters.getGroups()) {
+      for (JhCounter counter : group.getCounts()) {
         if (counterName
-            .equals(canonicalizeCounterName(counter.name.toString()))) {
-          thunk.set(counter.value);
+            .equals(canonicalizeCounterName(counter.getName().toString()))) {
+          thunk.set(counter.getValue());
           return;
         }
       }