-
Notifications
You must be signed in to change notification settings - Fork 3.1k
Closed
Description
Observation: Spark job intermittently hangs with driver thread stuck in reading avro file with S3A inputstream, according to Thread dump here.
java.io.DataInputStream.read(DataInputStream.java:149)
org.apache.iceberg.hadoop.HadoopStreams$HadoopSeekableInputStream.read(HadoopStreams.java:112)
org.apache.iceberg.avro.AvroIO$AvroInputStreamAdapter.read(AvroIO.java:106)
org.apache.iceberg.shaded.org.apache.avro.file.DataFileReader.openReader(DataFileReader.java:61)
org.apache.iceberg.avro.AvroIterable.newFileReader(AvroIterable.java:94)
org.apache.iceberg.avro.AvroIterable.iterator(AvroIterable.java:77)
org.apache.iceberg.avro.AvroIterable.iterator(AvroIterable.java:37)
org.apache.iceberg.relocated.com.google.common.collect.Iterables.addAll(Iterables.java:320)
org.apache.iceberg.relocated.com.google.common.collect.Lists.newLinkedList(Lists.java:237)
org.apache.iceberg.ManifestLists.read(ManifestLists.java:46)
org.apache.iceberg.BaseSnapshot.cacheManifests(BaseSnapshot.java:127)
org.apache.iceberg.BaseSnapshot.dataManifests(BaseSnapshot.java:149)
org.apache.iceberg.DataTableScan.planFiles(DataTableScan.java:74)
org.apache.iceberg.BaseTableScan.planFiles(BaseTableScan.java:212)
org.apache.iceberg.DataTableScan.planFiles(DataTableScan.java:28)
org.apache.iceberg.BaseTableScan.planTasks(BaseTableScan.java:245)
org.apache.iceberg.DataTableScan.planTasks(DataTableScan.java:28)
org.apache.iceberg.spark.source.Reader.tasks(Reader.java:365)
Affected code: DataFileReader.openReader
iceberg/core/src/main/java/org/apache/iceberg/avro/AvroIterable.java
Lines 98 to 105 in f17879c
| private DataFileReader<D> newFileReader() { | |
| try { | |
| return (DataFileReader<D>) DataFileReader.openReader( | |
| AvroIO.stream(file.newStream(), file.getLength()), reader); | |
| } catch (IOException e) { | |
| throw new RuntimeIOException(e, "Failed to open file: %s", file); | |
| } | |
| } |
Root cause: Avro reader checking header has incorrect logic and will get stuck in infinite loop when S3A input stream reads fewer bytes than requested. This has been raised with Avro https://issues.apache.org/jira/browse/AVRO-2944
for (int c = 0; c < magic.length; c = in.read(magic, c, magic.length - c)) {}
Fix in Avro:
for (int c = 0; c < magic.length; c += in.read(magic, c, magic.length - c)) {}
Iceberg change needed: There is PR for Avro apache/avro#969 and I'm wondering if Iceberg can take this Avro patch in, once merged.
Reactions are currently unavailable
Metadata
Metadata
Assignees
Labels
No labels