Skip to content

Commit

Permalink
Refactor block building decoder creation to connector specific class
Browse files Browse the repository at this point in the history
  • Loading branch information
jklamer committed Jun 6, 2024
1 parent 2afbc52 commit b7ddabb
Show file tree
Hide file tree
Showing 34 changed files with 2,131 additions and 1,623 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -44,23 +44,23 @@ public class AvroFileReader
public AvroFileReader(
TrinoInputFile inputFile,
Schema schema,
AvroTypeManager avroTypeManager)
AvroTypeBlockHandler avroTypeBlockHandler)
throws IOException, AvroTypeException
{
this(inputFile, schema, avroTypeManager, 0, OptionalLong.empty());
this(inputFile, schema, avroTypeBlockHandler, 0, OptionalLong.empty());
}

public AvroFileReader(
TrinoInputFile inputFile,
Schema schema,
AvroTypeManager avroTypeManager,
AvroTypeBlockHandler avroTypeBlockHandler,
long offset,
OptionalLong length)
throws IOException, AvroTypeException
{
requireNonNull(inputFile, "inputFile is null");
requireNonNull(schema, "schema is null");
requireNonNull(avroTypeManager, "avroTypeManager is null");
requireNonNull(avroTypeBlockHandler, "avroTypeBlockHandler is null");
long fileSize = inputFile.length();

verify(offset >= 0, "offset is negative");
Expand All @@ -69,7 +69,7 @@ public AvroFileReader(
end = length.stream().map(l -> l + offset).findFirst();
end.ifPresent(endLong -> verify(endLong <= fileSize, "offset plus length is greater than data size"));
input = new TrinoDataInputStream(inputFile.newStream());
dataReader = new AvroPageDataReader(schema, avroTypeManager);
dataReader = new AvroPageDataReader(schema, avroTypeBlockHandler);
try {
fileReader = new DataFileReader<>(new TrinoDataInputStreamAsAvroSeekableInput(input, fileSize), dataReader);
fileReader.sync(offset);
Expand All @@ -79,7 +79,7 @@ public AvroFileReader(
// so the exception is wrapped in a runtime exception that must be unwrapped
throw runtimeWrapper.getAvroTypeException();
}
avroTypeManager.configure(fileReader.getMetaKeys().stream().collect(toImmutableMap(Function.identity(), fileReader::getMeta)));
avroTypeBlockHandler.configure(fileReader.getMetaKeys().stream().collect(toImmutableMap(Function.identity(), fileReader::getMeta)));
}

public long getCompletedBytes()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -11,7 +11,7 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package io.trino.plugin.hive.avro;
package io.trino.hive.formats.avro;

public final class AvroHiveConstants
{
Expand Down
Loading

0 comments on commit b7ddabb

Please sign in to comment.