risingwave_meta_model/
fragment.rs

1// Copyright 2025 RisingWave Labs
2//
3// Licensed under the Apache License, Version 2.0 (the "License");
4// you may not use this file except in compliance with the License.
5// You may obtain a copy of the License at
6//
7//     http://www.apache.org/licenses/LICENSE-2.0
8//
9// Unless required by applicable law or agreed to in writing, software
10// distributed under the License is distributed on an "AS IS" BASIS,
11// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
12// See the License for the specific language governing permissions and
13// limitations under the License.
14
15use risingwave_common::id::JobId;
16use risingwave_pb::meta::table_fragments::fragment::PbFragmentDistributionType;
17use sea_orm::entity::prelude::*;
18use serde::{Deserialize, Serialize};
19
20use crate::{FragmentId, I32Array, StreamNode, StreamingParallelism};
21
22#[derive(Clone, Debug, PartialEq, DeriveEntityModel, Eq, Serialize, Deserialize)]
23#[sea_orm(table_name = "fragment")]
24pub struct Model {
25    #[sea_orm(primary_key)]
26    pub fragment_id: FragmentId,
27    pub job_id: JobId,
28    pub fragment_type_mask: i32,
29    pub distribution_type: DistributionType,
30    /// Note: the `StreamNode` is different from the final plan node used by actors.
31    /// Specifically, `Merge` nodes' `upstream_actor_id` will be filled. (See `compose_fragment`)
32    pub stream_node: StreamNode,
33    pub state_table_ids: I32Array,
34    #[deprecated]
35    pub upstream_fragment_id: I32Array,
36    pub vnode_count: i32,
37    #[sea_orm(column_type = "JsonBinary", nullable)]
38    pub parallelism: Option<StreamingParallelism>,
39}
40
41#[derive(Copy, Clone, Debug, PartialEq, Eq, EnumIter, DeriveActiveEnum, Serialize, Deserialize)]
42#[sea_orm(rs_type = "String", db_type = "string(None)")]
43pub enum DistributionType {
44    #[sea_orm(string_value = "SINGLE")]
45    Single,
46    #[sea_orm(string_value = "HASH")]
47    Hash,
48}
49
50impl From<DistributionType> for PbFragmentDistributionType {
51    fn from(val: DistributionType) -> Self {
52        match val {
53            DistributionType::Single => PbFragmentDistributionType::Single,
54            DistributionType::Hash => PbFragmentDistributionType::Hash,
55        }
56    }
57}
58
59impl From<PbFragmentDistributionType> for DistributionType {
60    fn from(val: PbFragmentDistributionType) -> Self {
61        match val {
62            PbFragmentDistributionType::Unspecified => unreachable!(),
63            PbFragmentDistributionType::Single => DistributionType::Single,
64            PbFragmentDistributionType::Hash => DistributionType::Hash,
65        }
66    }
67}
68
69#[derive(Copy, Clone, Debug, EnumIter, DeriveRelation)]
70pub enum Relation {
71    #[sea_orm(
72        belongs_to = "super::object::Entity",
73        from = "Column::JobId",
74        to = "super::object::Column::Oid",
75        on_update = "NoAction",
76        on_delete = "Cascade"
77    )]
78    Object,
79}
80
81impl Related<super::object::Entity> for Entity {
82    fn to() -> RelationDef {
83        Relation::Object.def()
84    }
85}
86
87impl ActiveModelBehavior for ActiveModel {}