-
Notifications
You must be signed in to change notification settings - Fork 5.9k
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
[data][api] implement
HudiDataSource
@MicroCheck //python:ray/data/tests/test_hudi Signed-off-by: Shiyan Xu <2701446+xushiyan@users.noreply.github.com>
- Loading branch information
Showing
10 changed files
with
271 additions
and
1 deletion.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -7,6 +7,7 @@ Data('s)? | |
[Dd]iscretizer(s)? | ||
dtype | ||
[Gg]roupby | ||
[Hh]udi | ||
[Ii]ndexable | ||
[Ii]ngest | ||
[Ii]nqueue(s)? | ||
|
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,87 @@ | ||
import logging | ||
from functools import partial | ||
from pathlib import PurePosixPath | ||
from typing import Dict, Iterator, List, Optional | ||
from urllib.parse import urljoin | ||
|
||
from ray.data._internal.util import _check_import | ||
from ray.data.block import BlockMetadata | ||
from ray.data.datasource.datasource import Datasource, ReadTask | ||
|
||
logger = logging.getLogger(__name__) | ||
|
||
|
||
class HudiDatasource(Datasource): | ||
"""Hudi datasource, for reading Apache Hudi table.""" | ||
|
||
def __init__( | ||
self, | ||
table_uri: str, | ||
storage_options: Optional[Dict[str, str]] = None, | ||
): | ||
_check_import(self, module="hudi", package="hudi-python") | ||
|
||
self._table_uri = table_uri | ||
self._storage_options = storage_options | ||
|
||
def get_read_tasks(self, parallelism: int) -> List["ReadTask"]: | ||
import pyarrow | ||
from hudi import HudiFileGroupReader, HudiTable | ||
|
||
def _perform_read( | ||
table_uri: str, | ||
base_file_paths: List[str], | ||
options: Dict[str, str], | ||
) -> Iterator["pyarrow.Table"]: | ||
for p in base_file_paths: | ||
fg_reader = HudiFileGroupReader(table_uri, options) | ||
batches = fg_reader.read_file_slice_by_base_file_path(p) | ||
yield pyarrow.Table.from_batches(batches) | ||
|
||
hudi_table = HudiTable(self._table_uri, self._storage_options) | ||
|
||
reader_options = {} | ||
reader_options.update(hudi_table.storage_options()) | ||
reader_options.update(hudi_table.hudi_options()) | ||
|
||
schema = hudi_table.get_schema() | ||
read_tasks = [] | ||
for file_slices in hudi_table.split_file_slices(parallelism): | ||
if len(file_slices) <= 0: | ||
continue | ||
|
||
num_rows = 0 | ||
relative_paths = [] | ||
input_files = [] | ||
size_bytes = 0 | ||
for f in file_slices: | ||
num_rows += f.num_records | ||
relative_path = f.base_file_relative_path() | ||
relative_paths.append(relative_path) | ||
full_path = urljoin( | ||
self._table_uri, PurePosixPath(relative_path).as_posix() | ||
) | ||
input_files.append(full_path) | ||
size_bytes += f.base_file_size | ||
|
||
metadata = BlockMetadata( | ||
num_rows=num_rows, | ||
schema=schema, | ||
input_files=input_files, | ||
size_bytes=size_bytes, | ||
exec_stats=None, | ||
) | ||
|
||
read_task = ReadTask( | ||
read_fn=partial( | ||
_perform_read, self._table_uri, relative_paths, reader_options | ||
), | ||
metadata=metadata, | ||
) | ||
read_tasks.append(read_task) | ||
|
||
return read_tasks | ||
|
||
def estimate_inmemory_data_size(self) -> Optional[int]: | ||
# TODO(xushiyan) add APIs to provide estimated in-memory size | ||
return None |
Binary file not shown.
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,108 @@ | ||
import os | ||
import zipfile | ||
|
||
import pyarrow as pa | ||
import pytest | ||
from pytest_lazyfixture import lazy_fixture | ||
|
||
import ray | ||
from ray.data.datasource.path_util import ( | ||
_resolve_paths_and_filesystem, | ||
_unwrap_protocol, | ||
) | ||
from ray.data.tests.conftest import * # noqa | ||
from ray.data.tests.mock_http_server import * # noqa | ||
from ray.tests.conftest import * # noqa | ||
|
||
PYARROW_LE_8_0_0 = tuple(int(s) for s in pa.__version__.split(".") if s.isnumeric()) < ( | ||
8, | ||
0, | ||
0, | ||
) | ||
pytestmark = pytest.mark.skipif( | ||
PYARROW_LE_8_0_0, reason="hudi only supported if pyarrow >= 8.0.0" | ||
) | ||
|
||
|
||
def _extract_testing_table(fixture_path: str, table_dir: str, target_dir: str) -> str: | ||
with zipfile.ZipFile(fixture_path, "r") as zip_ref: | ||
zip_ref.extractall(target_dir) | ||
return os.path.join(target_dir, table_dir) | ||
|
||
|
||
@pytest.mark.parametrize( | ||
"fs,data_path", | ||
[ | ||
(None, lazy_fixture("local_path")), | ||
(lazy_fixture("local_fs"), lazy_fixture("local_path")), | ||
], | ||
) | ||
def test_read_hudi_simple_cow_table(ray_start_regular_shared, fs, data_path): | ||
setup_data_path = _unwrap_protocol(data_path) | ||
target_testing_dir = os.path.join(setup_data_path, "test_hudi") | ||
fixture_path, _ = _resolve_paths_and_filesystem( | ||
"example://hudi-tables/0.x_cow_partitioned.zip", fs | ||
) | ||
target_table_path = _extract_testing_table( | ||
fixture_path[0], "trips_table", target_testing_dir | ||
) | ||
|
||
ds = ray.data.read_hudi(target_table_path) | ||
|
||
assert ds.schema().names == [ | ||
"_hoodie_commit_time", | ||
"_hoodie_commit_seqno", | ||
"_hoodie_record_key", | ||
"_hoodie_partition_path", | ||
"_hoodie_file_name", | ||
"ts", | ||
"uuid", | ||
"rider", | ||
"driver", | ||
"fare", | ||
"city", | ||
] | ||
assert ds.count() == 5 | ||
rows = ( | ||
ds.select_columns(["_hoodie_commit_time", "ts", "uuid", "fare"]) | ||
.sort("ts") | ||
.take_all() | ||
) | ||
assert rows == [ | ||
[ | ||
"20240402123035233", | ||
1695046462179, | ||
"9909a8b1-2d15-4d3d-8ec9-efc48c536a00", | ||
33.9, | ||
], | ||
[ | ||
"20240402123035233", | ||
1695091554788, | ||
"e96c4396-3fad-413a-a942-4cb36106d721", | ||
27.7, | ||
], | ||
[ | ||
"20240402123035233", | ||
1695115999911, | ||
"c8abbe79-8d89-47ea-b4ce-4d224bae5bfa", | ||
17.85, | ||
], | ||
[ | ||
"20240402123035233", | ||
1695159649087, | ||
"334e26e9-8355-45cc-97c6-c31daf0df330", | ||
19.1, | ||
], | ||
[ | ||
"20240402123035233", | ||
1695516137016, | ||
"e3cf430c-889d-4015-bc98-59bdce1e530c", | ||
34.15, | ||
], | ||
] | ||
|
||
|
||
if __name__ == "__main__": | ||
import sys | ||
|
||
sys.exit(pytest.main(["-v", __file__])) |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters