risingwave_frontend/optimizer/plan_node/generic/
file_scan.rsuse educe::Educe;
use risingwave_common::catalog::{ColumnDesc, ColumnId, Schema};
use super::GenericPlanNode;
use crate::optimizer::optimizer_context::OptimizerContextRef;
use crate::optimizer::property::FunctionalDependencySet;
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub enum FileFormat {
Parquet,
}
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub enum StorageType {
S3,
}
#[derive(Debug, Clone, Educe)]
#[educe(PartialEq, Eq, Hash)]
pub struct FileScan {
pub schema: Schema,
pub file_format: FileFormat,
pub storage_type: StorageType,
pub s3_region: String,
pub s3_access_key: String,
pub s3_secret_key: String,
pub file_location: Vec<String>,
#[educe(PartialEq(ignore))]
#[educe(Hash(ignore))]
pub ctx: OptimizerContextRef,
}
impl GenericPlanNode for FileScan {
fn schema(&self) -> Schema {
self.schema.clone()
}
fn stream_key(&self) -> Option<Vec<usize>> {
None
}
fn ctx(&self) -> OptimizerContextRef {
self.ctx.clone()
}
fn functional_dependency(&self) -> FunctionalDependencySet {
FunctionalDependencySet::new(self.schema.len())
}
}
impl FileScan {
pub fn columns(&self) -> Vec<ColumnDesc> {
self.schema
.fields
.iter()
.enumerate()
.map(|(i, f)| {
ColumnDesc::named(f.name.clone(), ColumnId::new(i as i32), f.data_type.clone())
})
.collect()
}
}