Skip to content
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

Support reusing dictionary when encoding values #2008

Merged
merged 6 commits into from
Jan 20, 2025
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension


Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
1 change: 0 additions & 1 deletion Cargo.lock

Some generated files are not rendered by default. Learn more about how customized files appear on GitHub.

16 changes: 4 additions & 12 deletions bench-vortex/benches/compressor_throughput.rs
Original file line number Diff line number Diff line change
Expand Up @@ -8,7 +8,7 @@ use vortex::array::{ConstantArray, VarBinViewArray};
use vortex::buffer::Buffer;
use vortex::compute::{compare, try_cast, Operator};
use vortex::dtype::PType;
use vortex::encodings::dict::{dict_encode_varbinview, DictArray};
use vortex::encodings::dict::dict_encode;
use vortex::encodings::fsst::{fsst_compress, fsst_train_compressor};
use vortex::sampling_compressor::compressors::alp::ALPCompressor;
use vortex::sampling_compressor::compressors::alp_rd::ALPRDCompressor;
Expand Down Expand Up @@ -106,24 +106,16 @@ fn strings(c: &mut Criterion) {
group.throughput(Throughput::Bytes(num_values * 8));

let varbinview_arr = VarBinViewArray::from_iter_str(gen_varbin_words(1_000_000, 0.00005));
let (codes, values) = dict_encode_varbinview(&varbinview_arr);
let dict = dict_encode(varbinview_arr.as_ref()).unwrap();
group.throughput(Throughput::Bytes(varbinview_arr.to_array().nbytes() as u64));
group.bench_function("dict_decode_varbinview", |b| {
b.iter_batched(
|| DictArray::try_new(codes.to_array(), values.to_array()).unwrap(),
|dict_arr| black_box(dict_arr.into_canonical().unwrap()),
BatchSize::SmallInput,
);
b.iter(|| black_box(dict.clone().into_canonical().unwrap()));
});

let fsst_compressor = fsst_train_compressor(&varbinview_arr.to_array()).unwrap();
let fsst_array = fsst_compress(&varbinview_arr.to_array(), &fsst_compressor).unwrap();
group.bench_function("fsst_decompress_varbinview", |b| {
b.iter_batched(
|| fsst_array.clone(),
|fsst_arr| black_box(fsst_arr.into_canonical().unwrap()),
BatchSize::SmallInput,
);
b.iter(|| black_box(fsst_array.clone().into_canonical().unwrap()));
});
}

Expand Down
2 changes: 1 addition & 1 deletion docs/quickstart.rst
Original file line number Diff line number Diff line change
Expand Up @@ -46,7 +46,7 @@ Use :func:`~vortex.encoding.compress` to compress the Vortex array and check the

>>> cvtx = vortex.compress(vtx)
>>> cvtx.nbytes
16604
16791
>>> cvtx.nbytes / vtx.nbytes
0.11...

Expand Down
1 change: 0 additions & 1 deletion encodings/dict/Cargo.toml
Original file line number Diff line number Diff line change
Expand Up @@ -15,7 +15,6 @@ readme = { workspace = true }

[dependencies]
arrow-buffer = { workspace = true }
hashbrown = { workspace = true }
num-traits = { workspace = true }
serde = { workspace = true }
vortex-array = { workspace = true }
Expand Down
46 changes: 17 additions & 29 deletions encodings/dict/benches/dict_compress.rs
Original file line number Diff line number Diff line change
@@ -1,16 +1,16 @@
#![allow(clippy::unwrap_used)]
#![allow(clippy::cast_possible_truncation)]

use criterion::{black_box, criterion_group, criterion_main, BatchSize, Criterion, Throughput};
use criterion::{black_box, criterion_group, criterion_main, Criterion, Throughput};
use rand::distributions::{Alphanumeric, Uniform};
use rand::prelude::SliceRandom;
use rand::{thread_rng, Rng};
use vortex_array::array::{PrimitiveArray, VarBinArray, VarBinViewArray};
use vortex_array::nbytes::ArrayNBytes;
use vortex_array::validity::Validity;
use vortex_array::{IntoCanonical as _, ToArrayData};
use vortex_array::IntoCanonical as _;
use vortex_buffer::Buffer;
use vortex_dict::{dict_encode_primitive, dict_encode_varbin, dict_encode_varbinview, DictArray};
use vortex_dict::dict_encode;

fn gen_primitive_dict(len: usize, uniqueness: f64) -> PrimitiveArray {
let mut rng = thread_rng();
Expand All @@ -37,64 +37,52 @@ fn gen_varbin_words(len: usize, uniqueness: f64) -> Vec<String> {
.collect()
}

fn dict_encode(c: &mut Criterion) {
fn encode(c: &mut Criterion) {
let mut group = c.benchmark_group("dict_encode");

let primitive_arr = gen_primitive_dict(1_000_000, 0.00005);
group.throughput(Throughput::Bytes(primitive_arr.nbytes() as u64));
group.bench_function("dict_encode_primitives", |b| {
b.iter(|| black_box(dict_encode_primitive(&primitive_arr)));
b.iter(|| black_box(dict_encode(primitive_arr.as_ref())));
});

let varbin_arr = VarBinArray::from(gen_varbin_words(1_000_000, 0.00005));
group.throughput(Throughput::Bytes(varbin_arr.nbytes() as u64));
group.bench_function("dict_encode_varbin", |b| {
b.iter(|| black_box(dict_encode_varbin(&varbin_arr)));
b.iter(|| black_box(dict_encode(varbin_arr.as_ref())));
});

let varbinview_arr = VarBinViewArray::from_iter_str(gen_varbin_words(1_000_000, 0.00005));
group.throughput(Throughput::Bytes(varbinview_arr.nbytes() as u64));
group.bench_function("dict_encode_varbinview", |b| {
b.iter(|| black_box(dict_encode_varbinview(&varbinview_arr)));
group.bench_function("dict_encode_view", |b| {
b.iter(|| black_box(dict_encode(varbinview_arr.as_ref())));
});
}

fn dict_decode(c: &mut Criterion) {
fn decode(c: &mut Criterion) {
let mut group = c.benchmark_group("dict_decode");

let primitive_arr = gen_primitive_dict(1_000_000, 0.00005);
let (codes, values) = dict_encode_primitive(&primitive_arr);
let dict = dict_encode(primitive_arr.as_ref()).unwrap();
group.throughput(Throughput::Bytes(primitive_arr.nbytes() as u64));
group.bench_function("dict_decode_primitives", |b| {
b.iter_batched(
|| DictArray::try_new(codes.to_array(), values.to_array()).unwrap(),
|dict_arr| black_box(dict_arr.into_canonical().unwrap()),
BatchSize::SmallInput,
);
b.iter(|| black_box(dict.clone().into_canonical().unwrap()));
});

let varbin_arr = VarBinArray::from(gen_varbin_words(1_000_000, 0.00005));
let (codes, values) = dict_encode_varbin(&varbin_arr);
let dict = dict_encode(varbin_arr.as_ref()).unwrap();
group.throughput(Throughput::Bytes(varbin_arr.nbytes() as u64));
group.bench_function("dict_decode_varbin", |b| {
b.iter_batched(
|| DictArray::try_new(codes.to_array(), values.to_array()).unwrap(),
|dict_arr| black_box(dict_arr.into_canonical().unwrap()),
BatchSize::SmallInput,
);
b.iter(|| black_box(dict.clone().into_canonical().unwrap()));
});

let varbinview_arr = VarBinViewArray::from_iter_str(gen_varbin_words(1_000_000, 0.00005));
let (codes, values) = dict_encode_varbinview(&varbinview_arr);
let dict = dict_encode(varbinview_arr.as_ref()).unwrap();
group.throughput(Throughput::Bytes(varbin_arr.nbytes() as u64));
group.bench_function("dict_decode_varbinview", |b| {
b.iter_batched(
|| DictArray::try_new(codes.to_array(), values.to_array()).unwrap(),
|dict_arr| black_box(dict_arr.into_canonical().unwrap()),
BatchSize::SmallInput,
);
group.bench_function("dict_decode_view", |b| {
b.iter(|| black_box(dict.clone().into_canonical().unwrap()));
});
}

criterion_group!(benches, dict_encode, dict_decode);
criterion_group!(benches, encode, decode);
criterion_main!(benches);
Loading
Loading