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

Return error rather than panic when too many row groups are written #6629

Merged
merged 2 commits into from
Oct 28, 2024
Merged
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
45 changes: 44 additions & 1 deletion parquet/src/file/writer.rs
Original file line number Diff line number Diff line change
Expand Up @@ -378,7 +378,12 @@ fn write_bloom_filters<W: Write + Send>(
.ordinal()
.expect("Missing row group ordinal")
.try_into()
.expect("Negative row group ordinal");
.map_err(|_| {
ParquetError::General(format!(
"Negative row group ordinal: {})",
row_group.ordinal().unwrap()
))
})?;
let row_group_idx = row_group_idx as usize;
for (column_idx, column_chunk) in row_group.columns_mut().iter_mut().enumerate() {
if let Some(bloom_filter) = bloom_filters[row_group_idx][column_idx].take() {
Expand Down Expand Up @@ -1892,6 +1897,44 @@ mod tests {
assert_eq!(page_sizes[0], unenc_size);
}

#[test]
fn test_too_many_rowgroups() {
let message_type = "
message test_schema {
REQUIRED BYTE_ARRAY a (UTF8);
}
";
let schema = Arc::new(parse_message_type(message_type).unwrap());
let file: File = tempfile::tempfile().unwrap();
let props = Arc::new(
WriterProperties::builder()
.set_statistics_enabled(EnabledStatistics::None)
.set_max_row_group_size(1)
.build(),
);
let mut writer = SerializedFileWriter::new(&file, schema, props).unwrap();

// Create 32k empty rowgroups. Should error when i == 32768.
for i in 0..0x8001 {
match writer.next_row_group() {
Ok(mut row_group_writer) => {
assert_ne!(i, 0x8000);
let col_writer = row_group_writer.next_column().unwrap().unwrap();
col_writer.close().unwrap();
row_group_writer.close().unwrap();
}
Err(e) => {
assert_eq!(i, 0x8000);
assert_eq!(
e.to_string(),
"Parquet error: Parquet does not support more than 32767 row groups per file (currently: 32768)"
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

💯

);
}
}
}
writer.close().unwrap();
}

#[test]
fn test_size_statistics_with_repetition_and_nulls() {
let message_type = "
Expand Down
Loading