内部设计

概览

12 rectangular blocks arranged as a 4-row, 3-column layout. Each block includes 'x' and its location in the table starting with ('x',0,0) in the top-left, and a size of 5x8.

Dask 数组使用一个块网格来定义一个大型数组,这些块是更小的数组。这些数组可以是实际的数组,也可以是生成数组的函数。我们使用以下组件定义一个 Dask 数组:

  • 一个 Dask 图,其中包含一组特殊的键用于指定块,例如 ('x', 0, 0), ('x', 0, 1), ...(更多详情请参见Dask 图文档

  • 沿每个维度的一系列块大小,称为 chunks,例如 ((5, 5, 5, 5), (8, 8, 8))

  • 一个名称,用于标识 Dask 图中哪些键引用此数组,例如 'x'

  • 一个 NumPy dtype

示例

>>> import dask.array as da
>>> x = da.arange(0, 15, chunks=(5,))

>>> x.name
'arange-539766a'

>>> x.__dask_graph__()
<dask.highlevelgraph.HighLevelGraph at 0x7f9f6f686d68>

>>> dict(x.__dask_graph__())  # somewhat simplified
{('arange-539766a', 0): (np.arange, 0, 5),
 ('arange-539766a', 1): (np.arange, 5, 10),
 ('arange-539766a', 2): (np.arange, 10, 15)}

>>> x.chunks
((5, 5, 5),)

>>> x.dtype
dtype('int64')

Dask 图的键

根据特殊约定,我们使用 (name, i, j, k) 形式的元组来引用数组的每个块,其中 i, j, k 是块的索引,范围从 0 到该维度中的块数量。Dask 图必须包含引用这些键的键值对。此外,它可能还包含最终计算所需值所需的其他键值对(通常组织在 HighLevelGraph 中,但此处为了说明目的以扁平形式显示)

{
 ('x', 0, 0): (add, 1, ('y', 0, 0)),
 ('x', 0, 1): (add, 1, ('y', 0, 1)),
 ...
 ('y', 0, 0): (getitem, dataset, (slice(0, 1000), slice(0, 1000))),
 ('y', 0, 1): (getitem, dataset, (slice(0, 1000), slice(1000, 2000)))
 ...
}

可以通过 name 属性找到 Array 对象的名称。可以使用 .__dask_keys__() 方法获取嵌套的键列表。此外,可以使用 dask.array.core.flatten() 展平此列表。这在构建新字典时有时很有用。

分块

我们还存储沿每个轴的每个块的大小。这由一个元组的元组组成,外层元组的长度等于数组的维度数,内层元组的长度等于沿每个维度的块数。在上面示例中,此值如下所示

chunks = ((5, 5, 5, 5), (8, 8, 8))

请注意,这些数字不一定需要是规则的。我们通常创建规则大小的网格,但块在复杂切片后会改变形状。请注意,某些操作确实期望块形状具有某些对称性。例如,矩阵乘法要求两侧的块具有反对称形状。

chunks 反映数组属性的一些方式

  1. len(x.chunks) == x.ndim:chunks 的长度是维数

  2. tuple(map(sum, x.chunks)) == x.shape:每个内部 chunk 的总和是该维度的长度

  3. 每个内部 chunk 的长度是该维度中的键数。例如,对于 chunks == ((a, b), (d, e, f)) 且 name == 'x',我们的数组具有以下键的任务

    ('x', 0, 0), ('x', 0, 1), ('x', 0, 2)
    ('x', 1, 0), ('x', 1, 1), ('x', 1, 2)
    

元数据

许多数组操作依赖于知道 dtype (int, float, ...) 和 type (numpy, cupy, ...)。为了跟踪这些信息,所有 Dask 数组对象都有一个 _meta 属性,它包含一个具有相同 dtypes 的空 Numpy 对象。例如

>>> np_array = np.arange(15).reshape(3, 5)
>>> da_array = da.from_array(np_array, npartitions=2)
>>> da_array._meta
Empty Array
Shape: (0, 0)
dtype: int64
array([], shape=(0, 0), dtype=int64)

>>> ddf._meta.dtype
dtype: int64

在内部,Dask 数组会尽最大努力在所有操作中传播此信息,因此大多数情况下用户不必担心这一点。

创建数组对象

为了创建一个 da.Array 对象,我们需要一个包含这些特殊键的图

layer = {('x', 0, 0): ...}
dsk = HighLevelGraph.from_collections('x', layer, dependencies=())

一个指定此数组引用哪些键的名称

name = 'x'

以及一个 chunks 元组

chunks = ((5, 5, 5, 5), (8, 8, 8))

然后,使用这些元素,可以构造一个数组

x = da.Array(dsk, name, chunks)

简而言之,dask.array 操作更新 Dask 图,更新 dtypes,并跟踪 chunk 形状。

示例 - eye 函数

举个例子,让我们为 dask.array 构建 np.eye 函数来创建单位矩阵

def eye(n, blocksize):
    chunks = ((blocksize,) * (n // blocksize),
              (blocksize,) * (n // blocksize))

    name = 'eye' + next(tokens)  # unique identifier

    layer = {(name, i, j): (np.eye, blocksize)
                           if i == j else
                           (np.zeros, (blocksize, blocksize))
             for i in range(n // blocksize)
             for j in range(n // blocksize)}
    dsk = dask.highlevelgraph.HighLevelGraph.from_collections(name, layer, dependencies=())

    dtype = np.eye(0).dtype  # take dtype default from numpy

    return dask.array.Array(dsk, name, chunks, dtype)