From 51a52010457326e64aaa258bcbf4439ae4b1ec9c Mon Sep 17 00:00:00 2001 From: Peter Andreas Entschev Date: Tue, 23 Jul 2024 21:09:57 +0200 Subject: [PATCH] Add cuDF spilling argument tests --- dask_cuda/tests/test_dask_cuda_worker.py | 63 ++++++++++++++++++++++ dask_cuda/tests/test_local_cuda_cluster.py | 44 +++++++++++++++ 2 files changed, 107 insertions(+) diff --git a/dask_cuda/tests/test_dask_cuda_worker.py b/dask_cuda/tests/test_dask_cuda_worker.py index 974ad1319..aeab9c7f7 100644 --- a/dask_cuda/tests/test_dask_cuda_worker.py +++ b/dask_cuda/tests/test_dask_cuda_worker.py @@ -231,6 +231,69 @@ def test_rmm_logging(loop): # noqa: F811 assert v is rmm.mr.LoggingResourceAdaptor +def test_cudf_spill_disabled(loop): # noqa: F811 + cudf = pytest.importorskip("cudf") + with popen(["dask", "scheduler", "--port", "9369", "--no-dashboard"]): + with popen( + [ + "dask", + "cuda", + "worker", + "127.0.0.1:9369", + "--host", + "127.0.0.1", + "--no-dashboard", + ] + ): + with Client("127.0.0.1:9369", loop=loop) as client: + assert wait_workers(client, n_gpus=get_n_gpus()) + + cudf_spill = client.run( + cudf.get_option, "spill", + ) + for v in cudf_spill.values(): + assert v is False + + cudf_spill_stats = client.run( + cudf.get_option, "spill_stats" + ) + for v in cudf_spill_stats.values(): + assert v == 0 + + +def test_cudf_spill(loop): # noqa: F811 + cudf = pytest.importorskip("cudf") + with popen(["dask", "scheduler", "--port", "9369", "--no-dashboard"]): + with popen( + [ + "dask", + "cuda", + "worker", + "127.0.0.1:9369", + "--host", + "127.0.0.1", + "--no-dashboard", + "--enable-cudf-spill", + "--cudf-spill-stats", + "2", + ] + ): + with Client("127.0.0.1:9369", loop=loop) as client: + assert wait_workers(client, n_gpus=get_n_gpus()) + + cudf_spill = client.run( + cudf.get_option, "spill" + ) + for v in cudf_spill.values(): + assert v is True + + cudf_spill_stats = client.run( + cudf.get_option, "spill_stats" + ) + for v in cudf_spill_stats.values(): + assert v == 2 + + @patch.dict(os.environ, {"CUDA_VISIBLE_DEVICES": "0"}) def test_dashboard_address(loop): # noqa: F811 with popen(["dask", "scheduler", "--port", "9369", "--no-dashboard"]): diff --git a/dask_cuda/tests/test_local_cuda_cluster.py b/dask_cuda/tests/test_local_cuda_cluster.py index b05389e4c..b5e67d41a 100644 --- a/dask_cuda/tests/test_local_cuda_cluster.py +++ b/dask_cuda/tests/test_local_cuda_cluster.py @@ -500,6 +500,50 @@ async def test_worker_fraction_limits(): ) +@gen_test(timeout=20) +async def test_cudf_spill_disabled(): + cudf = pytest.importorskip("cudf") + + async with LocalCUDACluster( + asynchronous=True, + ) as cluster: + async with Client(cluster, asynchronous=True) as client: + cudf_spill = await client.run( + cudf.get_option, "spill", + ) + for v in cudf_spill.values(): + assert v is False + + cudf_spill_stats = await client.run( + cudf.get_option, "spill_stats", + ) + for v in cudf_spill_stats.values(): + assert v == 0 + + +@gen_test(timeout=20) +async def test_cudf_spill(): + cudf = pytest.importorskip("cudf") + + async with LocalCUDACluster( + enable_cudf_spill=True, + cudf_spill_stats=2, + asynchronous=True, + ) as cluster: + async with Client(cluster, asynchronous=True) as client: + cudf_spill = await client.run( + cudf.get_option, "spill", + ) + for v in cudf_spill.values(): + assert v is True + + cudf_spill_stats = await client.run( + cudf.get_option, "spill_stats", + ) + for v in cudf_spill_stats.values(): + assert v == 2 + + @pytest.mark.parametrize( "protocol", ["ucx", "ucxx"],