cxzl25 commented on code in PR #24896:
URL: https://github.com/apache/flink/pull/24896#discussion_r1628734257


##########
flink-filesystems/flink-hadoop-fs/src/main/java/org/apache/flink/runtime/fs/hdfs/BaseHadoopFsRecoverableFsDataOutputStream.java:
##########
@@ -59,7 +59,7 @@ public void write(byte[] b, int off, int len) throws 
IOException {
 
     @Override
     public void flush() throws IOException {
-        out.hsync();
+        out.hflush();

Review Comment:
   
https://github.com/apache/hadoop/blob/2ee0bf953492b66765d3d2c902407fbf9bceddec/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/DFSOutputStream.java#L586-L608
   
   ```java
   "write.hdfs: Writer -> write.hdfs: Committer (23/80)#0" Id=112 TIMED_WAITING 
on java.util.LinkedList@544a8b34
        at java.lang.Object.wait(Native Method)
        -  waiting on java.util.LinkedList@544a8b34
        at 
org.apache.hadoop.hdfs.DFSOutputStream.waitForAckedSeqno(DFSOutputStream.java:2534)
        at 
org.apache.hadoop.hdfs.DFSOutputStream.flushOrSync(DFSOutputStream.java:2421)
        at 
org.apache.hadoop.hdfs.DFSOutputStream.hsync(DFSOutputStream.java:2324)
        at 
org.apache.hadoop.fs.FSDataOutputStream.hsync(FSDataOutputStream.java:139)
        at 
org.apache.flink.runtime.fs.hdfs.BaseHadoopFsRecoverableFsDataOutputStream.flush(BaseHadoopFsRecoverableFsDataOutputStream.java:63)
        at 
org.apache.flink.util.CloseShieldOutputStream.flush(CloseShieldOutputStream.java:49)
        at java.io.FilterOutputStream.flush(FilterOutputStream.java:140)
        at java.io.BufferedOutputStream.flush(BufferedOutputStream.java:141)
        -  locked 
org.apache.avro.file.DataFileWriter$BufferedFileOutputStream@4d7428c3
        at 
org.apache.avro.file.DataFileWriter$BufferedFileOutputStream.flush(DataFileWriter.java:493)
        -  locked 
org.apache.avro.file.DataFileWriter$BufferedFileOutputStream@4d7428c3
        at 
org.apache.avro.io.DirectBinaryEncoder.flush(DirectBinaryEncoder.java:63)
        at 
org.apache.avro.file.DataFileStream$DataBlock.writeBlockTo(DataFileStream.java:409)
        at 
org.apache.avro.file.DataFileWriter.writeBlock(DataFileWriter.java:408)
        at 
org.apache.avro.file.DataFileWriter.writeIfBlockFull(DataFileWriter.java:351)
        at org.apache.avro.file.DataFileWriter.append(DataFileWriter.java:320)
        at 
org.apache.flink.formats.avro.AvroBulkWriter.addElement(AvroBulkWriter.java:44)
        at 
org.apache.flink.streaming.api.functions.sink.filesystem.BulkPartWriter.write(BulkPartWriter.java:51)
        at 
org.apache.flink.connector.file.sink.writer.FileWriterBucket.write(FileWriterBucket.java:191)
        at 
org.apache.flink.connector.file.sink.writer.FileWriter.write(FileWriter.java:198)
        at 
org.apache.flink.streaming.runtime.operators.sink.SinkWriterOperator.processElement(SinkWriterOperator.java:158)
        at 
org.apache.flink.streaming.runtime.io.RecordProcessorUtils$$Lambda$782/964030759.accept(Unknown
 Source)
        at 
org.apache.flink.streaming.runtime.tasks.OneInputStreamTask$StreamTaskNetworkOutput.emitRecord(OneInputStreamTask.java:237)
   ```



-- 
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...@flink.apache.org

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

Reply via email to