Merge branch 'main' into sync-improvements

This commit is contained in:
Pekka Enberg
2025-09-22 07:35:39 +03:00
committed by GitHub
46 changed files with 10025 additions and 4020 deletions

13
Cargo.lock generated
View File

@@ -1100,6 +1100,19 @@ version = "1.15.0"
source = "registry+https://github.com/rust-lang/crates.io-index"
checksum = "48c757948c5ede0e46177b7add2e67155f70e33c07fea8284df6576da70b3719"
[[package]]
name = "encryption-throughput"
version = "0.1.0"
dependencies = [
"clap",
"futures",
"hex",
"rand 0.9.2",
"tokio",
"tracing-subscriber",
"turso",
]
[[package]]
name = "endian-type"
version = "0.1.2"

View File

@@ -32,8 +32,11 @@ members = [
"whopper",
"perf/throughput/turso",
"perf/throughput/rusqlite",
"perf/encryption"
]
exclude = [
"perf/latency/limbo",
]
exclude = ["perf/latency/limbo"]
[workspace.package]
version = "0.2.0-pre.3"

View File

@@ -15,6 +15,7 @@ conn_raw_api = ["turso_core/conn_raw_api"]
experimental_indexes = []
antithesis = ["turso_core/antithesis"]
tracing_release = ["turso_core/tracing_release"]
encryption = ["turso_core/encryption"]
[dependencies]
turso_core = { workspace = true, features = ["io_uring"] }

View File

@@ -413,7 +413,7 @@ impl Connection {
.inner
.lock()
.map_err(|e| Error::MutexError(e.to_string()))?;
conn.busy_timeout(duration);
conn.set_busy_timeout(duration);
Ok(())
}
}

View File

@@ -34,7 +34,6 @@ impl<T> DerefMut for SpinLockGuard<'_, T> {
}
}
unsafe impl<T: Send> Send for SpinLock<T> {}
unsafe impl<T> Sync for SpinLock<T> {}
impl<T> SpinLock<T> {

File diff suppressed because it is too large Load Diff

File diff suppressed because it is too large Load Diff

View File

@@ -355,7 +355,7 @@ mod tests {
"View not materialized".to_string(),
));
}
let num_columns = view.columns.len();
let num_columns = view.column_schema.columns.len();
drop(view);
// Create a btree cursor

View File

@@ -75,6 +75,10 @@ impl HashableRow {
hasher.finish()
}
pub fn cached_hash(&self) -> u64 {
self.cached_hash
}
}
impl Hash for HashableRow {
@@ -168,7 +172,7 @@ impl Delta {
}
/// A pair of deltas for operators that process two inputs
#[derive(Debug, Clone)]
#[derive(Debug, Clone, Default)]
pub struct DeltaPair {
pub left: Delta,
pub right: Delta,
@@ -400,4 +404,57 @@ mod tests {
let weight = zset.iter().find(|(k, _)| **k == 1).map(|(_, w)| w);
assert_eq!(weight, Some(1));
}
#[test]
fn test_hashable_row_delta_operations() {
let mut delta = Delta::new();
// Test INSERT
delta.insert(1, vec![Value::Integer(1), Value::Integer(100)]);
assert_eq!(delta.len(), 1);
// Test UPDATE (DELETE + INSERT) - order matters!
delta.delete(1, vec![Value::Integer(1), Value::Integer(100)]);
delta.insert(1, vec![Value::Integer(1), Value::Integer(200)]);
assert_eq!(delta.len(), 3); // Should have 3 operations before consolidation
// Verify order is preserved
let ops: Vec<_> = delta.changes.iter().collect();
assert_eq!(ops[0].1, 1); // First insert
assert_eq!(ops[1].1, -1); // Delete
assert_eq!(ops[2].1, 1); // Second insert
// Test consolidation
delta.consolidate();
// After consolidation, the first insert and delete should cancel out
// leaving only the second insert
assert_eq!(delta.len(), 1);
let final_row = &delta.changes[0];
assert_eq!(final_row.0.rowid, 1);
assert_eq!(
final_row.0.values,
vec![Value::Integer(1), Value::Integer(200)]
);
assert_eq!(final_row.1, 1);
}
#[test]
fn test_duplicate_row_consolidation() {
let mut delta = Delta::new();
// Insert same row twice
delta.insert(2, vec![Value::Integer(2), Value::Integer(300)]);
delta.insert(2, vec![Value::Integer(2), Value::Integer(300)]);
assert_eq!(delta.len(), 2);
delta.consolidate();
assert_eq!(delta.len(), 1);
// Weight should be 2 (sum of both inserts)
let final_row = &delta.changes[0];
assert_eq!(final_row.0.rowid, 2);
assert_eq!(final_row.1, 2);
}
}

View File

@@ -0,0 +1,295 @@
#![allow(dead_code)]
// Filter operator for DBSP-style incremental computation
// This operator filters rows based on predicates
use crate::incremental::dbsp::{Delta, DeltaPair};
use crate::incremental::operator::{
ComputationTracker, DbspStateCursors, EvalState, IncrementalOperator,
};
use crate::types::IOResult;
use crate::{Result, Value};
use std::sync::{Arc, Mutex};
/// Filter predicate for filtering rows
#[derive(Debug, Clone)]
pub enum FilterPredicate {
/// Column = value (using column index)
Equals { column_idx: usize, value: Value },
/// Column != value (using column index)
NotEquals { column_idx: usize, value: Value },
/// Column > value (using column index)
GreaterThan { column_idx: usize, value: Value },
/// Column >= value (using column index)
GreaterThanOrEqual { column_idx: usize, value: Value },
/// Column < value (using column index)
LessThan { column_idx: usize, value: Value },
/// Column <= value (using column index)
LessThanOrEqual { column_idx: usize, value: Value },
/// Column = Column comparisons
ColumnEquals { left_idx: usize, right_idx: usize },
/// Column != Column comparisons
ColumnNotEquals { left_idx: usize, right_idx: usize },
/// Column > Column comparisons
ColumnGreaterThan { left_idx: usize, right_idx: usize },
/// Column >= Column comparisons
ColumnGreaterThanOrEqual { left_idx: usize, right_idx: usize },
/// Column < Column comparisons
ColumnLessThan { left_idx: usize, right_idx: usize },
/// Column <= Column comparisons
ColumnLessThanOrEqual { left_idx: usize, right_idx: usize },
/// Logical AND of two predicates
And(Box<FilterPredicate>, Box<FilterPredicate>),
/// Logical OR of two predicates
Or(Box<FilterPredicate>, Box<FilterPredicate>),
/// No predicate (accept all rows)
None,
}
/// Filter operator - filters rows based on predicate
#[derive(Debug)]
pub struct FilterOperator {
predicate: FilterPredicate,
tracker: Option<Arc<Mutex<ComputationTracker>>>,
}
impl FilterOperator {
pub fn new(predicate: FilterPredicate) -> Self {
Self {
predicate,
tracker: None,
}
}
/// Get the predicate for this filter
pub fn predicate(&self) -> &FilterPredicate {
&self.predicate
}
pub fn evaluate_predicate(&self, values: &[Value]) -> bool {
match &self.predicate {
FilterPredicate::None => true,
FilterPredicate::Equals { column_idx, value } => {
if let Some(v) = values.get(*column_idx) {
return v == value;
}
false
}
FilterPredicate::NotEquals { column_idx, value } => {
if let Some(v) = values.get(*column_idx) {
return v != value;
}
false
}
FilterPredicate::GreaterThan { column_idx, value } => {
if let Some(v) = values.get(*column_idx) {
// Compare based on value types
match (v, value) {
(Value::Integer(a), Value::Integer(b)) => return a > b,
(Value::Float(a), Value::Float(b)) => return a > b,
(Value::Text(a), Value::Text(b)) => return a.as_str() > b.as_str(),
_ => {}
}
}
false
}
FilterPredicate::GreaterThanOrEqual { column_idx, value } => {
if let Some(v) = values.get(*column_idx) {
match (v, value) {
(Value::Integer(a), Value::Integer(b)) => return a >= b,
(Value::Float(a), Value::Float(b)) => return a >= b,
(Value::Text(a), Value::Text(b)) => return a.as_str() >= b.as_str(),
_ => {}
}
}
false
}
FilterPredicate::LessThan { column_idx, value } => {
if let Some(v) = values.get(*column_idx) {
match (v, value) {
(Value::Integer(a), Value::Integer(b)) => return a < b,
(Value::Float(a), Value::Float(b)) => return a < b,
(Value::Text(a), Value::Text(b)) => return a.as_str() < b.as_str(),
_ => {}
}
}
false
}
FilterPredicate::LessThanOrEqual { column_idx, value } => {
if let Some(v) = values.get(*column_idx) {
match (v, value) {
(Value::Integer(a), Value::Integer(b)) => return a <= b,
(Value::Float(a), Value::Float(b)) => return a <= b,
(Value::Text(a), Value::Text(b)) => return a.as_str() <= b.as_str(),
_ => {}
}
}
false
}
FilterPredicate::And(left, right) => {
// Temporarily create sub-filters to evaluate
let left_filter = FilterOperator::new((**left).clone());
let right_filter = FilterOperator::new((**right).clone());
left_filter.evaluate_predicate(values) && right_filter.evaluate_predicate(values)
}
FilterPredicate::Or(left, right) => {
let left_filter = FilterOperator::new((**left).clone());
let right_filter = FilterOperator::new((**right).clone());
left_filter.evaluate_predicate(values) || right_filter.evaluate_predicate(values)
}
// Column-to-column comparisons
FilterPredicate::ColumnEquals {
left_idx,
right_idx,
} => {
if let (Some(left), Some(right)) = (values.get(*left_idx), values.get(*right_idx)) {
return left == right;
}
false
}
FilterPredicate::ColumnNotEquals {
left_idx,
right_idx,
} => {
if let (Some(left), Some(right)) = (values.get(*left_idx), values.get(*right_idx)) {
return left != right;
}
false
}
FilterPredicate::ColumnGreaterThan {
left_idx,
right_idx,
} => {
if let (Some(left), Some(right)) = (values.get(*left_idx), values.get(*right_idx)) {
match (left, right) {
(Value::Integer(a), Value::Integer(b)) => return a > b,
(Value::Float(a), Value::Float(b)) => return a > b,
(Value::Text(a), Value::Text(b)) => return a.as_str() > b.as_str(),
_ => {}
}
}
false
}
FilterPredicate::ColumnGreaterThanOrEqual {
left_idx,
right_idx,
} => {
if let (Some(left), Some(right)) = (values.get(*left_idx), values.get(*right_idx)) {
match (left, right) {
(Value::Integer(a), Value::Integer(b)) => return a >= b,
(Value::Float(a), Value::Float(b)) => return a >= b,
(Value::Text(a), Value::Text(b)) => return a.as_str() >= b.as_str(),
_ => {}
}
}
false
}
FilterPredicate::ColumnLessThan {
left_idx,
right_idx,
} => {
if let (Some(left), Some(right)) = (values.get(*left_idx), values.get(*right_idx)) {
match (left, right) {
(Value::Integer(a), Value::Integer(b)) => return a < b,
(Value::Float(a), Value::Float(b)) => return a < b,
(Value::Text(a), Value::Text(b)) => return a.as_str() < b.as_str(),
_ => {}
}
}
false
}
FilterPredicate::ColumnLessThanOrEqual {
left_idx,
right_idx,
} => {
if let (Some(left), Some(right)) = (values.get(*left_idx), values.get(*right_idx)) {
match (left, right) {
(Value::Integer(a), Value::Integer(b)) => return a <= b,
(Value::Float(a), Value::Float(b)) => return a <= b,
(Value::Text(a), Value::Text(b)) => return a.as_str() <= b.as_str(),
_ => {}
}
}
false
}
}
}
}
impl IncrementalOperator for FilterOperator {
fn eval(
&mut self,
state: &mut EvalState,
_cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
let delta = match state {
EvalState::Init { deltas } => {
// Filter operators only use left_delta, right_delta must be empty
assert!(
deltas.right.is_empty(),
"FilterOperator expects right_delta to be empty"
);
std::mem::take(&mut deltas.left)
}
_ => unreachable!(
"FilterOperator doesn't execute the state machine. Should be in Init state"
),
};
let mut output_delta = Delta::new();
// Process the delta through the filter
for (row, weight) in delta.changes {
if let Some(tracker) = &self.tracker {
tracker.lock().unwrap().record_filter();
}
// Only pass through rows that satisfy the filter predicate
// For deletes (weight < 0), we only pass them if the row values
// would have passed the filter (meaning it was in the view)
if self.evaluate_predicate(&row.values) {
output_delta.changes.push((row, weight));
}
}
*state = EvalState::Done;
Ok(IOResult::Done(output_delta))
}
fn commit(
&mut self,
deltas: DeltaPair,
_cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
// Filter operator only uses left delta, right must be empty
assert!(
deltas.right.is_empty(),
"FilterOperator expects right delta to be empty in commit"
);
let mut output_delta = Delta::new();
// Commit the delta to our internal state
// Only pass through and track rows that satisfy the filter predicate
for (row, weight) in deltas.left.changes {
if let Some(tracker) = &self.tracker {
tracker.lock().unwrap().record_filter();
}
// Only track and output rows that pass the filter
// For deletes, this means the row was in the view (its values pass the filter)
// For inserts, this means the row should be in the view
if self.evaluate_predicate(&row.values) {
output_delta.changes.push((row, weight));
}
}
Ok(IOResult::Done(output_delta))
}
fn set_tracker(&mut self, tracker: Arc<Mutex<ComputationTracker>>) {
self.tracker = Some(tracker);
}
}

View File

@@ -0,0 +1,66 @@
// Input operator for DBSP-style incremental computation
// This operator serves as the entry point for data into the incremental computation pipeline
use crate::incremental::dbsp::{Delta, DeltaPair};
use crate::incremental::operator::{
ComputationTracker, DbspStateCursors, EvalState, IncrementalOperator,
};
use crate::types::IOResult;
use crate::Result;
use std::sync::{Arc, Mutex};
/// Input operator - source of data for the circuit
/// Represents base relations/tables that receive external updates
#[derive(Debug)]
pub struct InputOperator {
#[allow(dead_code)]
name: String,
}
impl InputOperator {
pub fn new(name: String) -> Self {
Self { name }
}
}
impl IncrementalOperator for InputOperator {
fn eval(
&mut self,
state: &mut EvalState,
_cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
match state {
EvalState::Init { deltas } => {
// Input operators only use left_delta, right_delta must be empty
assert!(
deltas.right.is_empty(),
"InputOperator expects right_delta to be empty"
);
let output = std::mem::take(&mut deltas.left);
*state = EvalState::Done;
Ok(IOResult::Done(output))
}
_ => unreachable!(
"InputOperator doesn't execute the state machine. Should be in Init state"
),
}
}
fn commit(
&mut self,
deltas: DeltaPair,
_cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
// Input operator only uses left delta, right must be empty
assert!(
deltas.right.is_empty(),
"InputOperator expects right delta to be empty in commit"
);
// Input operator passes through the delta unchanged during commit
Ok(IOResult::Done(deltas.left))
}
fn set_tracker(&mut self, _tracker: Arc<Mutex<ComputationTracker>>) {
// Input operator doesn't need tracking
}
}

View File

@@ -0,0 +1,787 @@
#![allow(dead_code)]
use crate::incremental::dbsp::{Delta, DeltaPair, HashableRow};
use crate::incremental::operator::{
generate_storage_id, ComputationTracker, DbspStateCursors, EvalState, IncrementalOperator,
};
use crate::incremental::persistence::WriteRow;
use crate::types::{IOResult, ImmutableRecord, SeekKey, SeekOp, SeekResult};
use crate::{return_and_restore_if_io, return_if_io, Result, Value};
use std::sync::{Arc, Mutex};
#[derive(Debug, Clone, PartialEq)]
pub enum JoinType {
Inner,
Left,
Right,
Full,
Cross,
}
// Helper function to read the next row from the BTree for joins
fn read_next_join_row(
storage_id: i64,
join_key: &HashableRow,
last_element_id: i64,
cursors: &mut DbspStateCursors,
) -> Result<IOResult<Option<(i64, HashableRow, isize)>>> {
// Build the index key: (storage_id, zset_id, element_id)
// zset_id is the hash of the join key
let zset_id = join_key.cached_hash() as i64;
let index_key_values = vec![
Value::Integer(storage_id),
Value::Integer(zset_id),
Value::Integer(last_element_id),
];
let index_record = ImmutableRecord::from_values(&index_key_values, index_key_values.len());
let seek_result = return_if_io!(cursors
.index_cursor
.seek(SeekKey::IndexKey(&index_record), SeekOp::GT));
if !matches!(seek_result, SeekResult::Found) {
return Ok(IOResult::Done(None));
}
// Check if we're still in the same (storage_id, zset_id) range
let current_record = return_if_io!(cursors.index_cursor.record());
// Extract all needed values from the record before dropping it
let (found_storage_id, found_zset_id, element_id) = if let Some(rec) = current_record {
let values = rec.get_values();
// Index has 4 values: storage_id, zset_id, element_id, rowid (appended by WriteRow)
if values.len() >= 3 {
let found_storage_id = match &values[0].to_owned() {
Value::Integer(id) => *id,
_ => return Ok(IOResult::Done(None)),
};
let found_zset_id = match &values[1].to_owned() {
Value::Integer(id) => *id,
_ => return Ok(IOResult::Done(None)),
};
let element_id = match &values[2].to_owned() {
Value::Integer(id) => *id,
_ => {
return Ok(IOResult::Done(None));
}
};
(found_storage_id, found_zset_id, element_id)
} else {
return Ok(IOResult::Done(None));
}
} else {
return Ok(IOResult::Done(None));
};
// Now we can safely check if we're in the right range
// If we've moved to a different storage_id or zset_id, we're done
if found_storage_id != storage_id || found_zset_id != zset_id {
return Ok(IOResult::Done(None));
}
// Now get the actual row from the table using the rowid from the index
let rowid = return_if_io!(cursors.index_cursor.rowid());
if let Some(rowid) = rowid {
return_if_io!(cursors
.table_cursor
.seek(SeekKey::TableRowId(rowid), SeekOp::GE { eq_only: true }));
let table_record = return_if_io!(cursors.table_cursor.record());
if let Some(rec) = table_record {
let table_values = rec.get_values();
// Table format: [storage_id, zset_id, element_id, value_blob, weight]
if table_values.len() >= 5 {
// Deserialize the row from the blob
let value_at_3 = table_values[3].to_owned();
let blob = match value_at_3 {
Value::Blob(ref b) => b,
_ => return Ok(IOResult::Done(None)),
};
// The blob contains the serialized HashableRow
// For now, let's deserialize it simply
let row = deserialize_hashable_row(blob)?;
let weight = match &table_values[4].to_owned() {
Value::Integer(w) => *w as isize,
_ => return Ok(IOResult::Done(None)),
};
return Ok(IOResult::Done(Some((element_id, row, weight))));
}
}
}
Ok(IOResult::Done(None))
}
// Join-specific eval states
#[derive(Debug)]
pub enum JoinEvalState {
ProcessDeltaJoin {
deltas: DeltaPair,
output: Delta,
},
ProcessLeftJoin {
deltas: DeltaPair,
output: Delta,
current_idx: usize,
last_row_scanned: i64,
},
ProcessRightJoin {
deltas: DeltaPair,
output: Delta,
current_idx: usize,
last_row_scanned: i64,
},
Done {
output: Delta,
},
}
impl JoinEvalState {
fn combine_rows(
left_row: &HashableRow,
left_weight: i64,
right_row: &HashableRow,
right_weight: i64,
output: &mut Delta,
) {
// Combine the rows
let mut combined_values = left_row.values.clone();
combined_values.extend(right_row.values.clone());
// Use hash of the combined values as rowid to ensure uniqueness
let temp_row = HashableRow::new(0, combined_values.clone());
let joined_rowid = temp_row.cached_hash() as i64;
let joined_row = HashableRow::new(joined_rowid, combined_values);
// Add to output with combined weight
let combined_weight = left_weight * right_weight;
output.changes.push((joined_row, combined_weight as isize));
}
fn process_join_state(
&mut self,
cursors: &mut DbspStateCursors,
left_key_indices: &[usize],
right_key_indices: &[usize],
left_storage_id: i64,
right_storage_id: i64,
) -> Result<IOResult<Delta>> {
loop {
match self {
JoinEvalState::ProcessDeltaJoin { deltas, output } => {
// Move to ProcessLeftJoin
*self = JoinEvalState::ProcessLeftJoin {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: 0,
last_row_scanned: i64::MIN,
};
}
JoinEvalState::ProcessLeftJoin {
deltas,
output,
current_idx,
last_row_scanned,
} => {
if *current_idx >= deltas.left.changes.len() {
*self = JoinEvalState::ProcessRightJoin {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: 0,
last_row_scanned: i64::MIN,
};
} else {
let (left_row, left_weight) = &deltas.left.changes[*current_idx];
// Extract join key using provided indices
let key_values: Vec<Value> = left_key_indices
.iter()
.map(|&idx| left_row.values.get(idx).cloned().unwrap_or(Value::Null))
.collect();
let left_key = HashableRow::new(0, key_values);
let next_row = return_if_io!(read_next_join_row(
right_storage_id,
&left_key,
*last_row_scanned,
cursors
));
match next_row {
Some((element_id, right_row, right_weight)) => {
Self::combine_rows(
left_row,
(*left_weight) as i64,
&right_row,
right_weight as i64,
output,
);
// Continue scanning with this left row
*self = JoinEvalState::ProcessLeftJoin {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: *current_idx,
last_row_scanned: element_id,
};
}
None => {
// No more matches for this left row, move to next
*self = JoinEvalState::ProcessLeftJoin {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: *current_idx + 1,
last_row_scanned: i64::MIN,
};
}
}
}
}
JoinEvalState::ProcessRightJoin {
deltas,
output,
current_idx,
last_row_scanned,
} => {
if *current_idx >= deltas.right.changes.len() {
*self = JoinEvalState::Done {
output: std::mem::take(output),
};
} else {
let (right_row, right_weight) = &deltas.right.changes[*current_idx];
// Extract join key using provided indices
let key_values: Vec<Value> = right_key_indices
.iter()
.map(|&idx| right_row.values.get(idx).cloned().unwrap_or(Value::Null))
.collect();
let right_key = HashableRow::new(0, key_values);
let next_row = return_if_io!(read_next_join_row(
left_storage_id,
&right_key,
*last_row_scanned,
cursors
));
match next_row {
Some((element_id, left_row, left_weight)) => {
Self::combine_rows(
&left_row,
left_weight as i64,
right_row,
(*right_weight) as i64,
output,
);
// Continue scanning with this right row
*self = JoinEvalState::ProcessRightJoin {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: *current_idx,
last_row_scanned: element_id,
};
}
None => {
// No more matches for this right row, move to next
*self = JoinEvalState::ProcessRightJoin {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: *current_idx + 1,
last_row_scanned: i64::MIN,
};
}
}
}
}
JoinEvalState::Done { output } => {
return Ok(IOResult::Done(std::mem::take(output)));
}
}
}
}
}
#[derive(Debug)]
enum JoinCommitState {
Idle,
Eval {
eval_state: EvalState,
},
CommitLeftDelta {
deltas: DeltaPair,
output: Delta,
current_idx: usize,
write_row: WriteRow,
},
CommitRightDelta {
deltas: DeltaPair,
output: Delta,
current_idx: usize,
write_row: WriteRow,
},
Invalid,
}
/// Join operator - performs incremental join between two relations
/// Implements the DBSP formula: δ(R ⋈ S) = (δR ⋈ S) (R ⋈ δS) (δR ⋈ δS)
#[derive(Debug)]
pub struct JoinOperator {
/// Unique operator ID for indexing in persistent storage
operator_id: usize,
/// Type of join to perform
join_type: JoinType,
/// Column indices for extracting join keys from left input
left_key_indices: Vec<usize>,
/// Column indices for extracting join keys from right input
right_key_indices: Vec<usize>,
/// Column names from left input
left_columns: Vec<String>,
/// Column names from right input
right_columns: Vec<String>,
/// Tracker for computation statistics
tracker: Option<Arc<Mutex<ComputationTracker>>>,
commit_state: JoinCommitState,
}
impl JoinOperator {
pub fn new(
operator_id: usize,
join_type: JoinType,
left_key_indices: Vec<usize>,
right_key_indices: Vec<usize>,
left_columns: Vec<String>,
right_columns: Vec<String>,
) -> Result<Self> {
// Check for unsupported join types
match join_type {
JoinType::Left => {
return Err(crate::LimboError::ParseError(
"LEFT OUTER JOIN is not yet supported in incremental views".to_string(),
))
}
JoinType::Right => {
return Err(crate::LimboError::ParseError(
"RIGHT OUTER JOIN is not yet supported in incremental views".to_string(),
))
}
JoinType::Full => {
return Err(crate::LimboError::ParseError(
"FULL OUTER JOIN is not yet supported in incremental views".to_string(),
))
}
JoinType::Cross => {
return Err(crate::LimboError::ParseError(
"CROSS JOIN is not yet supported in incremental views".to_string(),
))
}
JoinType::Inner => {} // Inner join is supported
}
Ok(Self {
operator_id,
join_type,
left_key_indices,
right_key_indices,
left_columns,
right_columns,
tracker: None,
commit_state: JoinCommitState::Idle,
})
}
/// Extract join key from row values using the specified indices
fn extract_join_key(&self, values: &[Value], indices: &[usize]) -> HashableRow {
let key_values: Vec<Value> = indices
.iter()
.map(|&idx| values.get(idx).cloned().unwrap_or(Value::Null))
.collect();
// Use 0 as a dummy rowid for join keys. They don't come from a table,
// so they don't need a rowid. Their key will be the hash of the row values.
HashableRow::new(0, key_values)
}
/// Generate storage ID for left table
fn left_storage_id(&self) -> i64 {
// Use column_index=0 for left side
generate_storage_id(self.operator_id, 0, 0)
}
/// Generate storage ID for right table
fn right_storage_id(&self) -> i64 {
// Use column_index=1 for right side
generate_storage_id(self.operator_id, 1, 0)
}
/// SQL-compliant comparison for join keys
/// Returns true if keys match according to SQL semantics (NULL != NULL)
fn sql_keys_equal(left_key: &HashableRow, right_key: &HashableRow) -> bool {
if left_key.values.len() != right_key.values.len() {
return false;
}
for (left_val, right_val) in left_key.values.iter().zip(right_key.values.iter()) {
// In SQL, NULL never equals NULL
if matches!(left_val, Value::Null) || matches!(right_val, Value::Null) {
return false;
}
// For non-NULL values, use regular comparison
if left_val != right_val {
return false;
}
}
true
}
fn process_join_state(
&mut self,
state: &mut EvalState,
cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
// Get the join state out of the enum
match state {
EvalState::Join(js) => js.process_join_state(
cursors,
&self.left_key_indices,
&self.right_key_indices,
self.left_storage_id(),
self.right_storage_id(),
),
_ => panic!("process_join_state called with non-join state"),
}
}
fn eval_internal(
&mut self,
state: &mut EvalState,
cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
loop {
let loop_state = std::mem::replace(state, EvalState::Uninitialized);
match loop_state {
EvalState::Uninitialized => {
panic!("Cannot eval JoinOperator with Uninitialized state");
}
EvalState::Init { deltas } => {
let mut output = Delta::new();
// Component 3: δR ⋈ δS (left delta join right delta)
for (left_row, left_weight) in &deltas.left.changes {
let left_key =
self.extract_join_key(&left_row.values, &self.left_key_indices);
for (right_row, right_weight) in &deltas.right.changes {
let right_key =
self.extract_join_key(&right_row.values, &self.right_key_indices);
if Self::sql_keys_equal(&left_key, &right_key) {
if let Some(tracker) = &self.tracker {
tracker.lock().unwrap().record_join_lookup();
}
// Combine the rows
let mut combined_values = left_row.values.clone();
combined_values.extend(right_row.values.clone());
// Create the joined row with a unique rowid
// Use hash of the combined values to ensure uniqueness
let temp_row = HashableRow::new(0, combined_values.clone());
let joined_rowid = temp_row.cached_hash() as i64;
let joined_row =
HashableRow::new(joined_rowid, combined_values.clone());
// Add to output with combined weight
let combined_weight = left_weight * right_weight;
output.changes.push((joined_row, combined_weight));
}
}
}
*state = EvalState::Join(Box::new(JoinEvalState::ProcessDeltaJoin {
deltas,
output,
}));
}
EvalState::Join(join_state) => {
*state = EvalState::Join(join_state);
let output = return_if_io!(self.process_join_state(state, cursors));
return Ok(IOResult::Done(output));
}
EvalState::Done => {
return Ok(IOResult::Done(Delta::new()));
}
EvalState::Aggregate(_) => {
panic!("Aggregate state should not appear in join operator");
}
}
}
}
}
// Helper to deserialize a HashableRow from a blob
fn deserialize_hashable_row(blob: &[u8]) -> Result<HashableRow> {
// Simple deserialization - this needs to match how we serialize in commit
// Format: [rowid:8 bytes][num_values:4 bytes][values...]
if blob.len() < 12 {
return Err(crate::LimboError::InternalError(
"Invalid blob size".to_string(),
));
}
let rowid = i64::from_le_bytes(blob[0..8].try_into().unwrap());
let num_values = u32::from_le_bytes(blob[8..12].try_into().unwrap()) as usize;
let mut values = Vec::new();
let mut offset = 12;
for _ in 0..num_values {
if offset >= blob.len() {
break;
}
let type_tag = blob[offset];
offset += 1;
match type_tag {
0 => values.push(Value::Null),
1 => {
if offset + 8 <= blob.len() {
let i = i64::from_le_bytes(blob[offset..offset + 8].try_into().unwrap());
values.push(Value::Integer(i));
offset += 8;
}
}
2 => {
if offset + 8 <= blob.len() {
let f = f64::from_le_bytes(blob[offset..offset + 8].try_into().unwrap());
values.push(Value::Float(f));
offset += 8;
}
}
3 => {
if offset + 4 <= blob.len() {
let len =
u32::from_le_bytes(blob[offset..offset + 4].try_into().unwrap()) as usize;
offset += 4;
if offset + len < blob.len() {
let text_bytes = blob[offset..offset + len].to_vec();
offset += len;
let subtype = match blob[offset] {
0 => crate::types::TextSubtype::Text,
1 => crate::types::TextSubtype::Json,
_ => crate::types::TextSubtype::Text,
};
offset += 1;
values.push(Value::Text(crate::types::Text {
value: text_bytes,
subtype,
}));
}
}
}
4 => {
if offset + 4 <= blob.len() {
let len =
u32::from_le_bytes(blob[offset..offset + 4].try_into().unwrap()) as usize;
offset += 4;
if offset + len <= blob.len() {
let blob_data = blob[offset..offset + len].to_vec();
values.push(Value::Blob(blob_data));
offset += len;
}
}
}
_ => break, // Unknown type tag
}
}
Ok(HashableRow::new(rowid, values))
}
// Helper to serialize a HashableRow to a blob
fn serialize_hashable_row(row: &HashableRow) -> Vec<u8> {
let mut blob = Vec::new();
// Write rowid
blob.extend_from_slice(&row.rowid.to_le_bytes());
// Write number of values
blob.extend_from_slice(&(row.values.len() as u32).to_le_bytes());
// Write each value directly with type tags (like AggregateState does)
for value in &row.values {
match value {
Value::Null => blob.push(0u8),
Value::Integer(i) => {
blob.push(1u8);
blob.extend_from_slice(&i.to_le_bytes());
}
Value::Float(f) => {
blob.push(2u8);
blob.extend_from_slice(&f.to_le_bytes());
}
Value::Text(s) => {
blob.push(3u8);
let bytes = &s.value;
blob.extend_from_slice(&(bytes.len() as u32).to_le_bytes());
blob.extend_from_slice(bytes);
blob.push(s.subtype as u8);
}
Value::Blob(b) => {
blob.push(4u8);
blob.extend_from_slice(&(b.len() as u32).to_le_bytes());
blob.extend_from_slice(b);
}
}
}
blob
}
impl IncrementalOperator for JoinOperator {
fn eval(
&mut self,
state: &mut EvalState,
cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
let delta = return_if_io!(self.eval_internal(state, cursors));
Ok(IOResult::Done(delta))
}
fn commit(
&mut self,
deltas: DeltaPair,
cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
loop {
let mut state = std::mem::replace(&mut self.commit_state, JoinCommitState::Invalid);
match &mut state {
JoinCommitState::Idle => {
self.commit_state = JoinCommitState::Eval {
eval_state: deltas.clone().into(),
}
}
JoinCommitState::Eval { ref mut eval_state } => {
let output = return_and_restore_if_io!(
&mut self.commit_state,
state,
self.eval(eval_state, cursors)
);
self.commit_state = JoinCommitState::CommitLeftDelta {
deltas: deltas.clone(),
output,
current_idx: 0,
write_row: WriteRow::new(),
};
}
JoinCommitState::CommitLeftDelta {
deltas,
output,
current_idx,
ref mut write_row,
} => {
if *current_idx >= deltas.left.changes.len() {
self.commit_state = JoinCommitState::CommitRightDelta {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: 0,
write_row: WriteRow::new(),
};
continue;
}
let (row, weight) = &deltas.left.changes[*current_idx];
// Extract join key from the left row
let join_key = self.extract_join_key(&row.values, &self.left_key_indices);
// The index key: (storage_id, zset_id, element_id)
// zset_id is the hash of the join key, element_id is hash of the row
let storage_id = self.left_storage_id();
let zset_id = join_key.cached_hash() as i64;
let element_id = row.cached_hash() as i64;
let index_key = vec![
Value::Integer(storage_id),
Value::Integer(zset_id),
Value::Integer(element_id),
];
// The record values: we'll store the serialized row as a blob
let row_blob = serialize_hashable_row(row);
let record_values = vec![
Value::Integer(self.left_storage_id()),
Value::Integer(join_key.cached_hash() as i64),
Value::Integer(row.cached_hash() as i64),
Value::Blob(row_blob),
];
// Use return_and_restore_if_io to handle I/O properly
return_and_restore_if_io!(
&mut self.commit_state,
state,
write_row.write_row(cursors, index_key, record_values, *weight)
);
self.commit_state = JoinCommitState::CommitLeftDelta {
deltas: deltas.clone(),
output: output.clone(),
current_idx: *current_idx + 1,
write_row: WriteRow::new(),
};
}
JoinCommitState::CommitRightDelta {
deltas,
output,
current_idx,
ref mut write_row,
} => {
if *current_idx >= deltas.right.changes.len() {
// Reset to Idle state for next commit
self.commit_state = JoinCommitState::Idle;
return Ok(IOResult::Done(output.clone()));
}
let (row, weight) = &deltas.right.changes[*current_idx];
// Extract join key from the right row
let join_key = self.extract_join_key(&row.values, &self.right_key_indices);
// The index key: (storage_id, zset_id, element_id)
let index_key = vec![
Value::Integer(self.right_storage_id()),
Value::Integer(join_key.cached_hash() as i64),
Value::Integer(row.cached_hash() as i64),
];
// The record values: we'll store the serialized row as a blob
let row_blob = serialize_hashable_row(row);
let record_values = vec![
Value::Integer(self.right_storage_id()),
Value::Integer(join_key.cached_hash() as i64),
Value::Integer(row.cached_hash() as i64),
Value::Blob(row_blob),
];
// Use return_and_restore_if_io to handle I/O properly
return_and_restore_if_io!(
&mut self.commit_state,
state,
write_row.write_row(cursors, index_key, record_values, *weight)
);
self.commit_state = JoinCommitState::CommitRightDelta {
deltas: std::mem::take(deltas),
output: std::mem::take(output),
current_idx: *current_idx + 1,
write_row: WriteRow::new(),
};
}
JoinCommitState::Invalid => {
panic!("Invalid join commit state");
}
}
}
}
fn set_tracker(&mut self, tracker: Arc<Mutex<ComputationTracker>>) {
self.tracker = Some(tracker);
}
}

View File

@@ -1,7 +1,12 @@
pub mod aggregate_operator;
pub mod compiler;
pub mod cursor;
pub mod dbsp;
pub mod expr_compiler;
pub mod filter_operator;
pub mod input_operator;
pub mod join_operator;
pub mod operator;
pub mod persistence;
pub mod project_operator;
pub mod view;

File diff suppressed because it is too large Load Diff

View File

@@ -1,12 +1,7 @@
use crate::incremental::dbsp::HashableRow;
use crate::incremental::operator::{
generate_storage_id, AggColumnInfo, AggregateFunction, AggregateOperator, AggregateState,
DbspStateCursors, MinMaxDeltas, AGG_TYPE_MINMAX,
};
use crate::incremental::operator::{AggregateFunction, AggregateState, DbspStateCursors};
use crate::storage::btree::{BTreeCursor, BTreeKey};
use crate::types::{IOResult, ImmutableRecord, RefValue, SeekKey, SeekOp, SeekResult};
use crate::types::{IOResult, ImmutableRecord, SeekKey, SeekOp, SeekResult};
use crate::{return_if_io, LimboError, Result, Value};
use std::collections::{HashMap, HashSet};
#[derive(Debug, Default)]
pub enum ReadRecord {
@@ -290,672 +285,3 @@ impl WriteRow {
}
}
}
/// State machine for recomputing MIN/MAX values after deletion
#[derive(Debug)]
pub enum RecomputeMinMax {
ProcessElements {
/// Current column being processed
current_column_idx: usize,
/// Columns to process (combined MIN and MAX)
columns_to_process: Vec<(String, String, bool)>, // (group_key, column_name, is_min)
/// MIN/MAX deltas for checking values and weights
min_max_deltas: MinMaxDeltas,
},
Scan {
/// Columns still to process
columns_to_process: Vec<(String, String, bool)>,
/// Current index in columns_to_process (will resume from here)
current_column_idx: usize,
/// MIN/MAX deltas for checking values and weights
min_max_deltas: MinMaxDeltas,
/// Current group key being processed
group_key: String,
/// Current column name being processed
column_name: String,
/// Whether we're looking for MIN (true) or MAX (false)
is_min: bool,
/// The scan state machine for finding the new MIN/MAX
scan_state: Box<ScanState>,
},
Done,
}
impl RecomputeMinMax {
pub fn new(
min_max_deltas: MinMaxDeltas,
existing_groups: &HashMap<String, AggregateState>,
operator: &AggregateOperator,
) -> Self {
let mut groups_to_check: HashSet<(String, String, bool)> = HashSet::new();
// Remember the min_max_deltas are essentially just the only column that is affected by
// this min/max, in delta (actually ZSet - consolidated delta) format. This makes it easier
// for us to consume it in here.
//
// The most challenging case is the case where there is a retraction, since we need to go
// back to the index.
for (group_key_str, values) in &min_max_deltas {
for ((col_name, hashable_row), weight) in values {
let col_info = operator.column_min_max.get(col_name);
let value = &hashable_row.values[0];
if *weight < 0 {
// Deletion detected - check if it's the current MIN/MAX
if let Some(state) = existing_groups.get(group_key_str) {
// Check for MIN
if let Some(current_min) = state.mins.get(col_name) {
if current_min == value {
groups_to_check.insert((
group_key_str.clone(),
col_name.clone(),
true,
));
}
}
// Check for MAX
if let Some(current_max) = state.maxs.get(col_name) {
if current_max == value {
groups_to_check.insert((
group_key_str.clone(),
col_name.clone(),
false,
));
}
}
}
} else if *weight > 0 {
// If it is not found in the existing groups, then we only need to care
// about this if this is a new record being inserted
if let Some(info) = col_info {
if info.has_min {
groups_to_check.insert((group_key_str.clone(), col_name.clone(), true));
}
if info.has_max {
groups_to_check.insert((
group_key_str.clone(),
col_name.clone(),
false,
));
}
}
}
}
}
if groups_to_check.is_empty() {
// No recomputation or initialization needed
Self::Done
} else {
// Convert HashSet to Vec for indexed processing
let groups_to_check_vec: Vec<_> = groups_to_check.into_iter().collect();
Self::ProcessElements {
current_column_idx: 0,
columns_to_process: groups_to_check_vec,
min_max_deltas,
}
}
}
pub fn process(
&mut self,
existing_groups: &mut HashMap<String, AggregateState>,
operator: &AggregateOperator,
cursors: &mut DbspStateCursors,
) -> Result<IOResult<()>> {
loop {
match self {
RecomputeMinMax::ProcessElements {
current_column_idx,
columns_to_process,
min_max_deltas,
} => {
if *current_column_idx >= columns_to_process.len() {
*self = RecomputeMinMax::Done;
return Ok(IOResult::Done(()));
}
let (group_key, column_name, is_min) =
columns_to_process[*current_column_idx].clone();
// Get column index from pre-computed info
let column_index = operator
.column_min_max
.get(&column_name)
.map(|info| info.index)
.unwrap(); // Should always exist since we're processing known columns
// Get current value from existing state
let current_value = existing_groups.get(&group_key).and_then(|state| {
if is_min {
state.mins.get(&column_name).cloned()
} else {
state.maxs.get(&column_name).cloned()
}
});
// Create storage keys for index lookup
let storage_id =
generate_storage_id(operator.operator_id, column_index, AGG_TYPE_MINMAX);
let zset_id = operator.generate_group_rowid(&group_key);
// Get the values for this group from min_max_deltas
let group_values = min_max_deltas.get(&group_key).cloned().unwrap_or_default();
let columns_to_process = std::mem::take(columns_to_process);
let min_max_deltas = std::mem::take(min_max_deltas);
let scan_state = if is_min {
Box::new(ScanState::new_for_min(
current_value,
group_key.clone(),
column_name.clone(),
storage_id,
zset_id,
group_values,
))
} else {
Box::new(ScanState::new_for_max(
current_value,
group_key.clone(),
column_name.clone(),
storage_id,
zset_id,
group_values,
))
};
*self = RecomputeMinMax::Scan {
columns_to_process,
current_column_idx: *current_column_idx,
min_max_deltas,
group_key,
column_name,
is_min,
scan_state,
};
}
RecomputeMinMax::Scan {
columns_to_process,
current_column_idx,
min_max_deltas,
group_key,
column_name,
is_min,
scan_state,
} => {
// Find new value using the scan state machine
let new_value = return_if_io!(scan_state.find_new_value(cursors));
// Update the state with new value (create if doesn't exist)
let state = existing_groups.entry(group_key.clone()).or_default();
if *is_min {
if let Some(min_val) = new_value {
state.mins.insert(column_name.clone(), min_val);
} else {
state.mins.remove(column_name);
}
} else if let Some(max_val) = new_value {
state.maxs.insert(column_name.clone(), max_val);
} else {
state.maxs.remove(column_name);
}
// Move to next column
let min_max_deltas = std::mem::take(min_max_deltas);
let columns_to_process = std::mem::take(columns_to_process);
*self = RecomputeMinMax::ProcessElements {
current_column_idx: *current_column_idx + 1,
columns_to_process,
min_max_deltas,
};
}
RecomputeMinMax::Done => {
return Ok(IOResult::Done(()));
}
}
}
}
}
/// State machine for scanning through the index to find new MIN/MAX values
#[derive(Debug)]
pub enum ScanState {
CheckCandidate {
/// Current candidate value for MIN/MAX
candidate: Option<Value>,
/// Group key being processed
group_key: String,
/// Column name being processed
column_name: String,
/// Storage ID for the index seek
storage_id: i64,
/// ZSet ID for the group
zset_id: i64,
/// Group values from MinMaxDeltas: (column_name, HashableRow) -> weight
group_values: HashMap<(String, HashableRow), isize>,
/// Whether we're looking for MIN (true) or MAX (false)
is_min: bool,
},
FetchNextCandidate {
/// Current candidate to seek past
current_candidate: Value,
/// Group key being processed
group_key: String,
/// Column name being processed
column_name: String,
/// Storage ID for the index seek
storage_id: i64,
/// ZSet ID for the group
zset_id: i64,
/// Group values from MinMaxDeltas: (column_name, HashableRow) -> weight
group_values: HashMap<(String, HashableRow), isize>,
/// Whether we're looking for MIN (true) or MAX (false)
is_min: bool,
},
Done {
/// The final MIN/MAX value found
result: Option<Value>,
},
}
impl ScanState {
pub fn new_for_min(
current_min: Option<Value>,
group_key: String,
column_name: String,
storage_id: i64,
zset_id: i64,
group_values: HashMap<(String, HashableRow), isize>,
) -> Self {
Self::CheckCandidate {
candidate: current_min,
group_key,
column_name,
storage_id,
zset_id,
group_values,
is_min: true,
}
}
// Extract a new candidate from the index. It is possible that, when searching,
// we end up going into a different operator altogether. That means we have
// exhausted this operator (or group) entirely, and no good candidate was found
fn extract_new_candidate(
cursors: &mut DbspStateCursors,
index_record: &ImmutableRecord,
seek_op: SeekOp,
storage_id: i64,
zset_id: i64,
) -> Result<IOResult<Option<Value>>> {
let seek_result = return_if_io!(cursors
.index_cursor
.seek(SeekKey::IndexKey(index_record), seek_op));
if !matches!(seek_result, SeekResult::Found) {
return Ok(IOResult::Done(None));
}
let record = return_if_io!(cursors.index_cursor.record()).ok_or_else(|| {
LimboError::InternalError(
"Record found on the cursor, but could not be read".to_string(),
)
})?;
let values = record.get_values();
if values.len() < 3 {
return Ok(IOResult::Done(None));
}
let Some(rec_storage_id) = values.first() else {
return Ok(IOResult::Done(None));
};
let Some(rec_zset_id) = values.get(1) else {
return Ok(IOResult::Done(None));
};
// Check if we're still in the same group
if let (RefValue::Integer(rec_sid), RefValue::Integer(rec_zid)) =
(rec_storage_id, rec_zset_id)
{
if *rec_sid != storage_id || *rec_zid != zset_id {
return Ok(IOResult::Done(None));
}
} else {
return Ok(IOResult::Done(None));
}
// Get the value (3rd element)
Ok(IOResult::Done(values.get(2).map(|v| v.to_owned())))
}
pub fn new_for_max(
current_max: Option<Value>,
group_key: String,
column_name: String,
storage_id: i64,
zset_id: i64,
group_values: HashMap<(String, HashableRow), isize>,
) -> Self {
Self::CheckCandidate {
candidate: current_max,
group_key,
column_name,
storage_id,
zset_id,
group_values,
is_min: false,
}
}
pub fn find_new_value(
&mut self,
cursors: &mut DbspStateCursors,
) -> Result<IOResult<Option<Value>>> {
loop {
match self {
ScanState::CheckCandidate {
candidate,
group_key,
column_name,
storage_id,
zset_id,
group_values,
is_min,
} => {
// First, check if we have a candidate
if let Some(cand_val) = candidate {
// Check if the candidate is retracted (weight <= 0)
// Create a HashableRow to look up the weight
let hashable_cand = HashableRow::new(0, vec![cand_val.clone()]);
let key = (column_name.clone(), hashable_cand);
let is_retracted =
group_values.get(&key).is_some_and(|weight| *weight <= 0);
if is_retracted {
// Candidate is retracted, need to fetch next from index
*self = ScanState::FetchNextCandidate {
current_candidate: cand_val.clone(),
group_key: std::mem::take(group_key),
column_name: std::mem::take(column_name),
storage_id: *storage_id,
zset_id: *zset_id,
group_values: std::mem::take(group_values),
is_min: *is_min,
};
continue;
}
}
// Candidate is valid or we have no candidate
// Now find the best value from insertions in group_values
let mut best_from_zset = None;
for ((col, hashable_val), weight) in group_values.iter() {
if col == column_name && *weight > 0 {
let value = &hashable_val.values[0];
// Skip NULL values - they don't participate in MIN/MAX
if value == &Value::Null {
continue;
}
// This is an insertion for our column
if let Some(ref current_best) = best_from_zset {
if *is_min {
if value.cmp(current_best) == std::cmp::Ordering::Less {
best_from_zset = Some(value.clone());
}
} else if value.cmp(current_best) == std::cmp::Ordering::Greater {
best_from_zset = Some(value.clone());
}
} else {
best_from_zset = Some(value.clone());
}
}
}
// Compare candidate with best from ZSet, filtering out NULLs
let result = match (&candidate, &best_from_zset) {
(Some(cand), Some(zset_val)) if cand != &Value::Null => {
if *is_min {
if zset_val.cmp(cand) == std::cmp::Ordering::Less {
Some(zset_val.clone())
} else {
Some(cand.clone())
}
} else if zset_val.cmp(cand) == std::cmp::Ordering::Greater {
Some(zset_val.clone())
} else {
Some(cand.clone())
}
}
(Some(cand), None) if cand != &Value::Null => Some(cand.clone()),
(None, Some(zset_val)) => Some(zset_val.clone()),
(Some(cand), Some(_)) if cand == &Value::Null => best_from_zset,
_ => None,
};
*self = ScanState::Done { result };
}
ScanState::FetchNextCandidate {
current_candidate,
group_key,
column_name,
storage_id,
zset_id,
group_values,
is_min,
} => {
// Seek to the next value in the index
let index_key = vec![
Value::Integer(*storage_id),
Value::Integer(*zset_id),
current_candidate.clone(),
];
let index_record = ImmutableRecord::from_values(&index_key, index_key.len());
let seek_op = if *is_min {
SeekOp::GT // For MIN, seek greater than current
} else {
SeekOp::LT // For MAX, seek less than current
};
let new_candidate = return_if_io!(Self::extract_new_candidate(
cursors,
&index_record,
seek_op,
*storage_id,
*zset_id
));
*self = ScanState::CheckCandidate {
candidate: new_candidate,
group_key: std::mem::take(group_key),
column_name: std::mem::take(column_name),
storage_id: *storage_id,
zset_id: *zset_id,
group_values: std::mem::take(group_values),
is_min: *is_min,
};
}
ScanState::Done { result } => {
return Ok(IOResult::Done(result.clone()));
}
}
}
}
}
/// State machine for persisting Min/Max values to storage
#[derive(Debug)]
pub enum MinMaxPersistState {
Init {
min_max_deltas: MinMaxDeltas,
group_keys: Vec<String>,
},
ProcessGroup {
min_max_deltas: MinMaxDeltas,
group_keys: Vec<String>,
group_idx: usize,
value_idx: usize,
},
WriteValue {
min_max_deltas: MinMaxDeltas,
group_keys: Vec<String>,
group_idx: usize,
value_idx: usize,
value: Value,
column_name: String,
weight: isize,
write_row: WriteRow,
},
Done,
}
impl MinMaxPersistState {
pub fn new(min_max_deltas: MinMaxDeltas) -> Self {
let group_keys: Vec<String> = min_max_deltas.keys().cloned().collect();
Self::Init {
min_max_deltas,
group_keys,
}
}
pub fn persist_min_max(
&mut self,
operator_id: usize,
column_min_max: &HashMap<String, AggColumnInfo>,
cursors: &mut DbspStateCursors,
generate_group_rowid: impl Fn(&str) -> i64,
) -> Result<IOResult<()>> {
loop {
match self {
MinMaxPersistState::Init {
min_max_deltas,
group_keys,
} => {
let min_max_deltas = std::mem::take(min_max_deltas);
let group_keys = std::mem::take(group_keys);
*self = MinMaxPersistState::ProcessGroup {
min_max_deltas,
group_keys,
group_idx: 0,
value_idx: 0,
};
}
MinMaxPersistState::ProcessGroup {
min_max_deltas,
group_keys,
group_idx,
value_idx,
} => {
// Check if we're past all groups
if *group_idx >= group_keys.len() {
*self = MinMaxPersistState::Done;
continue;
}
let group_key_str = &group_keys[*group_idx];
let values = &min_max_deltas[group_key_str]; // This should always exist
// Convert HashMap to Vec for indexed access
let values_vec: Vec<_> = values.iter().collect();
// Check if we have more values in current group
if *value_idx >= values_vec.len() {
*group_idx += 1;
*value_idx = 0;
// Continue to check if we're past all groups now
continue;
}
// Process current value and extract what we need before taking ownership
let ((column_name, hashable_row), weight) = values_vec[*value_idx];
let column_name = column_name.clone();
let value = hashable_row.values[0].clone(); // Extract the Value from HashableRow
let weight = *weight;
let min_max_deltas = std::mem::take(min_max_deltas);
let group_keys = std::mem::take(group_keys);
*self = MinMaxPersistState::WriteValue {
min_max_deltas,
group_keys,
group_idx: *group_idx,
value_idx: *value_idx,
column_name,
value,
weight,
write_row: WriteRow::new(),
};
}
MinMaxPersistState::WriteValue {
min_max_deltas,
group_keys,
group_idx,
value_idx,
value,
column_name,
weight,
write_row,
} => {
// Should have exited in the previous state
assert!(*group_idx < group_keys.len());
let group_key_str = &group_keys[*group_idx];
// Get the column index from the pre-computed map
let column_info = column_min_max
.get(&*column_name)
.expect("Column should exist in column_min_max map");
let column_index = column_info.index;
// Build the key components for MinMax storage using new encoding
let storage_id =
generate_storage_id(operator_id, column_index, AGG_TYPE_MINMAX);
let zset_id = generate_group_rowid(group_key_str);
// element_id is the actual value for Min/Max
let element_id_val = value.clone();
// Create index key
let index_key = vec![
Value::Integer(storage_id),
Value::Integer(zset_id),
element_id_val.clone(),
];
// Record values (operator_id, zset_id, element_id, unused_placeholder)
// For MIN/MAX, the element_id IS the value, so we use NULL for the 4th column
let record_values = vec![
Value::Integer(storage_id),
Value::Integer(zset_id),
element_id_val.clone(),
Value::Null, // Placeholder - not used for MIN/MAX
];
return_if_io!(write_row.write_row(
cursors,
index_key.clone(),
record_values,
*weight
));
// Move to next value
let min_max_deltas = std::mem::take(min_max_deltas);
let group_keys = std::mem::take(group_keys);
*self = MinMaxPersistState::ProcessGroup {
min_max_deltas,
group_keys,
group_idx: *group_idx,
value_idx: *value_idx + 1,
};
}
MinMaxPersistState::Done => {
return Ok(IOResult::Done(()));
}
}
}
}
}

View File

@@ -0,0 +1,168 @@
// Project operator for DBSP-style incremental computation
// This operator projects/transforms columns in a relational stream
use crate::incremental::dbsp::{Delta, DeltaPair, HashableRow};
use crate::incremental::expr_compiler::CompiledExpression;
use crate::incremental::operator::{
ComputationTracker, DbspStateCursors, EvalState, IncrementalOperator,
};
use crate::types::IOResult;
use crate::{Connection, Database, Result, Value};
use std::sync::{Arc, Mutex};
#[derive(Debug, Clone)]
pub struct ProjectColumn {
/// Compiled expression (handles both trivial columns and complex expressions)
pub compiled: CompiledExpression,
}
/// Project operator - selects/transforms columns
#[derive(Clone)]
pub struct ProjectOperator {
columns: Vec<ProjectColumn>,
input_column_names: Vec<String>,
output_column_names: Vec<String>,
tracker: Option<Arc<Mutex<ComputationTracker>>>,
// Internal in-memory connection for expression evaluation
// Programs are very dependent on having a connection, so give it one.
//
// We could in theory pass the current connection, but there are a host of problems with that.
// For example: during a write transaction, where views are usually updated, we have autocommit
// on. When the program we are executing calls Halt, it will try to commit the current
// transaction, which is absolutely incorrect.
//
// There are other ways to solve this, but a read-only connection to an empty in-memory
// database gives us the closest environment we need to execute expressions.
internal_conn: Arc<Connection>,
}
impl std::fmt::Debug for ProjectOperator {
fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
f.debug_struct("ProjectOperator")
.field("columns", &self.columns)
.field("input_column_names", &self.input_column_names)
.field("output_column_names", &self.output_column_names)
.finish()
}
}
impl ProjectOperator {
/// Create a ProjectOperator from pre-compiled expressions
pub fn from_compiled(
compiled_exprs: Vec<CompiledExpression>,
aliases: Vec<Option<String>>,
input_column_names: Vec<String>,
output_column_names: Vec<String>,
) -> crate::Result<Self> {
// Set up internal connection for expression evaluation
let io = Arc::new(crate::MemoryIO::new());
let db = Database::open_file(
io, ":memory:", false, // no MVCC needed for expression evaluation
false, // no indexes needed
)?;
let internal_conn = db.connect()?;
// Set to read-only mode and disable auto-commit since we're only evaluating expressions
internal_conn.query_only.set(true);
internal_conn.auto_commit.set(false);
// Create ProjectColumn structs from compiled expressions
let columns: Vec<ProjectColumn> = compiled_exprs
.into_iter()
.zip(aliases)
.map(|(compiled, _alias)| ProjectColumn { compiled })
.collect();
Ok(Self {
columns,
input_column_names,
output_column_names,
tracker: None,
internal_conn,
})
}
fn project_values(&self, values: &[Value]) -> Vec<Value> {
let mut output = Vec::new();
for col in &self.columns {
// Use the internal connection's pager for expression evaluation
let internal_pager = self.internal_conn.pager.borrow().clone();
// Execute the compiled expression (handles both columns and complex expressions)
let result = col
.compiled
.execute(values, internal_pager)
.expect("Failed to execute compiled expression for the Project operator");
output.push(result);
}
output
}
}
impl IncrementalOperator for ProjectOperator {
fn eval(
&mut self,
state: &mut EvalState,
_cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
let delta = match state {
EvalState::Init { deltas } => {
// Project operators only use left_delta, right_delta must be empty
assert!(
deltas.right.is_empty(),
"ProjectOperator expects right_delta to be empty"
);
std::mem::take(&mut deltas.left)
}
_ => unreachable!(
"ProjectOperator doesn't execute the state machine. Should be in Init state"
),
};
let mut output_delta = Delta::new();
for (row, weight) in delta.changes {
if let Some(tracker) = &self.tracker {
tracker.lock().unwrap().record_project();
}
let projected = self.project_values(&row.values);
let projected_row = HashableRow::new(row.rowid, projected);
output_delta.changes.push((projected_row, weight));
}
*state = EvalState::Done;
Ok(IOResult::Done(output_delta))
}
fn commit(
&mut self,
deltas: DeltaPair,
_cursors: &mut DbspStateCursors,
) -> Result<IOResult<Delta>> {
// Project operator only uses left delta, right must be empty
assert!(
deltas.right.is_empty(),
"ProjectOperator expects right delta to be empty in commit"
);
let mut output_delta = Delta::new();
// Commit the delta to our internal state and build output
for (row, weight) in &deltas.left.changes {
if let Some(tracker) = &self.tracker {
tracker.lock().unwrap().record_project();
}
let projected = self.project_values(&row.values);
let projected_row = HashableRow::new(row.rowid, projected);
output_delta.changes.push((projected_row, *weight));
}
Ok(crate::types::IOResult::Done(output_delta))
}
fn set_tracker(&mut self, tracker: Arc<Mutex<ComputationTracker>>) {
self.tracker = Some(tracker);
}
}

File diff suppressed because it is too large Load Diff

View File

@@ -12,7 +12,6 @@ use tracing::debug;
pub struct MemoryIO {
files: Arc<Mutex<HashMap<String, Arc<MemoryFile>>>>,
}
unsafe impl Send for MemoryIO {}
// TODO: page size flag
const PAGE_SIZE: usize = 4096;
@@ -76,7 +75,7 @@ pub struct MemoryFile {
pages: UnsafeCell<BTreeMap<usize, MemPage>>,
size: Cell<u64>,
}
unsafe impl Send for MemoryFile {}
unsafe impl Sync for MemoryFile {}
impl File for MemoryFile {

View File

@@ -17,9 +17,6 @@ use tracing::{instrument, trace, Level};
pub struct UnixIO {}
unsafe impl Send for UnixIO {}
unsafe impl Sync for UnixIO {}
impl UnixIO {
#[cfg(feature = "fs")]
pub fn new() -> Result<Self> {
@@ -128,8 +125,6 @@ impl IO for UnixIO {
pub struct UnixFile {
file: Arc<Mutex<std::fs::File>>,
}
unsafe impl Send for UnixFile {}
unsafe impl Sync for UnixFile {}
impl File for UnixFile {
fn lock_file(&self, exclusive: bool) -> Result<()> {

View File

@@ -2174,10 +2174,14 @@ impl Connection {
/// 5. Step through query -> returns Busy -> return Busy to user
///
/// This slight api change demonstrated a better throughtput in `perf/throughput/turso` benchmark
pub fn busy_timeout(&self, mut duration: Option<std::time::Duration>) {
pub fn set_busy_timeout(&self, mut duration: Option<std::time::Duration>) {
duration = duration.filter(|duration| !duration.is_zero());
self.busy_timeout.set(duration);
}
pub fn get_busy_timeout(&self) -> Option<std::time::Duration> {
self.busy_timeout.get()
}
}
#[derive(Debug, Default)]

View File

@@ -1,6 +1,5 @@
use crate::mvcc::clock::LogicalClock;
use crate::mvcc::persistent_storage::Storage;
use crate::return_if_io;
use crate::state_machine::StateMachine;
use crate::state_machine::StateTransition;
use crate::state_machine::TransitionResult;
@@ -542,24 +541,11 @@ impl<Clock: LogicalClock> StateTransition for CommitStateMachine<Clock> {
if mvcc_store.is_exclusive_tx(&self.tx_id) {
mvcc_store.release_exclusive_tx(&self.tx_id);
self.commit_coordinator.pager_commit_lock.unlock();
if !mvcc_store.storage.is_logical_log() {
// FIXME: this function isnt re-entrant
self.pager
.io
.block(|| self.pager.end_tx(false, &self.connection))?;
}
} else if !mvcc_store.storage.is_logical_log() {
self.pager.end_read_tx()?;
}
self.finalize(mvcc_store)?;
return Ok(TransitionResult::Done(()));
}
if mvcc_store.storage.is_logical_log() {
self.state = CommitState::Commit { end_ts };
return Ok(TransitionResult::Continue);
} else {
self.state = CommitState::BeginPagerTxn { end_ts };
}
self.state = CommitState::Commit { end_ts };
Ok(TransitionResult::Continue)
}
CommitState::BeginPagerTxn { end_ts } => {
@@ -851,7 +837,6 @@ impl<Clock: LogicalClock> StateTransition for CommitStateMachine<Clock> {
return Ok(TransitionResult::Continue);
}
CommitState::BeginCommitLogicalLog { end_ts, log_record } => {
assert!(mvcc_store.storage.is_logical_log());
if !mvcc_store.is_exclusive_tx(&self.tx_id) {
// logical log needs to be serialized
let locked = self.commit_coordinator.pager_commit_lock.write();
@@ -866,10 +851,6 @@ impl<Clock: LogicalClock> StateTransition for CommitStateMachine<Clock> {
match result {
IOResult::Done(_) => {}
IOResult::IO(io) => {
assert!(
mvcc_store.storage.is_logical_log(),
"for now logical log is the only storage that can return IO"
);
if !io.finished() {
return Ok(TransitionResult::Io(io));
}
@@ -897,13 +878,11 @@ impl<Clock: LogicalClock> StateTransition for CommitStateMachine<Clock> {
let schema = connection.schema.borrow().clone();
connection.db.update_schema_if_newer(schema)?;
}
if mvcc_store.storage.is_logical_log() {
let tx = mvcc_store.txs.get(&self.tx_id).unwrap();
let tx_unlocked = tx.value();
self.header.write().replace(*tx_unlocked.header.borrow());
tracing::trace!("end_commit_logical_log(tx_id={})", self.tx_id);
self.commit_coordinator.pager_commit_lock.unlock();
}
let tx = mvcc_store.txs.get(&self.tx_id).unwrap();
let tx_unlocked = tx.value();
self.header.write().replace(*tx_unlocked.header.borrow());
tracing::trace!("end_commit_logical_log(tx_id={})", self.tx_id);
self.commit_coordinator.pager_commit_lock.unlock();
self.state = CommitState::CommitEnd { end_ts: *end_ts };
return Ok(TransitionResult::Continue);
}
@@ -1422,38 +1401,12 @@ impl<Clock: LogicalClock> MvStore<Clock> {
///
/// This is used for IMMEDIATE and EXCLUSIVE transaction types where we need
/// to ensure exclusive write access as per SQLite semantics.
#[instrument(skip_all, level = Level::DEBUG)]
pub fn begin_exclusive_tx(
&self,
pager: Arc<Pager>,
maybe_existing_tx_id: Option<TxID>,
) -> Result<IOResult<TxID>> {
self._begin_exclusive_tx(pager, false, maybe_existing_tx_id)
}
/// Upgrades a read transaction to an exclusive write transaction.
///
/// This is used for IMMEDIATE and EXCLUSIVE transaction types where we need
/// to ensure exclusive write access as per SQLite semantics.
pub fn upgrade_to_exclusive_tx(
&self,
pager: Arc<Pager>,
maybe_existing_tx_id: Option<TxID>,
) -> Result<IOResult<TxID>> {
self._begin_exclusive_tx(pager, true, maybe_existing_tx_id)
}
/// Begins an exclusive write transaction that prevents concurrent writes.
///
/// This is used for IMMEDIATE and EXCLUSIVE transaction types where we need
/// to ensure exclusive write access as per SQLite semantics.
#[instrument(skip_all, level = Level::DEBUG)]
fn _begin_exclusive_tx(
&self,
pager: Arc<Pager>,
is_upgrade_from_read: bool,
maybe_existing_tx_id: Option<TxID>,
) -> Result<IOResult<TxID>> {
let is_logical_log = self.storage.is_logical_log();
let tx_id = maybe_existing_tx_id.unwrap_or_else(|| self.get_tx_id());
let begin_ts = if let Some(tx_id) = maybe_existing_tx_id {
self.txs.get(&tx_id).unwrap().value().begin_ts
@@ -1463,16 +1416,6 @@ impl<Clock: LogicalClock> MvStore<Clock> {
self.acquire_exclusive_tx(&tx_id)?;
// Try to acquire the pager read lock
if !is_upgrade_from_read && !is_logical_log {
pager.begin_read_tx().inspect_err(|_| {
tracing::debug!(
"begin_exclusive_tx: tx_id={} failed with Busy on pager_read_lock",
tx_id
);
self.release_exclusive_tx(&tx_id);
})?;
}
let locked = self.commit_coordinator.pager_commit_lock.write();
if !locked {
tracing::debug!(
@@ -1480,46 +1423,18 @@ impl<Clock: LogicalClock> MvStore<Clock> {
tx_id
);
self.release_exclusive_tx(&tx_id);
pager.end_read_tx()?;
return Err(LimboError::Busy);
}
let header = self.get_new_transaction_database_header(&pager);
if is_logical_log {
let tx = Transaction::new(tx_id, begin_ts, header);
tracing::trace!(
"begin_exclusive_tx(tx_id={}) - exclusive write logical log transaction",
tx_id
);
tracing::debug!("begin_exclusive_tx: tx_id={} succeeded", tx_id);
self.txs.insert(tx_id, tx);
return Ok(IOResult::Done(tx_id));
}
// Try to acquire the pager write lock
let begin_w_tx_res = pager.begin_write_tx();
if let Err(LimboError::Busy) = begin_w_tx_res {
tracing::debug!("begin_exclusive_tx: tx_id={} failed with Busy", tx_id);
// Failed to get pager lock - release our exclusive lock
self.commit_coordinator.pager_commit_lock.unlock();
self.release_exclusive_tx(&tx_id);
if maybe_existing_tx_id.is_none() {
// If we were upgrading an existing non-CONCURRENT mvcc transaction to write, we don't end the read tx on Busy.
// But if we were beginning a completely new non-CONCURRENT mvcc transaction, we do end it because the next time the connection
// attempts to do something, it will open a new read tx, which will fail if we don't end this one here.
pager.end_read_tx()?;
}
return Err(LimboError::Busy);
}
return_if_io!(begin_w_tx_res);
let tx = Transaction::new(tx_id, begin_ts, header);
tracing::trace!(
"begin_exclusive_tx(tx_id={}) - exclusive write transaction",
"begin_exclusive_tx(tx_id={}) - exclusive write logical log transaction",
tx_id
);
tracing::debug!("begin_exclusive_tx: tx_id={} succeeded", tx_id);
self.txs.insert(tx_id, tx);
Ok(IOResult::Done(tx_id))
}
@@ -1532,12 +1447,6 @@ impl<Clock: LogicalClock> MvStore<Clock> {
let tx_id = self.get_tx_id();
let begin_ts = self.get_timestamp();
// TODO: we need to tie a pager's read transaction to a transaction ID, so that future refactors to read
// pages from WAL/DB read from a consistent state to maintiain snapshot isolation.
if !self.storage.is_logical_log() {
pager.begin_read_tx()?;
}
// Set txn's header to the global header
let header = self.get_new_transaction_database_header(&pager);
let tx = Transaction::new(tx_id, begin_ts, header);

View File

@@ -1,6 +1,5 @@
use std::cell::RefCell;
use std::fmt::Debug;
use std::sync::Arc;
use std::sync::{Arc, RwLock};
mod logical_log;
use crate::mvcc::database::LogRecord;
@@ -9,32 +8,28 @@ use crate::types::IOResult;
use crate::{File, Result};
pub struct Storage {
logical_log: RefCell<LogicalLog>,
logical_log: RwLock<LogicalLog>,
}
impl Storage {
pub fn new(file: Arc<dyn File>) -> Self {
Self {
logical_log: RefCell::new(LogicalLog::new(file)),
logical_log: RwLock::new(LogicalLog::new(file)),
}
}
}
impl Storage {
pub fn log_tx(&self, m: &LogRecord) -> Result<IOResult<()>> {
self.logical_log.borrow_mut().log_tx(m)
self.logical_log.write().unwrap().log_tx(m)
}
pub fn read_tx_log(&self) -> Result<Vec<LogRecord>> {
todo!()
}
pub fn is_logical_log(&self) -> bool {
true
}
pub fn sync(&self) -> Result<IOResult<()>> {
self.logical_log.borrow_mut().sync()
self.logical_log.write().unwrap().sync()
}
}

View File

@@ -102,6 +102,10 @@ pub fn pragma_for(pragma: &PragmaName) -> Pragma {
PragmaFlags::NoColumns1 | PragmaFlags::Result0,
&["auto_vacuum"],
),
BusyTimeout => Pragma::new(
PragmaFlags::NoColumns1 | PragmaFlags::Result0,
&["busy_timeout"],
),
IntegrityCheck => Pragma::new(
PragmaFlags::NeedSchema | PragmaFlags::ReadOnly | PragmaFlags::Result0,
&["message"],

View File

@@ -527,7 +527,7 @@ impl Schema {
let table = Arc::new(Table::BTree(Arc::new(BTreeTable {
name: view_name.clone(),
root_page: main_root,
columns: incremental_view.columns.clone(),
columns: incremental_view.column_schema.flat_columns(),
primary_key_columns: Vec::new(),
has_rowid: true,
is_strict: false,
@@ -673,11 +673,12 @@ impl Schema {
..
} => {
// Extract actual columns from the SELECT statement
let view_columns = crate::util::extract_view_columns(&select, self);
let view_column_schema =
crate::util::extract_view_columns(&select, self)?;
// If column names were provided in CREATE VIEW (col1, col2, ...),
// use them to rename the columns
let mut final_columns = view_columns;
let mut final_columns = view_column_schema.flat_columns();
for (i, indexed_col) in column_names.iter().enumerate() {
if let Some(col) = final_columns.get_mut(i) {
col.name = Some(indexed_col.col_name.to_string());

View File

@@ -3631,6 +3631,7 @@ impl BTreeCursor {
);
let divider_cell_insert_idx_in_parent =
balance_info.first_divider_cell + sibling_page_idx;
#[cfg(debug_assertions)]
let overflow_cell_count_before = parent_contents.overflow_cells.len();
insert_into_cell(
parent_contents,
@@ -3638,9 +3639,9 @@ impl BTreeCursor {
divider_cell_insert_idx_in_parent,
usable_space,
)?;
let overflow_cell_count_after = parent_contents.overflow_cells.len();
#[cfg(debug_assertions)]
{
let overflow_cell_count_after = parent_contents.overflow_cells.len();
let divider_cell_is_overflow_cell =
overflow_cell_count_after > overflow_cell_count_before;
@@ -6664,6 +6665,20 @@ pub fn btree_init_page(page: &PageRef, page_type: PageType, offset: usize, usabl
contents.write_fragmented_bytes_count(0);
contents.write_rightmost_ptr(0);
#[cfg(debug_assertions)]
{
// we might get already used page from the pool. generally this is not a problem because
// b tree access is very controlled. However, for encrypted pages (and also checksums) we want
// to ensure that there are no reserved bytes that contain old data.
let buffer_len = contents.buffer.len();
turso_assert!(
usable_space <= buffer_len,
"usable_space must be <= buffer_len"
);
// this is no op if usable_space == buffer_len
contents.as_ptr()[usable_space..buffer_len].fill(0);
}
}
fn to_static_buf(buf: &mut [u8]) -> &'static mut [u8] {

View File

@@ -88,11 +88,6 @@ pub struct DatabaseFile {
file: Arc<dyn crate::io::File>,
}
#[cfg(feature = "fs")]
unsafe impl Send for DatabaseFile {}
#[cfg(feature = "fs")]
unsafe impl Sync for DatabaseFile {}
#[cfg(feature = "fs")]
impl DatabaseStorage for DatabaseFile {
#[instrument(skip_all, level = Level::DEBUG)]

View File

@@ -440,11 +440,19 @@ impl EncryptionContext {
};
let metadata_size = self.cipher_mode.metadata_size();
let reserved_bytes = &page[self.page_size - metadata_size..];
let reserved_bytes_zeroed = reserved_bytes.iter().all(|&b| b == 0);
assert!(
reserved_bytes_zeroed,
"last reserved bytes must be empty/zero, but found non-zero bytes"
);
#[cfg(debug_assertions)]
{
use crate::turso_assert;
// In debug builds, ensure that the reserved bytes are zeroed out. So even when we are
// reusing a page from buffer pool, we zero out in debug build so that we can be
// sure that b tree layer is not writing any data into the reserved space.
let reserved_bytes_zeroed = reserved_bytes.iter().all(|&b| b == 0);
turso_assert!(
reserved_bytes_zeroed,
"last reserved bytes must be empty/zero, but found non-zero bytes"
);
}
let payload = &page[encryption_start_offset..self.page_size - metadata_size];
let (encrypted, nonce) = self.encrypt_raw(payload)?;

View File

@@ -48,6 +48,15 @@ pub fn translate_alter_table(
)));
};
// Check if this table has dependent materialized views
let dependent_views = schema.get_dependent_materialized_views(table_name);
if !dependent_views.is_empty() {
return Err(LimboError::ParseError(format!(
"cannot alter table \"{table_name}\": it has dependent materialized view(s): {}",
dependent_views.join(", ")
)));
}
let mut btree = (*original_btree).clone();
Ok(match alter_table {

View File

@@ -1,5 +1,6 @@
use crate::schema::Table;
use crate::translate::emitter::emit_program;
use crate::translate::expr::ParamState;
use crate::translate::optimizer::optimize_plan;
use crate::translate::plan::{DeletePlan, Operation, Plan};
use crate::translate::planner::{parse_limit, parse_where};
@@ -108,6 +109,7 @@ pub fn prepare_delete_plan(
let mut table_references = TableReferences::new(joined_tables, vec![]);
let mut where_predicates = vec![];
let mut param_ctx = ParamState::default();
// Parse the WHERE clause
parse_where(
@@ -116,11 +118,13 @@ pub fn prepare_delete_plan(
None,
&mut where_predicates,
connection,
&mut param_ctx,
)?;
// Parse the LIMIT/OFFSET clause
let (resolved_limit, resolved_offset) =
limit.map_or(Ok((None, None)), |mut l| parse_limit(&mut l, connection))?;
let (resolved_limit, resolved_offset) = limit.map_or(Ok((None, None)), |mut l| {
parse_limit(&mut l, connection, &mut param_ctx)
})?;
let plan = DeletePlan {
table_references,

View File

@@ -1,3 +1,5 @@
use std::sync::Arc;
use tracing::{instrument, Level};
use turso_parser::ast::{self, As, Expr, UnaryOperator};
@@ -8,8 +10,12 @@ use super::plan::TableReferences;
use crate::function::JsonFunc;
use crate::function::{Func, FuncCtx, MathFuncArity, ScalarFunc, VectorFunc};
use crate::functions::datetime;
use crate::parameters::PARAM_PREFIX;
use crate::schema::{affinity, Affinity, Table, Type};
use crate::util::{exprs_are_equivalent, parse_numeric_literal};
use crate::translate::optimizer::TakeOwnership;
use crate::translate::plan::ResultSetColumn;
use crate::translate::planner::parse_row_id;
use crate::util::{exprs_are_equivalent, normalize_ident, parse_numeric_literal};
use crate::vdbe::builder::CursorKey;
use crate::vdbe::{
builder::ProgramBuilder,
@@ -3244,6 +3250,296 @@ where
Ok(WalkControl::Continue)
}
/// Context needed to walk all expressions in a INSERT|UPDATE|SELECT|DELETE body,
/// in the order they are encountered, to ensure that the parameters are rewritten from
/// anonymous ("?") to our internal named scheme so when the columns are re-ordered we are able
/// to bind the proper parameter values.
pub struct ParamState {
/// ALWAYS starts at 1
pub next_param_idx: usize,
}
impl Default for ParamState {
fn default() -> Self {
Self { next_param_idx: 1 }
}
}
/// Rewrite ast::Expr in place, binding Column references/rewriting Expr::Id -> Expr::Column
/// using the provided TableReferences, and replacing anonymous parameters with internal named
/// ones, as well as normalizing any DoublyQualified/Qualified quoted identifiers.
pub fn bind_and_rewrite_expr<'a>(
top_level_expr: &mut ast::Expr,
mut referenced_tables: Option<&'a mut TableReferences>,
result_columns: Option<&'a [ResultSetColumn]>,
connection: &'a Arc<crate::Connection>,
param_state: &mut ParamState,
) -> Result<WalkControl> {
walk_expr_mut(
top_level_expr,
&mut |expr: &mut ast::Expr| -> Result<WalkControl> {
match expr {
ast::Expr::Id(ast::Name::Ident(n)) if n.eq_ignore_ascii_case("true") => {
*expr = ast::Expr::Literal(ast::Literal::Numeric("1".to_string()));
}
ast::Expr::Id(ast::Name::Ident(n)) if n.eq_ignore_ascii_case("false") => {
*expr = ast::Expr::Literal(ast::Literal::Numeric("0".to_string()));
}
// Rewrite anonymous variables in encounter order.
ast::Expr::Variable(var) if var.is_empty() => {
*expr = ast::Expr::Variable(format!(
"{}{}",
PARAM_PREFIX, param_state.next_param_idx
));
param_state.next_param_idx += 1;
}
ast::Expr::Qualified(ast::Name::Quoted(ns), ast::Name::Quoted(c))
| ast::Expr::DoublyQualified(_, ast::Name::Quoted(ns), ast::Name::Quoted(c)) => {
*expr = ast::Expr::Qualified(
ast::Name::Ident(normalize_ident(ns.as_str())),
ast::Name::Ident(normalize_ident(c.as_str())),
);
}
ast::Expr::Between {
lhs,
not,
start,
end,
} => {
let (lower_op, upper_op) = if *not {
(ast::Operator::Greater, ast::Operator::Greater)
} else {
(ast::Operator::LessEquals, ast::Operator::LessEquals)
};
let start = start.take_ownership();
let lhs_v = lhs.take_ownership();
let end = end.take_ownership();
let lower =
ast::Expr::Binary(Box::new(start), lower_op, Box::new(lhs_v.clone()));
let upper = ast::Expr::Binary(Box::new(lhs_v), upper_op, Box::new(end));
*expr = if *not {
ast::Expr::Binary(Box::new(lower), ast::Operator::Or, Box::new(upper))
} else {
ast::Expr::Binary(Box::new(lower), ast::Operator::And, Box::new(upper))
};
}
_ => {}
}
if let Some(referenced_tables) = &mut referenced_tables {
match expr {
// Unqualified identifier binding (including rowid aliases, outer refs, result-column fallback).
Expr::Id(id) => {
let normalized_id = normalize_ident(id.as_str());
if !referenced_tables.joined_tables().is_empty() {
if let Some(row_id_expr) = parse_row_id(
&normalized_id,
referenced_tables.joined_tables()[0].internal_id,
|| referenced_tables.joined_tables().len() != 1,
)? {
*expr = row_id_expr;
return Ok(WalkControl::Continue);
}
}
let mut match_result = None;
// First check joined tables
for joined_table in referenced_tables.joined_tables().iter() {
let col_idx = joined_table.table.columns().iter().position(|c| {
c.name
.as_ref()
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_id))
});
if col_idx.is_some() {
if match_result.is_some() {
crate::bail_parse_error!("Column {} is ambiguous", id.as_str());
}
let col =
joined_table.table.columns().get(col_idx.unwrap()).unwrap();
match_result = Some((
joined_table.internal_id,
col_idx.unwrap(),
col.is_rowid_alias,
));
}
}
// Then check outer query references, if we still didn't find something.
// Normally finding multiple matches for a non-qualified column is an error (column x is ambiguous)
// but in the case of subqueries, the inner query takes precedence.
// For example:
// SELECT * FROM t WHERE x = (SELECT x FROM t2)
// In this case, there is no ambiguity:
// - x in the outer query refers to t.x,
// - x in the inner query refers to t2.x.
if match_result.is_none() {
for outer_ref in referenced_tables.outer_query_refs().iter() {
let col_idx = outer_ref.table.columns().iter().position(|c| {
c.name.as_ref().is_some_and(|name| {
name.eq_ignore_ascii_case(&normalized_id)
})
});
if col_idx.is_some() {
if match_result.is_some() {
crate::bail_parse_error!(
"Column {} is ambiguous",
id.as_str()
);
}
let col =
outer_ref.table.columns().get(col_idx.unwrap()).unwrap();
match_result = Some((
outer_ref.internal_id,
col_idx.unwrap(),
col.is_rowid_alias,
));
}
}
}
if let Some((table_id, col_idx, is_rowid_alias)) = match_result {
*expr = Expr::Column {
database: None, // TODO: support different databases
table: table_id,
column: col_idx,
is_rowid_alias,
};
referenced_tables.mark_column_used(table_id, col_idx);
return Ok(WalkControl::Continue);
}
if let Some(result_columns) = result_columns {
for result_column in result_columns.iter() {
if result_column
.name(referenced_tables)
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_id))
{
*expr = result_column.expr.clone();
return Ok(WalkControl::Continue);
}
}
}
// SQLite behavior: Only double-quoted identifiers get fallback to string literals
// Single quotes are handled as literals earlier, unquoted identifiers must resolve to columns
if id.is_double_quoted() {
// Convert failed double-quoted identifier to string literal
*expr = Expr::Literal(ast::Literal::String(id.as_str().to_string()));
return Ok(WalkControl::Continue);
} else {
// Unquoted identifiers must resolve to columns - no fallback
crate::bail_parse_error!("no such column: {}", id.as_str())
}
}
Expr::Qualified(tbl, id) => {
let normalized_table_name = normalize_ident(tbl.as_str());
let matching_tbl = referenced_tables
.find_table_and_internal_id_by_identifier(&normalized_table_name);
if matching_tbl.is_none() {
crate::bail_parse_error!("no such table: {}", normalized_table_name);
}
let (tbl_id, tbl) = matching_tbl.unwrap();
let normalized_id = normalize_ident(id.as_str());
if let Some(row_id_expr) = parse_row_id(&normalized_id, tbl_id, || false)? {
*expr = row_id_expr;
return Ok(WalkControl::Continue);
}
let col_idx = tbl.columns().iter().position(|c| {
c.name
.as_ref()
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_id))
});
let Some(col_idx) = col_idx else {
crate::bail_parse_error!("no such column: {}", normalized_id);
};
let col = tbl.columns().get(col_idx).unwrap();
*expr = Expr::Column {
database: None, // TODO: support different databases
table: tbl_id,
column: col_idx,
is_rowid_alias: col.is_rowid_alias,
};
referenced_tables.mark_column_used(tbl_id, col_idx);
return Ok(WalkControl::Continue);
}
Expr::DoublyQualified(db_name, tbl_name, col_name) => {
let normalized_col_name = normalize_ident(col_name.as_str());
// Create a QualifiedName and use existing resolve_database_id method
let qualified_name = ast::QualifiedName {
db_name: Some(db_name.clone()),
name: tbl_name.clone(),
alias: None,
};
let database_id = connection.resolve_database_id(&qualified_name)?;
// Get the table from the specified database
let table = connection
.with_schema(database_id, |schema| schema.get_table(tbl_name.as_str()))
.ok_or_else(|| {
crate::LimboError::ParseError(format!(
"no such table: {}.{}",
db_name.as_str(),
tbl_name.as_str()
))
})?;
// Find the column in the table
let col_idx = table
.columns()
.iter()
.position(|c| {
c.name.as_ref().is_some_and(|name| {
name.eq_ignore_ascii_case(&normalized_col_name)
})
})
.ok_or_else(|| {
crate::LimboError::ParseError(format!(
"Column: {}.{}.{} not found",
db_name.as_str(),
tbl_name.as_str(),
col_name.as_str()
))
})?;
let col = table.columns().get(col_idx).unwrap();
// Check if this is a rowid alias
let is_rowid_alias = col.is_rowid_alias;
// Convert to Column expression - since this is a cross-database reference,
// we need to create a synthetic table reference for it
// For now, we'll error if the table isn't already in the referenced tables
let normalized_tbl_name = normalize_ident(tbl_name.as_str());
let matching_tbl = referenced_tables
.find_table_and_internal_id_by_identifier(&normalized_tbl_name);
if let Some((tbl_id, _)) = matching_tbl {
// Table is already in referenced tables, use existing internal ID
*expr = Expr::Column {
database: Some(database_id),
table: tbl_id,
column: col_idx,
is_rowid_alias,
};
referenced_tables.mark_column_used(tbl_id, col_idx);
} else {
return Err(crate::LimboError::ParseError(format!(
"table {normalized_tbl_name} is not in FROM clause - cross-database column references require the table to be explicitly joined"
)));
}
}
_ => {}
}
}
Ok(WalkControl::Continue)
},
)
}
/// Recursively walks a mutable expression, applying a function to each sub-expression.
pub fn walk_expr_mut<F>(expr: &mut ast::Expr, func: &mut F) -> Result<WalkControl>
where
@@ -3709,12 +4005,12 @@ pub fn process_returning_clause(
table_name: &str,
program: &mut ProgramBuilder,
connection: &std::sync::Arc<crate::Connection>,
param_ctx: &mut ParamState,
) -> Result<(
Vec<super::plan::ResultSetColumn>,
super::plan::TableReferences,
)> {
use super::plan::{ColumnUsedMask, JoinedTable, Operation, ResultSetColumn, TableReferences};
use super::planner::bind_column_references;
let mut result_columns = vec![];
@@ -3741,7 +4037,13 @@ pub fn process_returning_clause(
ast::ResultColumn::Expr(expr, alias) => {
let column_alias = determine_column_alias(expr, alias, table);
bind_column_references(expr, &mut table_references, None, connection)?;
bind_and_rewrite_expr(
expr,
Some(&mut table_references),
None,
connection,
param_ctx,
)?;
result_columns.push(ResultSetColumn {
expr: *expr.clone(),

View File

@@ -10,7 +10,8 @@ use crate::translate::emitter::{
emit_cdc_insns, emit_cdc_patch_record, prepare_cdc_if_necessary, OperationMode,
};
use crate::translate::expr::{
emit_returning_results, process_returning_clause, ReturningValueRegisters,
bind_and_rewrite_expr, emit_returning_results, process_returning_clause, ParamState,
ReturningValueRegisters,
};
use crate::translate::planner::ROWID;
use crate::translate::upsert::{
@@ -31,7 +32,6 @@ use crate::{Result, SymbolTable, VirtualTable};
use super::emitter::Resolver;
use super::expr::{translate_expr, translate_expr_no_constant_opt, NoConstantOptReason};
use super::optimizer::rewrite_expr;
use super::plan::QueryDestination;
use super::select::translate_select;
@@ -118,7 +118,7 @@ pub fn translate_insert(
let mut values: Option<Vec<Box<Expr>>> = None;
let mut upsert_opt: Option<Upsert> = None;
let mut param_idx = 1;
let mut param_ctx = ParamState::default();
let mut inserting_multiple_rows = false;
if let InsertBody::Select(select, upsert) = &mut body {
match &mut select.body.select {
@@ -144,7 +144,7 @@ pub fn translate_insert(
}
_ => {}
}
rewrite_expr(expr, &mut param_idx)?;
bind_and_rewrite_expr(expr, None, None, connection, &mut param_ctx)?;
}
values = values_expr.pop();
}
@@ -157,10 +157,10 @@ pub fn translate_insert(
} = &mut upsert.do_clause
{
for set in sets.iter_mut() {
rewrite_expr(set.expr.as_mut(), &mut param_idx)?;
bind_and_rewrite_expr(&mut set.expr, None, None, connection, &mut param_ctx)?;
}
if let Some(ref mut where_expr) = where_clause {
rewrite_expr(where_expr.as_mut(), &mut param_idx)?;
bind_and_rewrite_expr(where_expr, None, None, connection, &mut param_ctx)?;
}
}
}
@@ -180,6 +180,7 @@ pub fn translate_insert(
table_name.as_str(),
&mut program,
connection,
&mut param_ctx,
)?;
let mut yield_reg_opt = None;

File diff suppressed because it is too large Load Diff

View File

@@ -8,15 +8,13 @@ use join::{compute_best_join_order, BestJoinOrderResult};
use lift_common_subexpressions::lift_common_subexpressions_from_binary_or_terms;
use order::{compute_order_target, plan_satisfies_order_target, EliminatesSortBy};
use turso_ext::{ConstraintInfo, ConstraintUsage};
use turso_macros::match_ignore_ascii_case;
use turso_parser::ast::{self, Expr, SortOrder};
use crate::{
parameters::PARAM_PREFIX,
schema::{Index, IndexColumn, Schema, Table},
translate::{
expr::walk_expr_mut, expr::WalkControl, optimizer::access_method::AccessMethodParams,
optimizer::constraints::TableConstraints, plan::Scan, plan::TerminationKey,
optimizer::access_method::AccessMethodParams, optimizer::constraints::TableConstraints,
plan::Scan, plan::TerminationKey,
},
types::SeekOp,
LimboError, Result,
@@ -64,7 +62,7 @@ pub fn optimize_plan(plan: &mut Plan, schema: &Schema) -> Result<()> {
*/
pub fn optimize_select_plan(plan: &mut SelectPlan, schema: &Schema) -> Result<()> {
optimize_subqueries(plan, schema)?;
rewrite_exprs_select(plan)?;
lift_common_subexpressions_from_binary_or_terms(&mut plan.where_clause)?;
if let ConstantConditionEliminationResult::ImpossibleCondition =
eliminate_constant_conditions(&mut plan.where_clause)?
{
@@ -89,7 +87,7 @@ pub fn optimize_select_plan(plan: &mut SelectPlan, schema: &Schema) -> Result<()
}
fn optimize_delete_plan(plan: &mut DeletePlan, schema: &Schema) -> Result<()> {
rewrite_exprs_delete(plan)?;
lift_common_subexpressions_from_binary_or_terms(&mut plan.where_clause)?;
if let ConstantConditionEliminationResult::ImpossibleCondition =
eliminate_constant_conditions(&mut plan.where_clause)?
{
@@ -110,7 +108,7 @@ fn optimize_delete_plan(plan: &mut DeletePlan, schema: &Schema) -> Result<()> {
}
fn optimize_update_plan(plan: &mut UpdatePlan, schema: &Schema) -> Result<()> {
rewrite_exprs_update(plan)?;
lift_common_subexpressions_from_binary_or_terms(&mut plan.where_clause)?;
if let ConstantConditionEliminationResult::ImpossibleCondition =
eliminate_constant_conditions(&mut plan.where_clause)?
{
@@ -558,62 +556,6 @@ fn eliminate_constant_conditions(
Ok(ConstantConditionEliminationResult::Continue)
}
fn rewrite_exprs_select(plan: &mut SelectPlan) -> Result<()> {
let mut param_count = 1;
for rc in plan.result_columns.iter_mut() {
rewrite_expr(&mut rc.expr, &mut param_count)?;
}
for agg in plan.aggregates.iter_mut() {
rewrite_expr(&mut agg.original_expr, &mut param_count)?;
}
lift_common_subexpressions_from_binary_or_terms(&mut plan.where_clause)?;
for cond in plan.where_clause.iter_mut() {
rewrite_expr(&mut cond.expr, &mut param_count)?;
}
if let Some(group_by) = &mut plan.group_by {
for expr in group_by.exprs.iter_mut() {
rewrite_expr(expr, &mut param_count)?;
}
}
for (expr, _) in plan.order_by.iter_mut() {
rewrite_expr(expr, &mut param_count)?;
}
if let Some(window) = &mut plan.window {
for func in window.functions.iter_mut() {
rewrite_expr(&mut func.original_expr, &mut param_count)?;
}
}
Ok(())
}
fn rewrite_exprs_delete(plan: &mut DeletePlan) -> Result<()> {
let mut param_idx = 1;
for cond in plan.where_clause.iter_mut() {
rewrite_expr(&mut cond.expr, &mut param_idx)?;
}
Ok(())
}
fn rewrite_exprs_update(plan: &mut UpdatePlan) -> Result<()> {
let mut param_idx = 1;
for (_, expr) in plan.set_clauses.iter_mut() {
rewrite_expr(expr, &mut param_idx)?;
}
for cond in plan.where_clause.iter_mut() {
rewrite_expr(&mut cond.expr, &mut param_idx)?;
}
for (expr, _) in plan.order_by.iter_mut() {
rewrite_expr(expr, &mut param_idx)?;
}
if let Some(rc) = plan.returning.as_mut() {
for rc in rc.iter_mut() {
rewrite_expr(&mut rc.expr, &mut param_idx)?;
}
}
Ok(())
}
#[derive(Debug, Clone, Copy, PartialEq, Eq)]
pub enum AlwaysTrueOrFalse {
AlwaysTrue,
@@ -1449,77 +1391,7 @@ fn build_seek_def(
})
}
pub fn rewrite_expr(top_level_expr: &mut ast::Expr, param_idx: &mut usize) -> Result<WalkControl> {
walk_expr_mut(
top_level_expr,
&mut |expr: &mut ast::Expr| -> Result<WalkControl> {
match expr {
ast::Expr::Id(id) => {
// Convert "true" and "false" to 1 and 0
let id_bytes = id.as_str().as_bytes();
match_ignore_ascii_case!(match id_bytes {
b"true" => {
*expr = ast::Expr::Literal(ast::Literal::Numeric("1".to_owned()));
}
b"false" => {
*expr = ast::Expr::Literal(ast::Literal::Numeric("0".to_owned()));
}
_ => {}
})
}
ast::Expr::Variable(var) => {
if var.is_empty() {
// rewrite anonymous variables only, ensure that the `param_idx` starts at 1 and
// all the expressions are rewritten in the order they come in the statement
*expr = ast::Expr::Variable(format!("{PARAM_PREFIX}{param_idx}"));
*param_idx += 1;
}
}
ast::Expr::Between {
lhs,
not,
start,
end,
} => {
// Convert `y NOT BETWEEN x AND z` to `x > y OR y > z`
let (lower_op, upper_op) = if *not {
(ast::Operator::Greater, ast::Operator::Greater)
} else {
// Convert `y BETWEEN x AND z` to `x <= y AND y <= z`
(ast::Operator::LessEquals, ast::Operator::LessEquals)
};
let start = start.take_ownership();
let lhs = lhs.take_ownership();
let end = end.take_ownership();
let lower_bound =
ast::Expr::Binary(Box::new(start), lower_op, Box::new(lhs.clone()));
let upper_bound = ast::Expr::Binary(Box::new(lhs), upper_op, Box::new(end));
if *not {
*expr = ast::Expr::Binary(
Box::new(lower_bound),
ast::Operator::Or,
Box::new(upper_bound),
);
} else {
*expr = ast::Expr::Binary(
Box::new(lower_bound),
ast::Operator::And,
Box::new(upper_bound),
);
}
}
_ => {}
}
Ok(WalkControl::Continue)
},
)
}
trait TakeOwnership {
pub trait TakeOwnership {
fn take_ownership(&mut self) -> Self;
}

View File

@@ -11,23 +11,23 @@ use super::{
select::prepare_select_plan,
SymbolTable,
};
use crate::function::{AggFunc, ExtFunc};
use crate::translate::expr::WalkControl;
use crate::translate::plan::{Window, WindowFunction};
use crate::{
ast::Limit,
function::Func,
schema::{Schema, Table},
translate::expr::walk_expr_mut,
util::{exprs_are_equivalent, normalize_ident},
vdbe::builder::TableRefIdCounter,
Result,
};
use turso_macros::match_ignore_ascii_case;
use crate::{
function::{AggFunc, ExtFunc},
translate::expr::{bind_and_rewrite_expr, ParamState},
};
use turso_parser::ast::Literal::Null;
use turso_parser::ast::{
self, As, Expr, FromClause, JoinType, Literal, Materialized, Over, QualifiedName,
TableInternalId, With,
self, As, Expr, FromClause, JoinType, Materialized, Over, QualifiedName, TableInternalId, With,
};
pub const ROWID: &str = "rowid";
@@ -262,231 +262,6 @@ fn add_aggregate_if_not_exists(
Ok(())
}
pub fn bind_column_references(
top_level_expr: &mut Expr,
referenced_tables: &mut TableReferences,
result_columns: Option<&[ResultSetColumn]>,
connection: &Arc<crate::Connection>,
) -> Result<WalkControl> {
walk_expr_mut(
top_level_expr,
&mut |expr: &mut Expr| -> Result<WalkControl> {
match expr {
Expr::Id(id) => {
// true and false are special constants that are effectively aliases for 1 and 0
// and not identifiers of columns
let id_bytes = id.as_str().as_bytes();
match_ignore_ascii_case!(match id_bytes {
b"true" | b"false" => {
return Ok(WalkControl::Continue);
}
_ => {}
});
let normalized_id = normalize_ident(id.as_str());
if !referenced_tables.joined_tables().is_empty() {
if let Some(row_id_expr) = parse_row_id(
&normalized_id,
referenced_tables.joined_tables()[0].internal_id,
|| referenced_tables.joined_tables().len() != 1,
)? {
*expr = row_id_expr;
return Ok(WalkControl::Continue);
}
}
let mut match_result = None;
// First check joined tables
for joined_table in referenced_tables.joined_tables().iter() {
let col_idx = joined_table.table.columns().iter().position(|c| {
c.name
.as_ref()
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_id))
});
if col_idx.is_some() {
if match_result.is_some() {
crate::bail_parse_error!("Column {} is ambiguous", id.as_str());
}
let col = joined_table.table.columns().get(col_idx.unwrap()).unwrap();
match_result = Some((
joined_table.internal_id,
col_idx.unwrap(),
col.is_rowid_alias,
));
}
}
// Then check outer query references, if we still didn't find something.
// Normally finding multiple matches for a non-qualified column is an error (column x is ambiguous)
// but in the case of subqueries, the inner query takes precedence.
// For example:
// SELECT * FROM t WHERE x = (SELECT x FROM t2)
// In this case, there is no ambiguity:
// - x in the outer query refers to t.x,
// - x in the inner query refers to t2.x.
if match_result.is_none() {
for outer_ref in referenced_tables.outer_query_refs().iter() {
let col_idx = outer_ref.table.columns().iter().position(|c| {
c.name
.as_ref()
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_id))
});
if col_idx.is_some() {
if match_result.is_some() {
crate::bail_parse_error!("Column {} is ambiguous", id.as_str());
}
let col = outer_ref.table.columns().get(col_idx.unwrap()).unwrap();
match_result = Some((
outer_ref.internal_id,
col_idx.unwrap(),
col.is_rowid_alias,
));
}
}
}
if let Some((table_id, col_idx, is_rowid_alias)) = match_result {
*expr = Expr::Column {
database: None, // TODO: support different databases
table: table_id,
column: col_idx,
is_rowid_alias,
};
referenced_tables.mark_column_used(table_id, col_idx);
return Ok(WalkControl::Continue);
}
if let Some(result_columns) = result_columns {
for result_column in result_columns.iter() {
if result_column
.name(referenced_tables)
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_id))
{
*expr = result_column.expr.clone();
return Ok(WalkControl::Continue);
}
}
}
// SQLite behavior: Only double-quoted identifiers get fallback to string literals
// Single quotes are handled as literals earlier, unquoted identifiers must resolve to columns
if id.is_double_quoted() {
// Convert failed double-quoted identifier to string literal
*expr = Expr::Literal(Literal::String(id.as_str().to_string()));
Ok(WalkControl::Continue)
} else {
// Unquoted identifiers must resolve to columns - no fallback
crate::bail_parse_error!("no such column: {}", id.as_str())
}
}
Expr::Qualified(tbl, id) => {
let normalized_table_name = normalize_ident(tbl.as_str());
let matching_tbl = referenced_tables
.find_table_and_internal_id_by_identifier(&normalized_table_name);
if matching_tbl.is_none() {
crate::bail_parse_error!("no such table: {}", normalized_table_name);
}
let (tbl_id, tbl) = matching_tbl.unwrap();
let normalized_id = normalize_ident(id.as_str());
if let Some(row_id_expr) = parse_row_id(&normalized_id, tbl_id, || false)? {
*expr = row_id_expr;
return Ok(WalkControl::Continue);
}
let col_idx = tbl.columns().iter().position(|c| {
c.name
.as_ref()
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_id))
});
let Some(col_idx) = col_idx else {
crate::bail_parse_error!("no such column: {}", normalized_id);
};
let col = tbl.columns().get(col_idx).unwrap();
*expr = Expr::Column {
database: None, // TODO: support different databases
table: tbl_id,
column: col_idx,
is_rowid_alias: col.is_rowid_alias,
};
referenced_tables.mark_column_used(tbl_id, col_idx);
Ok(WalkControl::Continue)
}
Expr::DoublyQualified(db_name, tbl_name, col_name) => {
let normalized_col_name = normalize_ident(col_name.as_str());
// Create a QualifiedName and use existing resolve_database_id method
let qualified_name = ast::QualifiedName {
db_name: Some(db_name.clone()),
name: tbl_name.clone(),
alias: None,
};
let database_id = connection.resolve_database_id(&qualified_name)?;
// Get the table from the specified database
let table = connection
.with_schema(database_id, |schema| schema.get_table(tbl_name.as_str()))
.ok_or_else(|| {
crate::LimboError::ParseError(format!(
"no such table: {}.{}",
db_name.as_str(),
tbl_name.as_str()
))
})?;
// Find the column in the table
let col_idx = table
.columns()
.iter()
.position(|c| {
c.name
.as_ref()
.is_some_and(|name| name.eq_ignore_ascii_case(&normalized_col_name))
})
.ok_or_else(|| {
crate::LimboError::ParseError(format!(
"Column: {}.{}.{} not found",
db_name.as_str(),
tbl_name.as_str(),
col_name.as_str()
))
})?;
let col = table.columns().get(col_idx).unwrap();
// Check if this is a rowid alias
let is_rowid_alias = col.is_rowid_alias;
// Convert to Column expression - since this is a cross-database reference,
// we need to create a synthetic table reference for it
// For now, we'll error if the table isn't already in the referenced tables
let normalized_tbl_name = normalize_ident(tbl_name.as_str());
let matching_tbl = referenced_tables
.find_table_and_internal_id_by_identifier(&normalized_tbl_name);
if let Some((tbl_id, _)) = matching_tbl {
// Table is already in referenced tables, use existing internal ID
*expr = Expr::Column {
database: Some(database_id),
table: tbl_id,
column: col_idx,
is_rowid_alias,
};
referenced_tables.mark_column_used(tbl_id, col_idx);
} else {
return Err(crate::LimboError::ParseError(format!(
"table {normalized_tbl_name} is not in FROM clause - cross-database column references require the table to be explicitly joined"
)));
}
Ok(WalkControl::Continue)
}
_ => Ok(WalkControl::Continue),
}
},
)
}
#[allow(clippy::too_many_arguments)]
fn parse_from_clause_table(
schema: &Schema,
@@ -663,7 +438,7 @@ fn parse_table(
let btree_table = Arc::new(crate::schema::BTreeTable {
name: view_guard.name().to_string(),
root_page,
columns: view_guard.columns.clone(),
columns: view_guard.column_schema.flat_columns(),
primary_key_columns: Vec::new(),
has_rowid: true,
is_strict: false,
@@ -776,6 +551,7 @@ pub fn parse_from(
table_references: &mut TableReferences,
table_ref_counter: &mut TableRefIdCounter,
connection: &Arc<crate::Connection>,
param_ctx: &mut ParamState,
) -> Result<()> {
if from.is_none() {
return Ok(());
@@ -874,6 +650,7 @@ pub fn parse_from(
table_references,
table_ref_counter,
connection,
param_ctx,
)?;
}
@@ -886,12 +663,19 @@ pub fn parse_where(
result_columns: Option<&[ResultSetColumn]>,
out_where_clause: &mut Vec<WhereTerm>,
connection: &Arc<crate::Connection>,
param_ctx: &mut ParamState,
) -> Result<()> {
if let Some(where_expr) = where_clause {
let start_idx = out_where_clause.len();
break_predicate_at_and_boundaries(where_expr, out_where_clause);
for expr in out_where_clause[start_idx..].iter_mut() {
bind_column_references(&mut expr.expr, table_references, result_columns, connection)?;
bind_and_rewrite_expr(
&mut expr.expr,
Some(table_references),
result_columns,
connection,
param_ctx,
)?;
}
Ok(())
} else {
@@ -1084,6 +868,7 @@ fn parse_join(
table_references: &mut TableReferences,
table_ref_counter: &mut TableRefIdCounter,
connection: &Arc<crate::Connection>,
param_ctx: &mut ParamState,
) -> Result<()> {
let ast::JoinedSelectTable {
operator: join_operator,
@@ -1171,11 +956,12 @@ fn parse_join(
} else {
None
};
bind_column_references(
bind_and_rewrite_expr(
&mut predicate.expr,
table_references,
Some(table_references),
None,
connection,
param_ctx,
)?;
}
}
@@ -1290,7 +1076,7 @@ pub fn break_predicate_at_and_boundaries<T: From<Expr>>(
}
}
fn parse_row_id<F>(
pub fn parse_row_id<F>(
column_name: &str,
table_id: TableInternalId,
fn_check: F,
@@ -1315,11 +1101,11 @@ where
pub fn parse_limit(
limit: &mut Limit,
connection: &std::sync::Arc<crate::Connection>,
param_ctx: &mut ParamState,
) -> Result<(Option<Box<Expr>>, Option<Box<Expr>>)> {
let mut empty_refs = TableReferences::new(Vec::new(), Vec::new());
bind_column_references(&mut limit.expr, &mut empty_refs, None, connection)?;
bind_and_rewrite_expr(&mut limit.expr, None, None, connection, param_ctx)?;
if let Some(ref mut off_expr) = limit.offset {
bind_column_references(off_expr, &mut empty_refs, None, connection)?;
bind_and_rewrite_expr(off_expr, None, None, connection, param_ctx)?;
}
Ok((Some(limit.expr.clone()), limit.offset.clone()))
}

View File

@@ -99,7 +99,7 @@ fn update_pragma(
let app_id_value = match data {
Value::Integer(i) => i as i32,
Value::Float(f) => f as i32,
_ => unreachable!(),
_ => bail_parse_error!("expected integer, got {:?}", data),
};
program.emit_insn(Insn::SetCookie {
@@ -110,6 +110,19 @@ fn update_pragma(
});
Ok((program, TransactionMode::Write))
}
PragmaName::BusyTimeout => {
let data = parse_signed_number(&value)?;
let busy_timeout_ms = match data {
Value::Integer(i) => i as i32,
Value::Float(f) => f as i32,
_ => bail_parse_error!("expected integer, got {:?}", data),
};
let busy_timeout_ms = busy_timeout_ms.max(0);
connection.set_busy_timeout(Some(std::time::Duration::from_millis(
busy_timeout_ms as u64,
)));
Ok((program, TransactionMode::Write))
}
PragmaName::CacheSize => {
let cache_size = match parse_signed_number(&value)? {
Value::Integer(size) => size,
@@ -388,6 +401,18 @@ fn query_pragma(
program.emit_result_row(register, 1);
Ok((program, TransactionMode::Read))
}
PragmaName::BusyTimeout => {
program.emit_int(
connection
.get_busy_timeout()
.map(|t| t.as_millis() as i64)
.unwrap_or_default(),
register,
);
program.emit_result_row(register, 1);
program.add_pragma_result_column(pragma.to_string());
Ok((program, TransactionMode::None))
}
PragmaName::CacheSize => {
program.emit_int(connection.get_cache_size() as i64, register);
program.emit_result_row(register, 1);
@@ -508,7 +533,8 @@ fn query_pragma(
emit_columns_for_table_info(&mut program, table.columns(), base_reg);
} else if let Some(view_mutex) = schema.get_materialized_view(&name) {
let view = view_mutex.lock().unwrap();
emit_columns_for_table_info(&mut program, &view.columns, base_reg);
let flat_columns = view.column_schema.flat_columns();
emit_columns_for_table_info(&mut program, &flat_columns, base_reg);
} else if let Some(view) = schema.get_view(&name) {
emit_columns_for_table_info(&mut program, &view.columns, base_reg);
}

View File

@@ -4,11 +4,12 @@ use super::plan::{
Search, TableReferences, WhereTerm, Window,
};
use crate::schema::Table;
use crate::translate::expr::{bind_and_rewrite_expr, ParamState};
use crate::translate::optimizer::optimize_plan;
use crate::translate::plan::{GroupBy, Plan, ResultSetColumn, SelectPlan};
use crate::translate::planner::{
bind_column_references, break_predicate_at_and_boundaries, parse_from, parse_limit,
parse_where, resolve_window_and_aggregate_functions,
break_predicate_at_and_boundaries, parse_from, parse_limit, parse_where,
resolve_window_and_aggregate_functions,
};
use crate::translate::window::plan_windows;
use crate::util::normalize_ident;
@@ -98,6 +99,7 @@ pub fn prepare_select_plan(
connection: &Arc<crate::Connection>,
) -> Result<Plan> {
let compounds = select.body.compounds;
let mut param_ctx = ParamState::default();
match compounds.is_empty() {
true => Ok(Plan::Select(prepare_one_select_plan(
schema,
@@ -110,6 +112,7 @@ pub fn prepare_select_plan(
table_ref_counter,
query_destination,
connection,
&mut param_ctx,
)?)),
false => {
let mut last = prepare_one_select_plan(
@@ -123,6 +126,7 @@ pub fn prepare_select_plan(
table_ref_counter,
query_destination.clone(),
connection,
&mut param_ctx,
)?;
let mut left = Vec::with_capacity(compounds.len());
@@ -139,6 +143,7 @@ pub fn prepare_select_plan(
table_ref_counter,
query_destination.clone(),
connection,
&mut param_ctx,
)?;
}
@@ -149,9 +154,9 @@ pub fn prepare_select_plan(
crate::bail_parse_error!("SELECTs to the left and right of {} do not have the same number of result columns", operator);
}
}
let (limit, offset) = select
.limit
.map_or(Ok((None, None)), |mut l| parse_limit(&mut l, connection))?;
let (limit, offset) = select.limit.map_or(Ok((None, None)), |mut l| {
parse_limit(&mut l, connection, &mut param_ctx)
})?;
// FIXME: handle ORDER BY for compound selects
if !select.order_by.is_empty() {
@@ -184,6 +189,7 @@ fn prepare_one_select_plan(
table_ref_counter: &mut TableRefIdCounter,
query_destination: QueryDestination,
connection: &Arc<crate::Connection>,
param_ctx: &mut ParamState,
) -> Result<SelectPlan> {
match select {
ast::OneSelect::Select {
@@ -230,6 +236,7 @@ fn prepare_one_select_plan(
&mut table_references,
table_ref_counter,
connection,
param_ctx,
)?;
// Preallocate space for the result columns
@@ -255,7 +262,6 @@ fn prepare_one_select_plan(
})
.sum(),
);
let mut plan = SelectPlan {
join_order: table_references
.joined_tables()
@@ -288,19 +294,21 @@ fn prepare_one_select_plan(
let mut window = Window::new(Some(name), &window_def.window)?;
for expr in window.partition_by.iter_mut() {
bind_column_references(
bind_and_rewrite_expr(
expr,
&mut plan.table_references,
Some(&plan.result_columns),
Some(&mut plan.table_references),
None,
connection,
param_ctx,
)?;
}
for (expr, _) in window.order_by.iter_mut() {
bind_column_references(
bind_and_rewrite_expr(
expr,
&mut plan.table_references,
Some(&plan.result_columns),
Some(&mut plan.table_references),
None,
connection,
param_ctx,
)?;
}
@@ -357,11 +365,12 @@ fn prepare_one_select_plan(
}
}
ResultColumn::Expr(ref mut expr, maybe_alias) => {
bind_column_references(
bind_and_rewrite_expr(
expr,
&mut plan.table_references,
Some(&plan.result_columns),
Some(&mut plan.table_references),
None,
connection,
param_ctx,
)?;
let contains_aggregates = resolve_window_and_aggregate_functions(
schema,
@@ -385,7 +394,12 @@ fn prepare_one_select_plan(
// This step can only be performed at this point, because all table references are now available.
// Virtual table predicates may depend on column bindings from tables to the right in the join order,
// so we must wait until the full set of references has been collected.
add_vtab_predicates_to_where_clause(&mut vtab_predicates, &mut plan, connection)?;
add_vtab_predicates_to_where_clause(
&mut vtab_predicates,
&mut plan,
connection,
param_ctx,
)?;
// Parse the actual WHERE clause and add its conditions to the plan WHERE clause that already contains the join conditions.
parse_where(
@@ -394,16 +408,18 @@ fn prepare_one_select_plan(
Some(&plan.result_columns),
&mut plan.where_clause,
connection,
param_ctx,
)?;
if let Some(mut group_by) = group_by {
for expr in group_by.exprs.iter_mut() {
replace_column_number_with_copy_of_column_expr(expr, &plan.result_columns)?;
bind_column_references(
bind_and_rewrite_expr(
expr,
&mut plan.table_references,
Some(&mut plan.table_references),
Some(&plan.result_columns),
connection,
param_ctx,
)?;
}
@@ -414,11 +430,12 @@ fn prepare_one_select_plan(
let mut predicates = vec![];
break_predicate_at_and_boundaries(&having, &mut predicates);
for expr in predicates.iter_mut() {
bind_column_references(
bind_and_rewrite_expr(
expr,
&mut plan.table_references,
Some(&mut plan.table_references),
Some(&plan.result_columns),
connection,
param_ctx,
)?;
let contains_aggregates = resolve_window_and_aggregate_functions(
schema,
@@ -452,11 +469,12 @@ fn prepare_one_select_plan(
for mut o in order_by {
replace_column_number_with_copy_of_column_expr(&mut o.expr, &plan.result_columns)?;
bind_column_references(
bind_and_rewrite_expr(
&mut o.expr,
&mut plan.table_references,
Some(&mut plan.table_references),
Some(&plan.result_columns),
connection,
param_ctx,
)?;
resolve_window_and_aggregate_functions(
schema,
@@ -471,8 +489,9 @@ fn prepare_one_select_plan(
plan.order_by = key;
// Parse the LIMIT/OFFSET clause
(plan.limit, plan.offset) =
limit.map_or(Ok((None, None)), |mut l| parse_limit(&mut l, connection))?;
(plan.limit, plan.offset) = limit.map_or(Ok((None, None)), |mut l| {
parse_limit(&mut l, connection, param_ctx)
})?;
if !windows.is_empty() {
plan_windows(schema, syms, &mut plan, table_ref_counter, &mut windows)?;
@@ -521,13 +540,15 @@ fn add_vtab_predicates_to_where_clause(
vtab_predicates: &mut Vec<Expr>,
plan: &mut SelectPlan,
connection: &Arc<Connection>,
param_ctx: &mut ParamState,
) -> Result<()> {
for expr in vtab_predicates.iter_mut() {
bind_column_references(
bind_and_rewrite_expr(
expr,
&mut plan.table_references,
Some(&mut plan.table_references),
Some(&plan.result_columns),
connection,
param_ctx,
)?;
}
for expr in vtab_predicates.drain(..) {

View File

@@ -2,6 +2,7 @@ use std::collections::HashMap;
use std::sync::Arc;
use crate::schema::{BTreeTable, Column, Type};
use crate::translate::expr::{bind_and_rewrite_expr, ParamState};
use crate::translate::optimizer::optimize_select_plan;
use crate::translate::plan::{Operation, QueryDestination, Scan, Search, SelectPlan};
use crate::translate::planner::parse_limit;
@@ -22,7 +23,7 @@ use super::plan::{
ColumnUsedMask, IterationDirection, JoinedTable, Plan, ResultSetColumn, TableReferences,
UpdatePlan,
};
use super::planner::{bind_column_references, parse_where};
use super::planner::parse_where;
/*
* Update is simple. By default we scan the table, and for each row, we check the WHERE
* clause. If it evaluates to true, we build the new record with the updated value and insert.
@@ -90,7 +91,6 @@ pub fn translate_update_for_schema_change(
}
optimize_plan(&mut plan, schema)?;
// TODO: freestyling these numbers
let opts = ProgramBuilderOpts {
num_cursors: 1,
approx_num_insns: 20,
@@ -181,11 +181,18 @@ pub fn prepare_update_plan(
.collect();
let mut set_clauses = Vec::with_capacity(body.sets.len());
let mut param_idx = ParamState::default();
// Process each SET assignment and map column names to expressions
// e.g the statement `SET x = 1, y = 2, z = 3` has 3 set assigments
for set in &mut body.sets {
bind_column_references(&mut set.expr, &mut table_references, None, connection)?;
bind_and_rewrite_expr(
&mut set.expr,
Some(&mut table_references),
None,
connection,
&mut param_idx,
)?;
let values = match set.expr.as_ref() {
Expr::Parenthesized(vals) => vals.clone(),
@@ -222,12 +229,22 @@ pub fn prepare_update_plan(
body.tbl_name.name.as_str(),
program,
connection,
&mut param_idx,
)?;
let order_by = body
.order_by
.iter()
.map(|o| (o.expr.clone(), o.order.unwrap_or(SortOrder::Asc)))
.iter_mut()
.map(|o| {
let _ = bind_and_rewrite_expr(
&mut o.expr,
Some(&mut table_references),
Some(&result_columns),
connection,
&mut param_idx,
);
(o.expr.clone(), o.order.unwrap_or(SortOrder::Asc))
})
.collect();
// Sqlite determines we should create an ephemeral table if we do not have a FROM clause
@@ -266,6 +283,7 @@ pub fn prepare_update_plan(
Some(&result_columns),
&mut where_clause,
connection,
&mut param_idx,
)?;
let table = Arc::new(BTreeTable {
@@ -342,14 +360,14 @@ pub fn prepare_update_plan(
Some(&result_columns),
&mut where_clause,
connection,
&mut param_idx,
)?;
};
// Parse the LIMIT/OFFSET clause
let (limit, offset) = body
.limit
.as_mut()
.map_or(Ok((None, None)), |l| parse_limit(l, connection))?;
let (limit, offset) = body.limit.as_mut().map_or(Ok((None, None)), |l| {
parse_limit(l, connection, &mut param_idx)
})?;
// Check what indexes will need to be updated by checking set_clauses and see
// if a column is contained in an index.

View File

@@ -42,7 +42,8 @@ pub fn translate_create_materialized_view(
// storing invalid view definitions
use crate::incremental::view::IncrementalView;
use crate::schema::BTreeTable;
let view_columns = IncrementalView::validate_and_extract_columns(select_stmt, schema)?;
let view_column_schema = IncrementalView::validate_and_extract_columns(select_stmt, schema)?;
let view_columns = view_column_schema.flat_columns();
// Reconstruct the SQL string for storage
let sql = create_materialized_view_to_str(view_name, select_stmt);

View File

@@ -1066,9 +1066,59 @@ pub fn extract_column_name_from_expr(expr: impl AsRef<ast::Expr>) -> Option<Stri
}
}
/// Information about a table referenced in a view
#[derive(Debug, Clone)]
pub struct ViewTable {
/// The full table name (potentially including database qualifier like "main.customers")
pub name: String,
/// Optional alias (e.g., "c" in "FROM customers c")
pub alias: Option<String>,
}
/// Information about a column in the view's output
#[derive(Debug, Clone)]
pub struct ViewColumn {
/// Index into ViewColumnSchema.tables indicating which table this column comes from
/// For computed columns or constants, this will be usize::MAX
pub table_index: usize,
/// The actual column definition
pub column: Column,
}
/// Schema information for a view, tracking which columns come from which tables
#[derive(Debug, Clone)]
pub struct ViewColumnSchema {
/// All tables referenced by the view (in order of appearance)
pub tables: Vec<ViewTable>,
/// The view's output columns with their table associations
pub columns: Vec<ViewColumn>,
}
impl ViewColumnSchema {
/// Get all columns as a flat vector (without table association info)
pub fn flat_columns(&self) -> Vec<Column> {
self.columns.iter().map(|vc| vc.column.clone()).collect()
}
/// Get columns that belong to a specific table
pub fn table_columns(&self, table_index: usize) -> Vec<Column> {
self.columns
.iter()
.filter(|vc| vc.table_index == table_index)
.map(|vc| vc.column.clone())
.collect()
}
}
/// Extract column information from a SELECT statement for view creation
pub fn extract_view_columns(select_stmt: &ast::Select, schema: &Schema) -> Vec<Column> {
pub fn extract_view_columns(
select_stmt: &ast::Select,
schema: &Schema,
) -> Result<ViewColumnSchema> {
let mut tables = Vec::new();
let mut columns = Vec::new();
let mut column_name_counts: HashMap<String, usize> = HashMap::new();
// Navigate to the first SELECT in the statement
if let ast::OneSelect::Select {
ref from,
@@ -1076,23 +1126,85 @@ pub fn extract_view_columns(select_stmt: &ast::Select, schema: &Schema) -> Vec<C
..
} = &select_stmt.body.select
{
// First, we need to figure out which table(s) are being selected from
let table_name = if let Some(from) = from {
if let ast::SelectTable::Table(qualified_name, _, _) = from.select.as_ref() {
Some(normalize_ident(qualified_name.name.as_str()))
} else {
None
// First, extract all tables (from FROM clause and JOINs)
if let Some(from) = from {
// Add the main table from FROM clause
match from.select.as_ref() {
ast::SelectTable::Table(qualified_name, alias, _) => {
let table_name = if qualified_name.db_name.is_some() {
// Include database qualifier if present
qualified_name.to_string()
} else {
normalize_ident(qualified_name.name.as_str())
};
tables.push(ViewTable {
name: table_name.clone(),
alias: alias.as_ref().map(|a| match a {
ast::As::As(name) => normalize_ident(name.as_str()),
ast::As::Elided(name) => normalize_ident(name.as_str()),
}),
});
}
_ => {
// Handle other types like subqueries if needed
}
}
} else {
None
// Add tables from JOINs
for join in &from.joins {
match join.table.as_ref() {
ast::SelectTable::Table(qualified_name, alias, _) => {
let table_name = if qualified_name.db_name.is_some() {
// Include database qualifier if present
qualified_name.to_string()
} else {
normalize_ident(qualified_name.name.as_str())
};
tables.push(ViewTable {
name: table_name.clone(),
alias: alias.as_ref().map(|a| match a {
ast::As::As(name) => normalize_ident(name.as_str()),
ast::As::Elided(name) => normalize_ident(name.as_str()),
}),
});
}
_ => {
// Handle other types like subqueries if needed
}
}
}
}
// Helper function to find table index by name or alias
let find_table_index = |name: &str| -> Option<usize> {
tables
.iter()
.position(|t| t.name == name || t.alias.as_ref().is_some_and(|a| a == name))
};
// Get the table for column resolution
let _table = table_name.as_ref().and_then(|name| schema.get_table(name));
// Process each column in the SELECT list
for (i, result_col) in select_columns.iter().enumerate() {
for result_col in select_columns.iter() {
match result_col {
ast::ResultColumn::Expr(expr, alias) => {
let name = alias
// Figure out which table this expression comes from
let table_index = match expr.as_ref() {
ast::Expr::Qualified(table_ref, _col_name) => {
// Column qualified with table name
find_table_index(table_ref.as_str())
}
ast::Expr::Id(_col_name) => {
// Unqualified column - would need to resolve based on schema
// For now, assume it's from the first table if there is one
if !tables.is_empty() {
Some(0)
} else {
None
}
}
_ => None, // Expression, literal, etc.
};
let col_name = alias
.as_ref()
.map(|a| match a {
ast::As::Elided(name) => name.as_str().to_string(),
@@ -1103,41 +1215,65 @@ pub fn extract_view_columns(select_stmt: &ast::Select, schema: &Schema) -> Vec<C
// If we can't extract a simple column name, use the expression itself
expr.to_string()
});
columns.push(Column {
name: Some(name),
ty: Type::Text, // Default to TEXT, could be refined with type analysis
ty_str: "TEXT".to_string(),
primary_key: false, // Views don't have primary keys
is_rowid_alias: false,
notnull: false, // Views typically don't enforce NOT NULL
default: None, // Views don't have default values
unique: false,
collation: None,
hidden: false,
columns.push(ViewColumn {
table_index: table_index.unwrap_or(usize::MAX),
column: Column {
name: Some(col_name),
ty: Type::Text, // Default to TEXT, could be refined with type analysis
ty_str: "TEXT".to_string(),
primary_key: false,
is_rowid_alias: false,
notnull: false,
default: None,
unique: false,
collation: None,
hidden: false,
},
});
}
ast::ResultColumn::Star => {
// For SELECT *, expand to all columns from the table
if let Some(ref table_name) = table_name {
if let Some(table) = schema.get_table(table_name) {
// Copy all columns from the table, but adjust for view constraints
for table_column in table.columns() {
columns.push(Column {
name: table_column.name.clone(),
ty: table_column.ty,
ty_str: table_column.ty_str.clone(),
primary_key: false, // Views don't have primary keys
is_rowid_alias: false,
notnull: false, // Views typically don't enforce NOT NULL
default: None, // Views don't have default values
unique: false,
collation: table_column.collation,
hidden: false,
// For SELECT *, expand to all columns from all tables
for (table_idx, table) in tables.iter().enumerate() {
if let Some(table_obj) = schema.get_table(&table.name) {
for table_column in table_obj.columns() {
let col_name =
table_column.name.clone().unwrap_or_else(|| "?".to_string());
// Handle duplicate column names by adding suffix
let final_name =
if let Some(count) = column_name_counts.get_mut(&col_name) {
*count += 1;
format!("{}:{}", col_name, *count - 1)
} else {
column_name_counts.insert(col_name.clone(), 1);
col_name.clone()
};
columns.push(ViewColumn {
table_index: table_idx,
column: Column {
name: Some(final_name),
ty: table_column.ty,
ty_str: table_column.ty_str.clone(),
primary_key: false,
is_rowid_alias: false,
notnull: false,
default: None,
unique: false,
collation: table_column.collation,
hidden: false,
},
});
}
} else {
// Table not found, create placeholder
columns.push(Column {
}
}
// If no tables, create a placeholder
if tables.is_empty() {
columns.push(ViewColumn {
table_index: usize::MAX,
column: Column {
name: Some("*".to_string()),
ty: Type::Text,
ty_str: "TEXT".to_string(),
@@ -1148,63 +1284,70 @@ pub fn extract_view_columns(select_stmt: &ast::Select, schema: &Schema) -> Vec<C
unique: false,
collation: None,
hidden: false,
});
}
} else {
// No FROM clause or couldn't determine table, create placeholder
columns.push(Column {
name: Some("*".to_string()),
ty: Type::Text,
ty_str: "TEXT".to_string(),
primary_key: false,
is_rowid_alias: false,
notnull: false,
default: None,
unique: false,
collation: None,
hidden: false,
},
});
}
}
ast::ResultColumn::TableStar(table_name) => {
ast::ResultColumn::TableStar(table_ref) => {
// For table.*, expand to all columns from the specified table
let table_name_str = normalize_ident(table_name.as_str());
if let Some(table) = schema.get_table(&table_name_str) {
// Copy all columns from the table, but adjust for view constraints
for table_column in table.columns() {
columns.push(Column {
name: table_column.name.clone(),
ty: table_column.ty,
ty_str: table_column.ty_str.clone(),
primary_key: false,
is_rowid_alias: false,
notnull: false,
default: None,
unique: false,
collation: table_column.collation,
hidden: false,
let table_name_str = normalize_ident(table_ref.as_str());
if let Some(table_idx) = find_table_index(&table_name_str) {
if let Some(table) = schema.get_table(&tables[table_idx].name) {
for table_column in table.columns() {
let col_name =
table_column.name.clone().unwrap_or_else(|| "?".to_string());
// Handle duplicate column names by adding suffix
let final_name =
if let Some(count) = column_name_counts.get_mut(&col_name) {
*count += 1;
format!("{}:{}", col_name, *count - 1)
} else {
column_name_counts.insert(col_name.clone(), 1);
col_name.clone()
};
columns.push(ViewColumn {
table_index: table_idx,
column: Column {
name: Some(final_name),
ty: table_column.ty,
ty_str: table_column.ty_str.clone(),
primary_key: false,
is_rowid_alias: false,
notnull: false,
default: None,
unique: false,
collation: table_column.collation,
hidden: false,
},
});
}
} else {
// Table not found, create placeholder
columns.push(ViewColumn {
table_index: usize::MAX,
column: Column {
name: Some(format!("{table_name_str}.*")),
ty: Type::Text,
ty_str: "TEXT".to_string(),
primary_key: false,
is_rowid_alias: false,
notnull: false,
default: None,
unique: false,
collation: None,
hidden: false,
},
});
}
} else {
// Table not found, create placeholder
columns.push(Column {
name: Some(format!("{table_name_str}.*")),
ty: Type::Text,
ty_str: "TEXT".to_string(),
primary_key: false,
is_rowid_alias: false,
notnull: false,
default: None,
unique: false,
collation: None,
hidden: false,
});
}
}
}
}
}
columns
Ok(ViewColumnSchema { tables, columns })
}
#[cfg(test)]

View File

@@ -2276,16 +2276,8 @@ pub fn op_transaction_inner(
if matches!(new_transaction_state, TransactionState::Write { .. })
&& matches!(actual_tx_mode, TransactionMode::Write)
{
let (tx_id, mv_tx_mode) = program.connection.mv_tx.get().unwrap();
if mv_tx_mode == TransactionMode::Read {
return_if_io!(
mv_store.upgrade_to_exclusive_tx(pager.clone(), Some(tx_id))
);
} else {
return_if_io!(
mv_store.begin_exclusive_tx(pager.clone(), Some(tx_id))
);
}
let (tx_id, _) = program.connection.mv_tx.get().unwrap();
return_if_io!(mv_store.begin_exclusive_tx(pager.clone(), Some(tx_id)));
}
}
} else {

View File

@@ -1312,6 +1312,8 @@ pub enum PragmaName {
ApplicationId,
/// set the autovacuum mode
AutoVacuum,
/// set the busy_timeout (see https://www.sqlite.org/pragma.html#pragma_busy_timeout)
BusyTimeout,
/// `cache_size` pragma
CacheSize,
/// encryption cipher algorithm name for encrypted databases

View File

@@ -749,7 +749,7 @@ impl<'a> Parser<'a> {
fn parse_create_materialized_view(&mut self) -> Result<Stmt> {
eat_assert!(self, TK_MATERIALIZED);
eat_assert!(self, TK_VIEW);
eat_expect!(self, TK_VIEW);
let if_not_exists = self.parse_if_not_exists()?;
let view_name = self.parse_fullname(false)?;
let columns = self.parse_eid_list(false)?;

View File

@@ -0,0 +1,17 @@
[package]
name = "encryption-throughput"
version = "0.1.0"
edition = "2021"
[[bin]]
name = "encryption-throughput"
path = "src/main.rs"
[dependencies]
turso = { workspace = true, features = ["encryption"] }
clap = { workspace = true, features = ["derive"] }
tokio = { workspace = true, default-features = true, features = ["full"] }
futures = { workspace = true }
tracing-subscriber = { workspace = true }
rand = { workspace = true, features = ["small_rng"] }
hex = { workspace = true }

28
perf/encryption/README.md Normal file
View File

@@ -0,0 +1,28 @@
# Encryption Throughput Benchmarking
```shell
$ cargo run --release -- --help
Usage: encryption-throughput [OPTIONS]
Options:
-t, --threads <THREADS> [default: 1]
-b, --batch-size <BATCH_SIZE> [default: 100]
-i, --iterations <ITERATIONS> [default: 10]
-r, --read-ratio <READ_RATIO> Percentage of operations that should be reads (0-100)
-w, --write-ratio <WRITE_RATIO> Percentage of operations that should be writes (0-100)
--encryption Enable database encryption
--cipher <CIPHER> Encryption cipher to use (only relevant if --encryption is set) [default: aegis-256]
--think <THINK> Per transaction think time (ms) [default: 0]
--timeout <TIMEOUT> Busy timeout in milliseconds [default: 30000]
--seed <SEED> Random seed for reproducible workloads [default: 2167532792061351037]
-h, --help Print help
```
```shell
# try these:
cargo run --release -- -b 100 -i 25000 --read-ratio 75
cargo run --release -- -b 100 -i 25000 --read-ratio 75 --encryption
```

457
perf/encryption/src/main.rs Normal file
View File

@@ -0,0 +1,457 @@
use clap::Parser;
use rand::rngs::SmallRng;
use rand::{Rng, RngCore, SeedableRng};
use std::sync::atomic::{AtomicU64, Ordering};
use std::sync::{Arc, Barrier};
use std::time::{Duration, Instant};
use turso::{Builder, Database, Result};
#[derive(Debug, Clone)]
struct EncryptionOpts {
cipher: String,
hexkey: String,
}
#[derive(Parser)]
#[command(name = "encryption-throughput")]
#[command(about = "Encryption throughput benchmark on Turso DB")]
struct Args {
/// More than one thread does not work yet
#[arg(short = 't', long = "threads", default_value = "1")]
threads: usize,
/// the number operations per transaction
#[arg(short = 'b', long = "batch-size", default_value = "100")]
batch_size: usize,
/// number of transactions per thread
#[arg(short = 'i', long = "iterations", default_value = "10")]
iterations: usize,
#[arg(
short = 'r',
long = "read-ratio",
help = "Percentage of operations that should be reads (0-100)"
)]
read_ratio: Option<u8>,
#[arg(
short = 'w',
long = "write-ratio",
help = "Percentage of operations that should be writes (0-100)"
)]
write_ratio: Option<u8>,
#[arg(
long = "encryption",
action = clap::ArgAction::SetTrue,
help = "Enable database encryption"
)]
encryption: bool,
#[arg(
long = "cipher",
default_value = "aegis-256",
help = "Encryption cipher to use (only relevant if --encryption is set)"
)]
cipher: String,
#[arg(
long = "think",
default_value = "0",
help = "Per transaction think time (ms)"
)]
think: u64,
#[arg(
long = "timeout",
default_value = "30000",
help = "Busy timeout in milliseconds"
)]
timeout: u64,
#[arg(
long = "seed",
default_value = "2167532792061351037",
help = "Random seed for reproducible workloads"
)]
seed: u64,
}
#[derive(Debug)]
struct WorkerStats {
transactions_completed: u64,
reads_completed: u64,
writes_completed: u64,
reads_found: u64,
reads_not_found: u64,
total_transaction_time: Duration,
}
#[derive(Debug, Clone)]
struct SharedState {
max_inserted_id: Arc<AtomicU64>,
}
#[tokio::main]
async fn main() -> Result<()> {
let _ = tracing_subscriber::fmt::try_init();
let args = Args::parse();
let read_ratio = match (args.read_ratio, args.write_ratio) {
(Some(_), Some(_)) => {
eprintln!("Error: Cannot specify both --read-ratio and --write-ratio");
std::process::exit(1);
}
(Some(r), None) => {
if r > 100 {
eprintln!("Error: read-ratio must be between 0 and 100");
std::process::exit(1);
}
r
}
(None, Some(w)) => {
if w > 100 {
eprintln!("Error: write-ratio must be between 0 and 100");
std::process::exit(1);
}
100 - w
}
// lets default to 0% reads (100% writes)
(None, None) => 0,
};
println!(
"Running encryption throughput benchmark with {} threads, {} batch size, {} iterations",
args.threads, args.batch_size, args.iterations
);
println!(
"Read/Write ratio: {}% reads, {}% writes",
read_ratio,
100 - read_ratio
);
println!("Encryption enabled: {}", args.encryption);
println!("Random seed: {}", args.seed);
let encryption_opts = if args.encryption {
let mut key_rng = SmallRng::seed_from_u64(args.seed);
let key_size = get_key_size_for_cipher(&args.cipher);
let mut key = vec![0u8; key_size];
key_rng.fill_bytes(&mut key);
let config = EncryptionOpts {
cipher: args.cipher.clone(),
hexkey: hex::encode(&key),
};
println!("Cipher: {}", config.cipher);
println!("Hexkey: {}", config.hexkey);
Some(config)
} else {
None
};
let db_path = "encryption_throughput_test.db";
if std::path::Path::new(db_path).exists() {
std::fs::remove_file(db_path).expect("Failed to remove existing database");
}
let wal_path = "encryption_throughput_test.db-wal";
if std::path::Path::new(wal_path).exists() {
std::fs::remove_file(wal_path).expect("Failed to remove existing WAL file");
}
let db = setup_database(db_path, &encryption_opts).await?;
// for create a var which is shared between all the threads, this we use to track the
// max inserted id so that we only read these
let shared_state = SharedState {
max_inserted_id: Arc::new(AtomicU64::new(0)),
};
let start_barrier = Arc::new(Barrier::new(args.threads));
let mut handles = Vec::new();
let timeout = Duration::from_millis(args.timeout);
let overall_start = Instant::now();
for thread_id in 0..args.threads {
let db_clone = db.clone();
let barrier = Arc::clone(&start_barrier);
let encryption_opts_clone = encryption_opts.clone();
let shared_state_clone = shared_state.clone();
let handle = tokio::task::spawn(worker_thread(
thread_id,
db_clone,
args.batch_size,
args.iterations,
barrier,
read_ratio,
encryption_opts_clone,
args.think,
timeout,
shared_state_clone,
args.seed,
));
handles.push(handle);
}
let mut total_transactions = 0;
let mut total_reads = 0;
let mut total_writes = 0;
let mut total_reads_found = 0;
let mut total_reads_not_found = 0;
for (idx, handle) in handles.into_iter().enumerate() {
match handle.await {
Ok(Ok(stats)) => {
total_transactions += stats.transactions_completed;
total_reads += stats.reads_completed;
total_writes += stats.writes_completed;
total_reads_found += stats.reads_found;
total_reads_not_found += stats.reads_not_found;
}
Ok(Err(e)) => {
eprintln!("Thread error {idx}: {e}");
return Err(e);
}
Err(_) => {
eprintln!("Thread panicked");
std::process::exit(1);
}
}
}
let overall_elapsed = overall_start.elapsed();
let total_operations = total_reads + total_writes;
let transaction_throughput = (total_transactions as f64) / overall_elapsed.as_secs_f64();
let operation_throughput = (total_operations as f64) / overall_elapsed.as_secs_f64();
let read_throughput = if total_reads > 0 {
(total_reads as f64) / overall_elapsed.as_secs_f64()
} else {
0.0
};
let write_throughput = if total_writes > 0 {
(total_writes as f64) / overall_elapsed.as_secs_f64()
} else {
0.0
};
let avg_ops_per_txn = (total_operations as f64) / (total_transactions as f64);
println!("\n=== BENCHMARK RESULTS ===");
println!("Total transactions: {total_transactions}");
println!("Total operations: {total_operations}");
println!("Operations per transaction: {avg_ops_per_txn:.1}");
println!("Total time: {:.2}s", overall_elapsed.as_secs_f64());
println!();
println!("Transaction throughput: {transaction_throughput:.2} txns/sec");
println!("Operation throughput: {operation_throughput:.2} ops/sec");
// not found should be zero since track the max inserted id
// todo(v): probably handle the not found error and remove max id
if total_reads > 0 {
println!(
" - Read operations: {total_reads} ({total_reads_found} found, {total_reads_not_found} not found)"
);
println!(" - Read throughput: {read_throughput:.2} reads/sec");
}
if total_writes > 0 {
println!(" - Write operations: {total_writes}");
println!(" - Write throughput: {write_throughput:.2} writes/sec");
}
println!("\nConfiguration:");
println!("Threads: {}", args.threads);
println!("Batch size: {}", args.batch_size);
println!("Iterations per thread: {}", args.iterations);
println!("Encryption: {}", args.encryption);
println!("Seed: {}", args.seed);
if let Ok(metadata) = std::fs::metadata(db_path) {
println!("Database file size: {} bytes", metadata.len());
}
Ok(())
}
fn get_key_size_for_cipher(cipher: &str) -> usize {
match cipher.to_lowercase().as_str() {
"aes-128-gcm" | "aegis-128l" | "aegis-128x2" | "aegis-128x4" => 16,
"aes-256-gcm" | "aegis-256" | "aegis-256x2" | "aegis-256x4" => 32,
_ => 32, // default to 256-bit key
}
}
async fn setup_database(
db_path: &str,
encryption_opts: &Option<EncryptionOpts>,
) -> Result<Database> {
let builder = Builder::new_local(db_path);
let db = builder.build().await?;
let conn = db.connect()?;
if let Some(config) = encryption_opts {
conn.execute(&format!("PRAGMA cipher='{}'", config.cipher), ())
.await?;
conn.execute(&format!("PRAGMA hexkey='{}'", config.hexkey), ())
.await?;
}
// todo(v): probably store blobs and then have option of randomblob size
conn.execute(
"CREATE TABLE IF NOT EXISTS test_table (
id INTEGER PRIMARY KEY,
data TEXT NOT NULL
)",
(),
)
.await?;
println!("Database created at: {db_path}");
Ok(db)
}
#[allow(clippy::too_many_arguments)]
async fn worker_thread(
thread_id: usize,
db: Database,
batch_size: usize,
iterations: usize,
start_barrier: Arc<Barrier>,
read_ratio: u8,
encryption_opts: Option<EncryptionOpts>,
think_ms: u64,
timeout: Duration,
shared_state: SharedState,
base_seed: u64,
) -> Result<WorkerStats> {
start_barrier.wait();
let start_time = Instant::now();
let mut stats = WorkerStats {
transactions_completed: 0,
reads_completed: 0,
writes_completed: 0,
reads_found: 0,
reads_not_found: 0,
total_transaction_time: Duration::ZERO,
};
let thread_seed = base_seed.wrapping_add(thread_id as u64);
let mut rng = SmallRng::seed_from_u64(thread_seed);
for iteration in 0..iterations {
let conn = db.connect()?;
if let Some(config) = &encryption_opts {
conn.execute(&format!("PRAGMA cipher='{}'", config.cipher), ())
.await?;
conn.execute(&format!("PRAGMA hexkey='{}'", config.hexkey), ())
.await?;
}
conn.busy_timeout(Some(timeout))?;
let mut insert_stmt = conn
.prepare("INSERT INTO test_table (id, data) VALUES (?, ?)")
.await?;
let transaction_start = Instant::now();
conn.execute("BEGIN", ()).await?;
for i in 0..batch_size {
let should_read = rng.random_range(0..100) < read_ratio;
if should_read {
// only attempt reads if we have inserted some data
let max_id = shared_state.max_inserted_id.load(Ordering::Relaxed);
if max_id > 0 {
let read_id = rng.random_range(1..=max_id);
let row = conn
.query(
"SELECT data FROM test_table WHERE id = ?",
turso::params::Params::Positional(vec![turso::Value::Integer(
read_id as i64,
)]),
)
.await;
match row {
Ok(_) => stats.reads_found += 1,
Err(turso::Error::QueryReturnedNoRows) => stats.reads_not_found += 1,
Err(e) => return Err(e),
};
stats.reads_completed += 1;
} else {
// if no data inserted yet, convert to a write
let id = thread_id * iterations * batch_size + iteration * batch_size + i + 1;
insert_stmt
.execute(turso::params::Params::Positional(vec![
turso::Value::Integer(id as i64),
turso::Value::Text(format!("data_{id}")),
]))
.await?;
shared_state
.max_inserted_id
.fetch_max(id as u64, Ordering::Relaxed);
stats.writes_completed += 1;
}
} else {
let id = thread_id * iterations * batch_size + iteration * batch_size + i + 1;
insert_stmt
.execute(turso::params::Params::Positional(vec![
turso::Value::Integer(id as i64),
turso::Value::Text(format!("data_{id}")),
]))
.await?;
shared_state
.max_inserted_id
.fetch_max(id as u64, Ordering::Relaxed);
stats.writes_completed += 1;
}
}
if think_ms > 0 {
tokio::time::sleep(Duration::from_millis(think_ms)).await;
}
conn.execute("COMMIT", ()).await?;
let transaction_elapsed = transaction_start.elapsed();
stats.transactions_completed += 1;
stats.total_transaction_time += transaction_elapsed;
}
let elapsed = start_time.elapsed();
let total_ops = stats.reads_completed + stats.writes_completed;
let transaction_throughput = (stats.transactions_completed as f64) / elapsed.as_secs_f64();
let operation_throughput = (total_ops as f64) / elapsed.as_secs_f64();
let avg_txn_latency =
stats.total_transaction_time.as_secs_f64() * 1000.0 / stats.transactions_completed as f64;
println!(
"Thread {}: {} txns ({} ops: {} reads, {} writes) in {:.2}s ({:.2} txns/sec, {:.2} ops/sec, {:.2}ms avg latency)",
thread_id,
stats.transactions_completed,
total_ops,
stats.reads_completed,
stats.writes_completed,
elapsed.as_secs_f64(),
transaction_throughput,
operation_throughput,
avg_txn_latency
);
if stats.reads_completed > 0 {
println!(
" Thread {} reads: {} found, {} not found",
thread_id, stats.reads_found, stats.reads_not_found
);
}
Ok(stats)
}

View File

@@ -44,13 +44,13 @@ do_execsql_test_on_specific_db {:memory:} matview-aggregation-population {
do_execsql_test_on_specific_db {:memory:} matview-filter-with-groupby {
CREATE TABLE t(a INTEGER, b INTEGER);
INSERT INTO t(a,b) VALUES (2,2), (3,3), (6,6), (7,7);
CREATE MATERIALIZED VIEW v AS
SELECT b as yourb, SUM(a) as mysum, COUNT(a) as mycount
FROM t
WHERE b > 2
GROUP BY b;
SELECT * FROM v ORDER BY yourb;
} {3|3|1
6|6|1
@@ -63,13 +63,13 @@ do_execsql_test_on_specific_db {:memory:} matview-insert-maintenance {
FROM t
WHERE b > 2
GROUP BY b;
INSERT INTO t VALUES (3,3), (6,6);
SELECT * FROM v ORDER BY b;
INSERT INTO t VALUES (4,3), (5,6);
SELECT * FROM v ORDER BY b;
INSERT INTO t VALUES (1,1), (2,2);
SELECT * FROM v ORDER BY b;
} {3|3|1
@@ -87,17 +87,17 @@ do_execsql_test_on_specific_db {:memory:} matview-delete-maintenance {
(3, 'A', 30),
(4, 'B', 40),
(5, 'A', 50);
CREATE MATERIALIZED VIEW category_sums AS
SELECT category, SUM(amount) as total, COUNT(*) as cnt
FROM items
GROUP BY category;
SELECT * FROM category_sums ORDER BY category;
DELETE FROM items WHERE id = 3;
SELECT * FROM category_sums ORDER BY category;
DELETE FROM items WHERE category = 'B';
SELECT * FROM category_sums ORDER BY category;
} {A|90|3
@@ -113,17 +113,17 @@ do_execsql_test_on_specific_db {:memory:} matview-update-maintenance {
(2, 200, 2),
(3, 300, 1),
(4, 400, 2);
CREATE MATERIALIZED VIEW status_totals AS
SELECT status, SUM(value) as total, COUNT(*) as cnt
FROM records
GROUP BY status;
SELECT * FROM status_totals ORDER BY status;
UPDATE records SET value = 150 WHERE id = 1;
SELECT * FROM status_totals ORDER BY status;
UPDATE records SET status = 2 WHERE id = 3;
SELECT * FROM status_totals ORDER BY status;
} {1|400|2
@@ -136,10 +136,10 @@ do_execsql_test_on_specific_db {:memory:} matview-update-maintenance {
do_execsql_test_on_specific_db {:memory:} matview-integer-primary-key-basic {
CREATE TABLE t(a INTEGER PRIMARY KEY, b INTEGER);
INSERT INTO t(a,b) VALUES (2,2), (3,3), (6,6), (7,7);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 2;
SELECT * FROM v ORDER BY a;
} {3|3
6|6
@@ -148,15 +148,15 @@ do_execsql_test_on_specific_db {:memory:} matview-integer-primary-key-basic {
do_execsql_test_on_specific_db {:memory:} matview-integer-primary-key-update-rowid {
CREATE TABLE t(a INTEGER PRIMARY KEY, b INTEGER);
INSERT INTO t(a,b) VALUES (2,2), (3,3), (6,6), (7,7);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 2;
SELECT * FROM v ORDER BY a;
UPDATE t SET a = 1 WHERE b = 3;
SELECT * FROM v ORDER BY a;
UPDATE t SET a = 10 WHERE a = 6;
SELECT * FROM v ORDER BY a;
} {3|3
@@ -172,15 +172,15 @@ do_execsql_test_on_specific_db {:memory:} matview-integer-primary-key-update-row
do_execsql_test_on_specific_db {:memory:} matview-integer-primary-key-update-value {
CREATE TABLE t(a INTEGER PRIMARY KEY, b INTEGER);
INSERT INTO t(a,b) VALUES (2,2), (3,3), (6,6), (7,7);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 2;
SELECT * FROM v ORDER BY a;
UPDATE t SET b = 1 WHERE a = 6;
SELECT * FROM v ORDER BY a;
UPDATE t SET b = 5 WHERE a = 2;
SELECT * FROM v ORDER BY a;
} {3|3
@@ -200,18 +200,18 @@ do_execsql_test_on_specific_db {:memory:} matview-integer-primary-key-with-aggre
(3, 20, 300),
(4, 20, 400),
(5, 10, 500);
CREATE MATERIALIZED VIEW v AS
SELECT b, SUM(c) as total, COUNT(*) as cnt
FROM t
WHERE a > 2
GROUP BY b;
SELECT * FROM v ORDER BY b;
UPDATE t SET a = 6 WHERE a = 1;
SELECT * FROM v ORDER BY b;
DELETE FROM t WHERE a = 3;
SELECT * FROM v ORDER BY b;
} {10|500|1
@@ -228,7 +228,7 @@ do_execsql_test_on_specific_db {:memory:} matview-complex-filter-aggregation {
amount INTEGER,
type INTEGER
);
INSERT INTO transactions VALUES
(1, 100, 50, 1),
(2, 100, 30, 2),
@@ -236,21 +236,21 @@ do_execsql_test_on_specific_db {:memory:} matview-complex-filter-aggregation {
(4, 100, 20, 1),
(5, 200, 40, 2),
(6, 300, 60, 1);
CREATE MATERIALIZED VIEW account_deposits AS
SELECT account, SUM(amount) as total_deposits, COUNT(*) as deposit_count
FROM transactions
WHERE type = 1
GROUP BY account;
SELECT * FROM account_deposits ORDER BY account;
INSERT INTO transactions VALUES (7, 100, 25, 1);
SELECT * FROM account_deposits ORDER BY account;
UPDATE transactions SET amount = 80 WHERE id = 1;
SELECT * FROM account_deposits ORDER BY account;
DELETE FROM transactions WHERE id = 3;
SELECT * FROM account_deposits ORDER BY account;
} {100|70|2
@@ -273,19 +273,19 @@ do_execsql_test_on_specific_db {:memory:} matview-sum-count-only {
(3, 30, 2),
(4, 40, 2),
(5, 50, 1);
CREATE MATERIALIZED VIEW category_stats AS
SELECT category,
SUM(value) as sum_val,
COUNT(*) as cnt
FROM data
GROUP BY category;
SELECT * FROM category_stats ORDER BY category;
INSERT INTO data VALUES (6, 5, 1);
SELECT * FROM category_stats ORDER BY category;
UPDATE data SET value = 35 WHERE id = 3;
SELECT * FROM category_stats ORDER BY category;
} {1|80|3
@@ -302,9 +302,9 @@ do_execsql_test_on_specific_db {:memory:} matview-empty-table-population {
FROM t
WHERE b > 5
GROUP BY b;
SELECT COUNT(*) FROM v;
INSERT INTO t VALUES (1, 3), (2, 7), (3, 9);
SELECT * FROM v ORDER BY b;
} {0
@@ -314,15 +314,15 @@ do_execsql_test_on_specific_db {:memory:} matview-empty-table-population {
do_execsql_test_on_specific_db {:memory:} matview-all-rows-filtered {
CREATE TABLE t(a INTEGER, b INTEGER);
INSERT INTO t VALUES (1, 1), (2, 2), (3, 3);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 10;
SELECT COUNT(*) FROM v;
INSERT INTO t VALUES (11, 11);
SELECT * FROM v;
UPDATE t SET b = 1 WHERE a = 11;
SELECT COUNT(*) FROM v;
} {0
@@ -335,26 +335,26 @@ do_execsql_test_on_specific_db {:memory:} matview-mixed-operations-sequence {
customer_id INTEGER,
amount INTEGER
);
INSERT INTO orders VALUES (1, 100, 50);
INSERT INTO orders VALUES (2, 200, 75);
CREATE MATERIALIZED VIEW customer_totals AS
SELECT customer_id, SUM(amount) as total, COUNT(*) as order_count
FROM orders
GROUP BY customer_id;
SELECT * FROM customer_totals ORDER BY customer_id;
INSERT INTO orders VALUES (3, 100, 25);
SELECT * FROM customer_totals ORDER BY customer_id;
UPDATE orders SET amount = 100 WHERE order_id = 2;
SELECT * FROM customer_totals ORDER BY customer_id;
DELETE FROM orders WHERE order_id = 1;
SELECT * FROM customer_totals ORDER BY customer_id;
INSERT INTO orders VALUES (4, 300, 150);
SELECT * FROM customer_totals ORDER BY customer_id;
} {100|50|1
@@ -389,17 +389,17 @@ do_execsql_test_on_specific_db {:memory:} matview-projections {
do_execsql_test_on_specific_db {:memory:} matview-rollback-insert {
CREATE TABLE t(a INTEGER, b INTEGER);
INSERT INTO t VALUES (1, 10), (2, 20), (3, 30);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 15;
SELECT * FROM v ORDER BY a;
BEGIN;
INSERT INTO t VALUES (4, 40), (5, 50);
SELECT * FROM v ORDER BY a;
ROLLBACK;
SELECT * FROM v ORDER BY a;
} {2|20
3|30
@@ -413,17 +413,17 @@ do_execsql_test_on_specific_db {:memory:} matview-rollback-insert {
do_execsql_test_on_specific_db {:memory:} matview-rollback-delete {
CREATE TABLE t(a INTEGER, b INTEGER);
INSERT INTO t VALUES (1, 10), (2, 20), (3, 30), (4, 40);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 15;
SELECT * FROM v ORDER BY a;
BEGIN;
DELETE FROM t WHERE a IN (2, 3);
SELECT * FROM v ORDER BY a;
ROLLBACK;
SELECT * FROM v ORDER BY a;
} {2|20
3|30
@@ -436,18 +436,18 @@ do_execsql_test_on_specific_db {:memory:} matview-rollback-delete {
do_execsql_test_on_specific_db {:memory:} matview-rollback-update {
CREATE TABLE t(a INTEGER, b INTEGER);
INSERT INTO t VALUES (1, 10), (2, 20), (3, 30);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 15;
SELECT * FROM v ORDER BY a;
BEGIN;
UPDATE t SET b = 5 WHERE a = 2;
UPDATE t SET b = 35 WHERE a = 1;
SELECT * FROM v ORDER BY a;
ROLLBACK;
SELECT * FROM v ORDER BY a;
} {2|20
3|30
@@ -459,19 +459,19 @@ do_execsql_test_on_specific_db {:memory:} matview-rollback-update {
do_execsql_test_on_specific_db {:memory:} matview-rollback-aggregation {
CREATE TABLE sales(product_id INTEGER, amount INTEGER);
INSERT INTO sales VALUES (1, 100), (1, 200), (2, 150), (2, 250);
CREATE MATERIALIZED VIEW product_totals AS
SELECT product_id, SUM(amount) as total, COUNT(*) as cnt
FROM sales
GROUP BY product_id;
SELECT * FROM product_totals ORDER BY product_id;
BEGIN;
INSERT INTO sales VALUES (1, 50), (3, 300);
SELECT * FROM product_totals ORDER BY product_id;
ROLLBACK;
SELECT * FROM product_totals ORDER BY product_id;
} {1|300|2
2|400|2
@@ -484,21 +484,21 @@ do_execsql_test_on_specific_db {:memory:} matview-rollback-aggregation {
do_execsql_test_on_specific_db {:memory:} matview-rollback-mixed-operations {
CREATE TABLE orders(id INTEGER PRIMARY KEY, customer INTEGER, amount INTEGER);
INSERT INTO orders VALUES (1, 100, 50), (2, 200, 75), (3, 100, 25);
CREATE MATERIALIZED VIEW customer_totals AS
SELECT customer, SUM(amount) as total, COUNT(*) as cnt
FROM orders
GROUP BY customer;
SELECT * FROM customer_totals ORDER BY customer;
BEGIN;
INSERT INTO orders VALUES (4, 100, 100);
UPDATE orders SET amount = 150 WHERE id = 2;
DELETE FROM orders WHERE id = 3;
SELECT * FROM customer_totals ORDER BY customer;
ROLLBACK;
SELECT * FROM customer_totals ORDER BY customer;
} {100|75|2
200|75|1
@@ -514,22 +514,22 @@ do_execsql_test_on_specific_db {:memory:} matview-rollback-filtered-aggregation
(2, 100, 30, 'withdraw'),
(3, 200, 100, 'deposit'),
(4, 200, 40, 'withdraw');
CREATE MATERIALIZED VIEW deposits AS
SELECT account, SUM(amount) as total_deposits, COUNT(*) as cnt
FROM transactions
WHERE type = 'deposit'
GROUP BY account;
SELECT * FROM deposits ORDER BY account;
BEGIN;
INSERT INTO transactions VALUES (5, 100, 75, 'deposit');
UPDATE transactions SET amount = 60 WHERE id = 1;
DELETE FROM transactions WHERE id = 3;
SELECT * FROM deposits ORDER BY account;
ROLLBACK;
SELECT * FROM deposits ORDER BY account;
} {100|50|1
200|100|1
@@ -540,12 +540,12 @@ do_execsql_test_on_specific_db {:memory:} matview-rollback-filtered-aggregation
do_execsql_test_on_specific_db {:memory:} matview-rollback-empty-view {
CREATE TABLE t(a INTEGER, b INTEGER);
INSERT INTO t VALUES (1, 5), (2, 8);
CREATE MATERIALIZED VIEW v AS
SELECT * FROM t WHERE b > 10;
SELECT COUNT(*) FROM v;
BEGIN;
INSERT INTO t VALUES (3, 15), (4, 20);
SELECT * FROM v ORDER BY a;
@@ -556,3 +556,538 @@ do_execsql_test_on_specific_db {:memory:} matview-rollback-empty-view {
3|15
4|20
0}
# Join tests for materialized views
do_execsql_test_on_specific_db {:memory:} matview-simple-join {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT, age INTEGER);
CREATE TABLE orders(order_id INTEGER PRIMARY KEY, user_id INTEGER, product_id INTEGER, quantity INTEGER);
INSERT INTO users VALUES (1, 'Alice', 25), (2, 'Bob', 30), (3, 'Charlie', 35);
INSERT INTO orders VALUES (1, 1, 100, 5), (2, 1, 101, 3), (3, 2, 100, 7);
CREATE MATERIALIZED VIEW user_orders AS
SELECT u.name, o.quantity
FROM users u
JOIN orders o ON u.id = o.user_id;
SELECT * FROM user_orders ORDER BY name, quantity;
} {Alice|3
Alice|5
Bob|7}
do_execsql_test_on_specific_db {:memory:} matview-join-with-aggregation {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT);
CREATE TABLE orders(order_id INTEGER PRIMARY KEY, user_id INTEGER, amount INTEGER);
INSERT INTO users VALUES (1, 'Alice'), (2, 'Bob');
INSERT INTO orders VALUES (1, 1, 100), (2, 1, 150), (3, 2, 200), (4, 2, 50);
CREATE MATERIALIZED VIEW user_totals AS
SELECT u.name, SUM(o.amount) as total_amount
FROM users u
JOIN orders o ON u.id = o.user_id
GROUP BY u.name;
SELECT * FROM user_totals ORDER BY name;
} {Alice|250
Bob|250}
do_execsql_test_on_specific_db {:memory:} matview-three-way-join {
CREATE TABLE customers(id INTEGER PRIMARY KEY, name TEXT, city TEXT);
CREATE TABLE orders(id INTEGER PRIMARY KEY, customer_id INTEGER, product_id INTEGER, quantity INTEGER);
CREATE TABLE products(id INTEGER PRIMARY KEY, name TEXT, price INTEGER);
INSERT INTO customers VALUES (1, 'Alice', 'NYC'), (2, 'Bob', 'LA');
INSERT INTO products VALUES (1, 'Widget', 10), (2, 'Gadget', 20);
INSERT INTO orders VALUES (1, 1, 1, 5), (2, 1, 2, 3), (3, 2, 1, 2);
CREATE MATERIALIZED VIEW sales_summary AS
SELECT c.name as customer_name, p.name as product_name, o.quantity
FROM customers c
JOIN orders o ON c.id = o.customer_id
JOIN products p ON o.product_id = p.id;
SELECT * FROM sales_summary ORDER BY customer_name, product_name;
} {Alice|Gadget|3
Alice|Widget|5
Bob|Widget|2}
do_execsql_test_on_specific_db {:memory:} matview-three-way-join-with-aggregation {
CREATE TABLE customers(id INTEGER PRIMARY KEY, name TEXT);
CREATE TABLE orders(id INTEGER PRIMARY KEY, customer_id INTEGER, product_id INTEGER, quantity INTEGER);
CREATE TABLE products(id INTEGER PRIMARY KEY, name TEXT, price INTEGER);
INSERT INTO customers VALUES (1, 'Alice'), (2, 'Bob');
INSERT INTO products VALUES (1, 'Widget', 10), (2, 'Gadget', 20);
INSERT INTO orders VALUES (1, 1, 1, 5), (2, 1, 2, 3), (3, 2, 1, 2), (4, 1, 1, 4);
CREATE MATERIALIZED VIEW sales_totals AS
SELECT c.name as customer_name, p.name as product_name,
SUM(o.quantity) as total_quantity,
SUM(o.quantity * p.price) as total_value
FROM customers c
JOIN orders o ON c.id = o.customer_id
JOIN products p ON o.product_id = p.id
GROUP BY c.name, p.name;
SELECT * FROM sales_totals ORDER BY customer_name, product_name;
} {Alice|Gadget|3|60
Alice|Widget|9|90
Bob|Widget|2|20}
do_execsql_test_on_specific_db {:memory:} matview-join-incremental-insert {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT);
CREATE TABLE orders(order_id INTEGER PRIMARY KEY, user_id INTEGER, amount INTEGER);
INSERT INTO users VALUES (1, 'Alice');
INSERT INTO orders VALUES (1, 1, 100);
CREATE MATERIALIZED VIEW user_orders AS
SELECT u.name, o.amount
FROM users u
JOIN orders o ON u.id = o.user_id;
SELECT COUNT(*) FROM user_orders;
INSERT INTO orders VALUES (2, 1, 150);
SELECT COUNT(*) FROM user_orders;
INSERT INTO users VALUES (2, 'Bob');
INSERT INTO orders VALUES (3, 2, 200);
SELECT COUNT(*) FROM user_orders;
} {1
2
3}
do_execsql_test_on_specific_db {:memory:} matview-join-incremental-delete {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT);
CREATE TABLE orders(order_id INTEGER PRIMARY KEY, user_id INTEGER, amount INTEGER);
INSERT INTO users VALUES (1, 'Alice'), (2, 'Bob');
INSERT INTO orders VALUES (1, 1, 100), (2, 1, 150), (3, 2, 200);
CREATE MATERIALIZED VIEW user_orders AS
SELECT u.name, o.amount
FROM users u
JOIN orders o ON u.id = o.user_id;
SELECT COUNT(*) FROM user_orders;
DELETE FROM orders WHERE order_id = 2;
SELECT COUNT(*) FROM user_orders;
DELETE FROM users WHERE id = 2;
SELECT COUNT(*) FROM user_orders;
} {3
2
1}
do_execsql_test_on_specific_db {:memory:} matview-join-incremental-update {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT);
CREATE TABLE orders(order_id INTEGER PRIMARY KEY, user_id INTEGER, amount INTEGER);
INSERT INTO users VALUES (1, 'Alice'), (2, 'Bob');
INSERT INTO orders VALUES (1, 1, 100), (2, 2, 200);
CREATE MATERIALIZED VIEW user_orders AS
SELECT u.name, o.amount
FROM users u
JOIN orders o ON u.id = o.user_id;
SELECT * FROM user_orders ORDER BY name;
UPDATE orders SET amount = 150 WHERE order_id = 1;
SELECT * FROM user_orders ORDER BY name;
UPDATE users SET name = 'Robert' WHERE id = 2;
SELECT * FROM user_orders ORDER BY name;
} {Alice|100
Bob|200
Alice|150
Bob|200
Alice|150
Robert|200}
do_execsql_test_on_specific_db {:memory:} matview-join-with-filter {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT, age INTEGER);
CREATE TABLE orders(order_id INTEGER PRIMARY KEY, user_id INTEGER, amount INTEGER);
INSERT INTO users VALUES (1, 'Alice', 25), (2, 'Bob', 35), (3, 'Charlie', 20);
INSERT INTO orders VALUES (1, 1, 100), (2, 2, 200), (3, 3, 150);
CREATE MATERIALIZED VIEW adult_orders AS
SELECT u.name, o.amount
FROM users u
JOIN orders o ON u.id = o.user_id
WHERE u.age > 21;
SELECT * FROM adult_orders ORDER BY name;
} {Alice|100
Bob|200}
do_execsql_test_on_specific_db {:memory:} matview-join-rollback {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT);
CREATE TABLE orders(order_id INTEGER PRIMARY KEY, user_id INTEGER, amount INTEGER);
INSERT INTO users VALUES (1, 'Alice'), (2, 'Bob');
INSERT INTO orders VALUES (1, 1, 100), (2, 2, 200);
CREATE MATERIALIZED VIEW user_orders AS
SELECT u.name, o.amount
FROM users u
JOIN orders o ON u.id = o.user_id;
SELECT COUNT(*) FROM user_orders;
BEGIN;
INSERT INTO users VALUES (3, 'Charlie');
INSERT INTO orders VALUES (3, 3, 300);
SELECT COUNT(*) FROM user_orders;
ROLLBACK;
SELECT COUNT(*) FROM user_orders;
} {2
3
2}
# ===== COMPREHENSIVE JOIN TESTS =====
# Test 1: Join with filter BEFORE the join (on base tables)
do_execsql_test_on_specific_db {:memory:} matview-join-with-pre-filter {
CREATE TABLE employees(id INTEGER PRIMARY KEY, name TEXT, department TEXT, salary INTEGER);
CREATE TABLE departments(id INTEGER PRIMARY KEY, dept_name TEXT, budget INTEGER);
INSERT INTO employees VALUES
(1, 'Alice', 'Engineering', 80000),
(2, 'Bob', 'Engineering', 90000),
(3, 'Charlie', 'Sales', 60000),
(4, 'David', 'Sales', 65000),
(5, 'Eve', 'HR', 70000);
INSERT INTO departments VALUES
(1, 'Engineering', 500000),
(2, 'Sales', 300000),
(3, 'HR', 200000);
-- View: Join only high-salary employees with their departments
CREATE MATERIALIZED VIEW high_earners_by_dept AS
SELECT e.name, e.salary, d.dept_name, d.budget
FROM employees e
JOIN departments d ON e.department = d.dept_name
WHERE e.salary > 70000;
SELECT * FROM high_earners_by_dept ORDER BY salary DESC;
} {Bob|90000|Engineering|500000
Alice|80000|Engineering|500000}
# Test 2: Join with filter AFTER the join
do_execsql_test_on_specific_db {:memory:} matview-join-with-post-filter {
CREATE TABLE products(id INTEGER PRIMARY KEY, name TEXT, category_id INTEGER, price INTEGER);
CREATE TABLE categories(id INTEGER PRIMARY KEY, name TEXT, min_price INTEGER);
INSERT INTO products VALUES
(1, 'Laptop', 1, 1200),
(2, 'Mouse', 1, 25),
(3, 'Shirt', 2, 50),
(4, 'Shoes', 2, 120);
INSERT INTO categories VALUES
(1, 'Electronics', 100),
(2, 'Clothing', 30);
-- View: Products that meet or exceed their category's minimum price
CREATE MATERIALIZED VIEW premium_products AS
SELECT p.name as product, c.name as category, p.price, c.min_price
FROM products p
JOIN categories c ON p.category_id = c.id
WHERE p.price >= c.min_price;
SELECT * FROM premium_products ORDER BY price DESC;
} {Laptop|Electronics|1200|100
Shoes|Clothing|120|30
Shirt|Clothing|50|30}
# Test 3: Join with aggregation BEFORE the join
do_execsql_test_on_specific_db {:memory:} matview-aggregation-before-join {
CREATE TABLE orders(id INTEGER PRIMARY KEY, customer_id INTEGER, product_id INTEGER, quantity INTEGER, order_date INTEGER);
CREATE TABLE customers(id INTEGER PRIMARY KEY, name TEXT, tier TEXT);
INSERT INTO orders VALUES
(1, 1, 101, 2, 1),
(2, 1, 102, 1, 1),
(3, 2, 101, 5, 1),
(4, 1, 101, 3, 2),
(5, 2, 103, 2, 2),
(6, 3, 102, 1, 2);
INSERT INTO customers VALUES
(1, 'Alice', 'Gold'),
(2, 'Bob', 'Silver'),
(3, 'Charlie', 'Bronze');
-- View: Customer order counts joined with customer details
-- Note: Simplified to avoid subquery issues with DBSP compiler
CREATE MATERIALIZED VIEW customer_order_summary AS
SELECT c.name, c.tier, COUNT(o.id) as order_count, SUM(o.quantity) as total_quantity
FROM customers c
JOIN orders o ON c.id = o.customer_id
GROUP BY c.id, c.name, c.tier;
SELECT * FROM customer_order_summary ORDER BY total_quantity DESC;
} {Bob|Silver|2|7
Alice|Gold|3|6
Charlie|Bronze|1|1}
# Test 4: Join with aggregation AFTER the join
do_execsql_test_on_specific_db {:memory:} matview-aggregation-after-join {
CREATE TABLE sales(id INTEGER PRIMARY KEY, product_id INTEGER, store_id INTEGER, units_sold INTEGER, revenue INTEGER);
CREATE TABLE stores(id INTEGER PRIMARY KEY, name TEXT, region TEXT);
INSERT INTO sales VALUES
(1, 1, 1, 10, 1000),
(2, 1, 2, 15, 1500),
(3, 2, 1, 5, 250),
(4, 2, 2, 8, 400),
(5, 1, 3, 12, 1200),
(6, 2, 3, 6, 300);
INSERT INTO stores VALUES
(1, 'StoreA', 'North'),
(2, 'StoreB', 'North'),
(3, 'StoreC', 'South');
-- View: Regional sales summary (aggregate after joining)
CREATE MATERIALIZED VIEW regional_sales AS
SELECT st.region, SUM(s.units_sold) as total_units, SUM(s.revenue) as total_revenue
FROM sales s
JOIN stores st ON s.store_id = st.id
GROUP BY st.region;
SELECT * FROM regional_sales ORDER BY total_revenue DESC;
} {North|38|3150
South|18|1500}
# Test 5: Modifying both tables in same transaction
do_execsql_test_on_specific_db {:memory:} matview-join-both-tables-modified {
CREATE TABLE authors(id INTEGER PRIMARY KEY, name TEXT);
CREATE TABLE books(id INTEGER PRIMARY KEY, title TEXT, author_id INTEGER, year INTEGER);
INSERT INTO authors VALUES (1, 'Orwell'), (2, 'Asimov');
INSERT INTO books VALUES (1, '1984', 1, 1949), (2, 'Foundation', 2, 1951);
CREATE MATERIALIZED VIEW author_books AS
SELECT a.name, b.title, b.year
FROM authors a
JOIN books b ON a.id = b.author_id;
SELECT COUNT(*) FROM author_books;
BEGIN;
INSERT INTO authors VALUES (3, 'Herbert');
INSERT INTO books VALUES (3, 'Dune', 3, 1965);
SELECT COUNT(*) FROM author_books;
COMMIT;
SELECT * FROM author_books ORDER BY year;
} {2
3
Orwell|1984|1949
Asimov|Foundation|1951
Herbert|Dune|1965}
# Test 6: Modifying only one table in transaction
do_execsql_test_on_specific_db {:memory:} matview-join-single-table-modified {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT, active INTEGER);
CREATE TABLE posts(id INTEGER PRIMARY KEY, user_id INTEGER, content TEXT);
INSERT INTO users VALUES (1, 'Alice', 1), (2, 'Bob', 1), (3, 'Charlie', 0);
INSERT INTO posts VALUES (1, 1, 'Hello'), (2, 1, 'World'), (3, 2, 'Test');
CREATE MATERIALIZED VIEW active_user_posts AS
SELECT u.name, p.content
FROM users u
JOIN posts p ON u.id = p.user_id
WHERE u.active = 1;
SELECT COUNT(*) FROM active_user_posts;
-- Add posts for existing user (modify only posts table)
BEGIN;
INSERT INTO posts VALUES (4, 1, 'NewPost'), (5, 2, 'Another');
SELECT COUNT(*) FROM active_user_posts;
COMMIT;
SELECT * FROM active_user_posts ORDER BY name, content;
} {3
5
Alice|Hello
Alice|NewPost
Alice|World
Bob|Another
Bob|Test}
do_execsql_test_on_specific_db {:memory:} matview-three-way-incremental {
CREATE TABLE students(id INTEGER PRIMARY KEY, name TEXT, major TEXT);
CREATE TABLE courses(id INTEGER PRIMARY KEY, name TEXT, department TEXT, credits INTEGER);
CREATE TABLE enrollments(student_id INTEGER, course_id INTEGER, grade TEXT, PRIMARY KEY(student_id, course_id));
INSERT INTO students VALUES (1, 'Alice', 'CS'), (2, 'Bob', 'Math');
INSERT INTO courses VALUES (1, 'DatabaseSystems', 'CS', 3), (2, 'Calculus', 'Math', 4);
INSERT INTO enrollments VALUES (1, 1, 'A'), (2, 2, 'B');
CREATE MATERIALIZED VIEW student_transcripts AS
SELECT s.name as student, c.name as course, c.credits, e.grade
FROM students s
JOIN enrollments e ON s.id = e.student_id
JOIN courses c ON e.course_id = c.id;
SELECT COUNT(*) FROM student_transcripts;
-- Add new student
INSERT INTO students VALUES (3, 'Charlie', 'CS');
SELECT COUNT(*) FROM student_transcripts;
-- Enroll new student
INSERT INTO enrollments VALUES (3, 1, 'A'), (3, 2, 'A');
SELECT COUNT(*) FROM student_transcripts;
-- Add new course
INSERT INTO courses VALUES (3, 'Algorithms', 'CS', 3);
SELECT COUNT(*) FROM student_transcripts;
-- Enroll existing students in new course
INSERT INTO enrollments VALUES (1, 3, 'B'), (3, 3, 'A');
SELECT COUNT(*) FROM student_transcripts;
SELECT * FROM student_transcripts ORDER BY student, course;
} {2
2
4
4
6
Alice|Algorithms|3|B
Alice|DatabaseSystems|3|A
Bob|Calculus|4|B
Charlie|Algorithms|3|A
Charlie|Calculus|4|A
Charlie|DatabaseSystems|3|A}
do_execsql_test_on_specific_db {:memory:} matview-self-join {
CREATE TABLE employees(id INTEGER PRIMARY KEY, name TEXT, manager_id INTEGER, salary INTEGER);
INSERT INTO employees VALUES
(1, 'CEO', NULL, 150000),
(2, 'VPSales', 1, 120000),
(3, 'VPEngineering', 1, 130000),
(4, 'Engineer1', 3, 90000),
(5, 'Engineer2', 3, 85000),
(6, 'SalesRep', 2, 70000);
CREATE MATERIALIZED VIEW org_chart AS
SELECT e.name as employee, m.name as manager, e.salary
FROM employees e
JOIN employees m ON e.manager_id = m.id;
SELECT * FROM org_chart ORDER BY salary DESC;
} {VPEngineering|CEO|130000
VPSales|CEO|120000
Engineer1|VPEngineering|90000
Engineer2|VPEngineering|85000
SalesRep|VPSales|70000}
do_execsql_test_on_specific_db {:memory:} matview-join-cascade-update {
CREATE TABLE categories(id INTEGER PRIMARY KEY, name TEXT, discount_rate INTEGER);
CREATE TABLE products(id INTEGER PRIMARY KEY, name TEXT, category_id INTEGER, base_price INTEGER);
INSERT INTO categories VALUES (1, 'Electronics', 10), (2, 'Books', 5);
INSERT INTO products VALUES
(1, 'Laptop', 1, 1000),
(2, 'Phone', 1, 500),
(3, 'Novel', 2, 20),
(4, 'Textbook', 2, 80);
CREATE MATERIALIZED VIEW discounted_prices AS
SELECT p.name as product, c.name as category,
p.base_price, c.discount_rate,
(p.base_price * (100 - c.discount_rate) / 100) as final_price
FROM products p
JOIN categories c ON p.category_id = c.id;
SELECT * FROM discounted_prices ORDER BY final_price DESC;
-- Update discount rate for Electronics
UPDATE categories SET discount_rate = 20 WHERE id = 1;
SELECT * FROM discounted_prices ORDER BY final_price DESC;
} {Laptop|Electronics|1000|10|900
Phone|Electronics|500|10|450
Textbook|Books|80|5|76
Novel|Books|20|5|19
Laptop|Electronics|1000|20|800
Phone|Electronics|500|20|400
Textbook|Books|80|5|76
Novel|Books|20|5|19}
do_execsql_test_on_specific_db {:memory:} matview-join-delete-cascade {
CREATE TABLE users(id INTEGER PRIMARY KEY, name TEXT, active INTEGER);
CREATE TABLE sessions(id INTEGER PRIMARY KEY, user_id INTEGER, duration INTEGER);
INSERT INTO users VALUES (1, 'Alice', 1), (2, 'Bob', 1), (3, 'Charlie', 0);
INSERT INTO sessions VALUES
(1, 1, 30),
(2, 1, 45),
(3, 2, 60),
(4, 3, 15),
(5, 2, 90);
CREATE MATERIALIZED VIEW active_sessions AS
SELECT u.name, s.duration
FROM users u
JOIN sessions s ON u.id = s.user_id
WHERE u.active = 1;
SELECT COUNT(*) FROM active_sessions;
-- Delete Bob's sessions
DELETE FROM sessions WHERE user_id = 2;
SELECT COUNT(*) FROM active_sessions;
SELECT * FROM active_sessions ORDER BY name, duration;
} {4
2
Alice|30
Alice|45}
do_execsql_test_on_specific_db {:memory:} matview-join-complex-where {
CREATE TABLE orders(id INTEGER PRIMARY KEY, customer_id INTEGER, product_id INTEGER, quantity INTEGER, price INTEGER, order_date INTEGER);
CREATE TABLE customers(id INTEGER PRIMARY KEY, name TEXT, tier TEXT, country TEXT);
INSERT INTO customers VALUES
(1, 'Alice', 'Gold', 'USA'),
(2, 'Bob', 'Silver', 'Canada'),
(3, 'Charlie', 'Gold', 'USA'),
(4, 'David', 'Bronze', 'UK');
INSERT INTO orders VALUES
(1, 1, 1, 5, 100, 20240101),
(2, 2, 2, 3, 50, 20240102),
(3, 3, 1, 10, 100, 20240103),
(4, 4, 3, 2, 75, 20240104),
(5, 1, 2, 4, 50, 20240105),
(6, 3, 3, 6, 75, 20240106);
-- View: Gold tier USA customers with high-value orders
CREATE MATERIALIZED VIEW premium_usa_orders AS
SELECT c.name, o.quantity, o.price, (o.quantity * o.price) as total
FROM customers c
JOIN orders o ON c.id = o.customer_id
WHERE c.tier = 'Gold'
AND c.country = 'USA'
AND (o.quantity * o.price) >= 400;
SELECT * FROM premium_usa_orders ORDER by total DESC;
} {Charlie|10|100|1000
Alice|5|100|500
Charlie|6|75|450}