[jira] [Created] (FLINK-18915) FIXED_PATH(dummy Hadoop Path) with WriterImpl cause ORC

classic Classic list List threaded Threaded
1 message Options
Reply | Threaded
Open this post in threaded view
|

[jira] [Created] (FLINK-18915) FIXED_PATH(dummy Hadoop Path) with WriterImpl cause ORC

Shang Yuanchun (Jira)
wei created FLINK-18915:
---------------------------

             Summary: FIXED_PATH(dummy Hadoop Path) with WriterImpl cause ORC
                 Key: FLINK-18915
                 URL: https://issues.apache.org/jira/browse/FLINK-18915
             Project: Flink
          Issue Type: Bug
          Components: Formats (JSON, Avro, Parquet, ORC, SequenceFile)
    Affects Versions: 1.11.1, 1.11.0
            Reporter: wei


# OrcBulkWriterFactory
{code:java}
@Override
public BulkWriter<T> create(FSDataOutputStream out) throws IOException {
   OrcFile.WriterOptions opts = getWriterOptions();
   opts.physicalWriter(new PhysicalWriterImpl(out, opts));

   return new OrcBulkWriter<>(vectorizer, new WriterImpl(null, FIXED_PATH, opts));
}{code}
 

# MemoryManagerImpl
{code:java}
//
public void addWriter(Path path, long requestedAllocation,
                            Callback callback) throws IOException {
  checkOwner();
  WriterInfo oldVal = writerList.get(path);
  // this should always be null, but we handle the case where the memory
  // manager wasn't told that a writer wasn't still in use and the task
  // starts writing to the same path.
  if (oldVal == null) {
    oldVal = new WriterInfo(requestedAllocation, callback);
    writerList.put(path, oldVal);
    totalAllocation += requestedAllocation;
  } else {
    // handle a new writer that is writing to the same path
    totalAllocation += requestedAllocation - oldVal.allocation;
    oldVal.allocation = requestedAllocation;
    oldVal.callback = callback;
  }
  updateScale(true);
}
{code}
SinkTask may have multi BulkWriter create, FIXED_PATH will cause overlay the last writer callback;Last writer's WriterImpl#checkMemory will never called;

 

 



--
This message was sent by Atlassian Jira
(v8.3.4#803005)