-
Notifications
You must be signed in to change notification settings - Fork 9.1k
HADOOP-19394. Integrate with AAL's readVectored(). #7720
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
base: trunk
Are you sure you want to change the base?
Conversation
@steveloughran @mukund-thakur could you please review this initial version, couple of small TODO's coming (more tests, and using the |
* @throws IOException IOE if any. | ||
*/ | ||
@Override | ||
public synchronized void readVectored(List<? extends FileRange> ranges, |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
just for my knowledge- why is this a synchronised method?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
sorry my miss.. we just discussed, this does not need to be synchronized. will cut
List<ObjectRange> objectRanges = new ArrayList<>(); | ||
|
||
for (FileRange range : ranges) { | ||
CompletableFuture<ByteBuffer> result = new CompletableFuture<>(); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: may be rename to data?
💔 -1 overall
This message was automatically generated. |
💔 -1 overall
This message was automatically generated. |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
LGTM
💔 -1 overall
This message was automatically generated. |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
looks good, though you'll need to include the sdk update on this PR, which can then be split off separately
import org.apache.hadoop.fs.statistics.IOStatistics; | ||
import org.apache.hadoop.fs.statistics.StreamStatisticNames; | ||
|
||
import org.junit.Test; |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: ordering
why the SDK upgrade? |
3219341
to
5121093
Compare
💔 -1 overall
This message was automatically generated. |
5121093
to
142d1bb
Compare
🎊 +1 overall
This message was automatically generated. |
🎊 +1 overall
This message was automatically generated. |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
another review
@@ -207,7 +207,7 @@ | |||
<aws-java-sdk.version>1.12.720</aws-java-sdk.version> | |||
<aws-java-sdk-v2.version>2.29.52</aws-java-sdk-v2.version> | |||
<amazon-s3-encryption-client-java.version>3.1.1</amazon-s3-encryption-client-java.version> | |||
<amazon-s3-analyticsaccelerator-s3.version>1.0.0</amazon-s3-analyticsaccelerator-s3.version> | |||
<amazon-s3-analyticsaccelerator-s3.version>1.2.0</amazon-s3-analyticsaccelerator-s3.version> |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
for the final merge, this should be pulled out to its own.
should we try to get into 3.4.2 now?
|
||
import org.apache.hadoop.fs.FileRange; |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: import block
@@ -128,6 +141,48 @@ public int read(byte[] buf, int off, int len) throws IOException { | |||
return bytesRead; | |||
} | |||
|
|||
/** | |||
* {@inheritDoc} |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I'd put that at L148 and cut the params/IOE as superfluous
} | ||
|
||
/** | ||
* {@inheritDoc} |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
same
return conf; | ||
} | ||
|
||
@Override | ||
protected AbstractFSContract createContract(Configuration conf) { | ||
return new S3AContract(conf); | ||
} | ||
|
||
@Override |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
add a javadoc explaining why the override
public void testReadVectoredWithAALStatsCollection() throws Exception { | ||
|
||
List<FileRange> fileRanges = createSampleNonOverlappingRanges(); | ||
try (FSDataInputStream in = openVectorFile()){ |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit, space before {
|
||
// AAL does not do any range coalescing, so input and combined ranges are the same. | ||
this.getS3AStreamStatistics().readVectoredOperationStarted(ranges.size(), ranges.size()); | ||
inputStream.readVectored(objectRanges, allocate, release); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
does this call release on errors? curious -and hopeful
|
||
List<FileRange> fileRanges = createSampleNonOverlappingRanges(); | ||
try (FSDataInputStream in = openVectorFile()){ | ||
in.readVectored(fileRanges, getAllocate()); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Are we not verifying the data after vectored read.
@@ -63,12 +75,35 @@ protected Configuration createConfiguration() { | |||
// This issue is tracked in: | |||
// https://github.com/awslabs/analytics-accelerator-s3/issues/218 | |||
skipForAnyEncryptionExceptSSES3(conf); | |||
conf.set("fs.contract.vector-io-early-eof-check", "false"); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
why this is removed?
Description of PR
Adds in support for AAL's readVectored().
How was this patch tested?
Testing in progress.