risingwave_meta/stream/
scale.rs

1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
1001
1002
1003
1004
1005
1006
1007
1008
1009
1010
1011
1012
1013
1014
1015
1016
1017
1018
1019
1020
1021
1022
1023
1024
1025
1026
1027
1028
1029
1030
1031
1032
1033
1034
1035
1036
1037
1038
1039
1040
1041
1042
1043
1044
1045
1046
1047
1048
1049
1050
1051
1052
1053
1054
1055
1056
1057
1058
1059
1060
1061
1062
1063
1064
1065
1066
1067
1068
1069
1070
1071
1072
1073
1074
1075
1076
1077
1078
1079
1080
1081
1082
1083
1084
1085
1086
1087
1088
1089
1090
1091
1092
1093
1094
1095
1096
1097
1098
1099
1100
1101
1102
1103
1104
1105
1106
1107
1108
1109
1110
1111
1112
1113
1114
1115
1116
1117
1118
1119
1120
1121
1122
1123
1124
1125
1126
1127
1128
1129
1130
1131
1132
1133
1134
1135
1136
1137
1138
1139
1140
1141
1142
1143
1144
1145
1146
1147
1148
1149
1150
1151
1152
1153
1154
1155
1156
1157
1158
1159
1160
1161
1162
1163
1164
1165
1166
1167
1168
1169
1170
1171
1172
1173
1174
1175
1176
1177
1178
1179
1180
1181
1182
1183
1184
1185
1186
1187
1188
1189
1190
1191
1192
1193
1194
1195
1196
1197
1198
1199
1200
1201
1202
1203
1204
1205
1206
1207
1208
1209
1210
1211
1212
1213
1214
1215
1216
1217
1218
1219
1220
1221
1222
1223
1224
1225
1226
1227
1228
1229
1230
1231
1232
1233
1234
1235
1236
1237
1238
1239
1240
1241
1242
1243
1244
1245
1246
1247
1248
1249
1250
1251
1252
1253
1254
1255
1256
1257
1258
1259
1260
1261
1262
1263
1264
1265
1266
1267
1268
1269
1270
1271
1272
1273
1274
1275
1276
1277
1278
1279
1280
1281
1282
1283
1284
1285
1286
1287
1288
1289
1290
1291
1292
1293
1294
1295
1296
1297
1298
1299
1300
1301
1302
1303
1304
1305
1306
1307
1308
1309
1310
1311
1312
1313
1314
1315
1316
1317
1318
1319
1320
1321
1322
1323
1324
1325
1326
1327
1328
1329
1330
1331
1332
1333
1334
1335
1336
1337
1338
1339
1340
1341
1342
1343
1344
1345
1346
1347
1348
1349
1350
1351
1352
1353
1354
1355
1356
1357
1358
1359
1360
1361
1362
1363
1364
1365
1366
1367
1368
1369
1370
1371
1372
1373
1374
1375
1376
1377
1378
1379
1380
1381
1382
1383
1384
1385
1386
1387
1388
1389
1390
1391
1392
1393
1394
1395
1396
1397
1398
1399
1400
1401
1402
1403
1404
1405
1406
1407
1408
1409
1410
1411
1412
1413
1414
1415
1416
1417
1418
1419
1420
1421
1422
1423
1424
1425
1426
1427
1428
1429
1430
1431
1432
1433
1434
1435
1436
1437
1438
1439
1440
1441
1442
1443
1444
1445
1446
1447
1448
1449
1450
1451
1452
1453
1454
1455
1456
1457
1458
1459
1460
1461
1462
1463
1464
1465
1466
1467
1468
1469
1470
1471
1472
1473
1474
1475
1476
1477
1478
1479
1480
1481
1482
1483
1484
1485
1486
1487
1488
1489
1490
1491
1492
1493
1494
1495
1496
1497
1498
1499
1500
1501
1502
1503
1504
1505
1506
1507
1508
1509
1510
1511
1512
1513
1514
1515
1516
1517
1518
1519
1520
1521
1522
1523
1524
1525
1526
1527
1528
1529
1530
1531
1532
1533
1534
1535
1536
1537
1538
1539
1540
1541
1542
1543
1544
1545
1546
1547
1548
1549
1550
1551
1552
1553
1554
1555
1556
1557
1558
1559
1560
1561
1562
1563
1564
1565
1566
1567
1568
1569
1570
1571
1572
1573
1574
1575
1576
1577
1578
1579
1580
1581
1582
1583
1584
1585
1586
1587
1588
1589
1590
1591
1592
1593
1594
1595
1596
1597
1598
1599
1600
1601
1602
1603
1604
1605
1606
1607
1608
1609
1610
1611
1612
1613
1614
1615
1616
1617
1618
1619
1620
1621
1622
1623
1624
1625
1626
1627
1628
1629
1630
1631
1632
1633
1634
1635
1636
1637
1638
1639
1640
1641
1642
1643
1644
1645
1646
1647
1648
1649
1650
1651
1652
1653
1654
1655
1656
1657
1658
1659
1660
1661
1662
1663
1664
1665
1666
1667
1668
1669
1670
1671
1672
1673
1674
1675
1676
1677
1678
1679
1680
1681
1682
1683
1684
1685
1686
1687
1688
1689
1690
1691
1692
1693
1694
1695
1696
1697
1698
1699
1700
1701
1702
1703
1704
1705
1706
1707
1708
1709
1710
1711
1712
1713
1714
1715
1716
1717
1718
1719
1720
1721
1722
1723
1724
1725
1726
1727
1728
1729
1730
1731
1732
1733
1734
1735
1736
1737
1738
1739
1740
1741
1742
1743
1744
1745
1746
1747
1748
1749
1750
1751
1752
1753
1754
1755
1756
1757
1758
1759
1760
1761
1762
1763
1764
1765
1766
1767
1768
1769
1770
1771
1772
1773
1774
1775
1776
1777
1778
1779
1780
1781
1782
1783
1784
1785
1786
1787
1788
1789
1790
1791
1792
1793
1794
1795
1796
1797
1798
1799
1800
1801
1802
1803
1804
1805
1806
1807
1808
1809
1810
1811
1812
1813
1814
1815
1816
1817
1818
1819
1820
1821
1822
1823
1824
1825
1826
1827
1828
1829
1830
1831
1832
1833
1834
1835
1836
1837
1838
1839
1840
1841
1842
1843
1844
1845
1846
1847
1848
1849
1850
1851
1852
1853
1854
1855
1856
1857
1858
1859
1860
1861
1862
1863
1864
1865
1866
1867
1868
1869
1870
1871
1872
1873
1874
1875
1876
1877
1878
1879
1880
1881
1882
1883
1884
1885
1886
1887
1888
1889
1890
1891
1892
1893
1894
1895
1896
1897
1898
1899
1900
1901
1902
1903
1904
1905
1906
1907
1908
1909
1910
1911
1912
1913
1914
1915
1916
1917
1918
1919
1920
1921
1922
1923
1924
1925
1926
1927
1928
1929
1930
1931
1932
1933
1934
1935
1936
1937
1938
1939
1940
1941
1942
1943
1944
1945
1946
1947
1948
1949
1950
1951
1952
1953
1954
1955
1956
1957
1958
1959
1960
1961
1962
1963
1964
1965
1966
1967
1968
1969
1970
1971
1972
1973
1974
1975
1976
1977
1978
1979
1980
1981
1982
1983
1984
1985
1986
1987
1988
1989
1990
1991
1992
1993
1994
1995
1996
1997
1998
1999
2000
2001
2002
2003
2004
2005
2006
2007
2008
2009
2010
2011
2012
2013
2014
2015
2016
2017
2018
2019
2020
2021
2022
2023
2024
2025
2026
2027
2028
2029
2030
2031
2032
2033
2034
2035
2036
2037
2038
2039
2040
2041
2042
2043
2044
2045
2046
2047
2048
2049
2050
2051
2052
2053
2054
2055
2056
2057
2058
2059
2060
2061
2062
2063
2064
2065
2066
2067
2068
2069
2070
2071
2072
2073
2074
2075
2076
2077
2078
2079
2080
2081
2082
2083
2084
2085
2086
2087
2088
2089
2090
2091
2092
2093
2094
2095
2096
2097
2098
2099
2100
2101
2102
2103
2104
2105
2106
2107
2108
2109
2110
2111
2112
2113
2114
2115
2116
2117
2118
2119
2120
2121
2122
2123
2124
2125
2126
2127
2128
2129
2130
2131
2132
2133
2134
2135
2136
2137
2138
2139
2140
2141
2142
2143
2144
2145
2146
2147
2148
2149
2150
2151
2152
2153
2154
2155
2156
2157
2158
2159
2160
2161
2162
2163
2164
2165
2166
2167
2168
2169
2170
2171
2172
2173
2174
2175
2176
2177
2178
2179
2180
2181
2182
2183
2184
2185
2186
2187
2188
2189
2190
2191
2192
2193
2194
2195
2196
2197
2198
2199
2200
2201
2202
2203
2204
2205
2206
2207
2208
2209
2210
2211
2212
2213
2214
2215
2216
2217
2218
2219
2220
2221
2222
2223
2224
2225
2226
2227
2228
2229
2230
2231
2232
2233
2234
2235
2236
2237
2238
2239
2240
2241
2242
2243
2244
2245
2246
2247
2248
2249
2250
2251
2252
2253
2254
2255
2256
2257
2258
2259
2260
2261
2262
2263
2264
2265
2266
2267
2268
2269
2270
2271
2272
2273
2274
2275
2276
2277
2278
2279
2280
2281
2282
2283
2284
2285
2286
2287
2288
2289
2290
2291
2292
2293
2294
2295
2296
2297
2298
2299
2300
2301
2302
2303
2304
2305
2306
2307
2308
2309
2310
2311
2312
2313
2314
2315
2316
2317
2318
2319
2320
2321
2322
2323
2324
2325
2326
2327
2328
2329
2330
2331
2332
2333
2334
2335
2336
2337
2338
2339
2340
2341
2342
2343
2344
2345
2346
2347
2348
2349
2350
2351
2352
2353
2354
2355
2356
2357
2358
2359
2360
2361
2362
2363
2364
2365
2366
2367
2368
2369
2370
2371
2372
2373
2374
2375
2376
2377
2378
2379
2380
2381
2382
2383
2384
2385
2386
2387
2388
2389
2390
2391
2392
2393
2394
2395
2396
2397
2398
2399
2400
2401
2402
2403
2404
2405
2406
2407
2408
2409
2410
2411
2412
2413
2414
2415
2416
2417
2418
2419
2420
2421
2422
2423
2424
2425
2426
2427
2428
2429
2430
2431
2432
2433
2434
2435
2436
2437
2438
2439
2440
2441
2442
2443
2444
2445
2446
2447
2448
2449
2450
2451
2452
2453
2454
2455
2456
2457
2458
2459
2460
2461
2462
2463
2464
2465
2466
2467
2468
2469
2470
2471
2472
2473
2474
2475
2476
2477
2478
2479
2480
2481
2482
2483
2484
2485
2486
2487
2488
2489
2490
2491
2492
2493
2494
2495
2496
2497
2498
2499
2500
2501
2502
2503
2504
2505
2506
2507
2508
2509
2510
2511
2512
2513
2514
2515
2516
2517
2518
2519
2520
2521
2522
2523
2524
2525
2526
2527
2528
2529
2530
2531
2532
2533
2534
2535
2536
2537
2538
2539
2540
2541
2542
2543
2544
2545
2546
2547
2548
2549
2550
2551
2552
2553
2554
2555
2556
2557
2558
2559
2560
2561
2562
2563
2564
2565
2566
2567
2568
2569
2570
2571
2572
2573
2574
2575
2576
2577
2578
2579
2580
2581
2582
2583
2584
2585
2586
2587
2588
2589
2590
2591
2592
2593
2594
2595
2596
2597
2598
2599
2600
2601
2602
2603
2604
2605
2606
2607
2608
2609
2610
2611
2612
2613
2614
2615
2616
2617
2618
2619
2620
2621
2622
2623
2624
2625
2626
2627
2628
2629
2630
2631
2632
2633
2634
2635
2636
2637
2638
2639
2640
2641
2642
2643
2644
2645
2646
2647
2648
2649
2650
2651
2652
2653
2654
2655
2656
2657
2658
2659
2660
2661
2662
2663
2664
2665
2666
2667
2668
2669
2670
2671
2672
2673
2674
2675
2676
2677
2678
2679
2680
2681
2682
2683
2684
2685
2686
2687
2688
2689
2690
2691
2692
2693
2694
2695
2696
2697
2698
2699
2700
2701
2702
2703
2704
2705
2706
2707
2708
2709
2710
2711
2712
2713
2714
2715
2716
2717
2718
2719
2720
2721
2722
2723
2724
2725
2726
2727
2728
2729
2730
2731
2732
2733
2734
2735
2736
2737
2738
2739
2740
2741
2742
2743
2744
2745
2746
2747
2748
2749
2750
2751
2752
2753
2754
2755
2756
2757
2758
2759
2760
2761
2762
2763
2764
2765
2766
2767
2768
2769
2770
2771
2772
2773
2774
2775
2776
2777
2778
2779
2780
2781
2782
2783
2784
2785
2786
2787
2788
2789
2790
2791
2792
2793
2794
2795
2796
2797
2798
2799
2800
2801
2802
2803
2804
2805
2806
2807
2808
2809
2810
2811
2812
2813
2814
2815
2816
2817
2818
2819
2820
// Copyright 2024 RisingWave Labs
//
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
//     http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.

use std::cmp::{min, Ordering};
use std::collections::hash_map::DefaultHasher;
use std::collections::{BTreeMap, BTreeSet, HashMap, HashSet, VecDeque};
use std::hash::{Hash, Hasher};
use std::iter::repeat;
use std::sync::Arc;
use std::time::Duration;

use anyhow::{anyhow, Context};
use futures::future::try_join_all;
use itertools::Itertools;
use num_integer::Integer;
use num_traits::abs;
use risingwave_common::bail;
use risingwave_common::bitmap::{Bitmap, BitmapBuilder};
use risingwave_common::catalog::{DatabaseId, TableId};
use risingwave_common::hash::ActorMapping;
use risingwave_common::util::iter_util::ZipEqDebug;
use risingwave_meta_model::{actor, fragment, ObjectId, StreamingParallelism, WorkerId};
use risingwave_pb::common::{PbActorLocation, WorkerNode, WorkerType};
use risingwave_pb::meta::subscribe_response::{Info, Operation};
use risingwave_pb::meta::table_fragments::actor_status::ActorState;
use risingwave_pb::meta::table_fragments::fragment::{
    FragmentDistributionType, PbFragmentDistributionType,
};
use risingwave_pb::meta::table_fragments::{self, ActorStatus, PbFragment, State};
use risingwave_pb::meta::FragmentWorkerSlotMappings;
use risingwave_pb::stream_plan::stream_node::NodeBody;
use risingwave_pb::stream_plan::{
    Dispatcher, DispatcherType, FragmentTypeFlag, PbDispatcher, PbStreamActor, StreamNode,
};
use thiserror_ext::AsReport;
use tokio::sync::oneshot::Receiver;
use tokio::sync::{oneshot, RwLock, RwLockReadGuard, RwLockWriteGuard};
use tokio::task::JoinHandle;
use tokio::time::{Instant, MissedTickBehavior};

use crate::barrier::{Command, Reschedule};
use crate::controller::scale::RescheduleWorkingSet;
use crate::manager::{LocalNotification, MetaSrvEnv, MetadataManager};
use crate::model::{ActorId, DispatcherId, FragmentId, TableParallelism};
use crate::serving::{
    to_deleted_fragment_worker_slot_mapping, to_fragment_worker_slot_mapping, ServingVnodeMapping,
};
use crate::stream::{GlobalStreamManager, SourceManagerRef};
use crate::{MetaError, MetaResult};

#[derive(Debug, Clone, Eq, PartialEq)]
pub struct WorkerReschedule {
    pub worker_actor_diff: BTreeMap<WorkerId, isize>,
}

pub struct CustomFragmentInfo {
    pub fragment_id: u32,
    pub fragment_type_mask: u32,
    pub distribution_type: PbFragmentDistributionType,
    pub state_table_ids: Vec<u32>,
    pub upstream_fragment_ids: Vec<u32>,
    pub actor_template: PbStreamActor,
    pub actors: Vec<CustomActorInfo>,
}

#[derive(Default, Clone)]
pub struct CustomActorInfo {
    pub actor_id: u32,
    pub fragment_id: u32,
    pub dispatcher: Vec<Dispatcher>,
    pub upstream_actor_id: Vec<u32>,
    /// `None` if singleton.
    pub vnode_bitmap: Option<Bitmap>,
}

impl From<&PbStreamActor> for CustomActorInfo {
    fn from(
        PbStreamActor {
            actor_id,
            fragment_id,
            dispatcher,
            upstream_actor_id,
            vnode_bitmap,
            ..
        }: &PbStreamActor,
    ) -> Self {
        CustomActorInfo {
            actor_id: *actor_id,
            fragment_id: *fragment_id,
            dispatcher: dispatcher.clone(),
            upstream_actor_id: upstream_actor_id.clone(),
            vnode_bitmap: vnode_bitmap.as_ref().map(Bitmap::from),
        }
    }
}

impl From<&PbFragment> for CustomFragmentInfo {
    fn from(fragment: &PbFragment) -> Self {
        CustomFragmentInfo {
            fragment_id: fragment.fragment_id,
            fragment_type_mask: fragment.fragment_type_mask,
            distribution_type: fragment.distribution_type(),
            state_table_ids: fragment.state_table_ids.clone(),
            upstream_fragment_ids: fragment.upstream_fragment_ids.clone(),
            actor_template: fragment
                .actors
                .first()
                .cloned()
                .expect("no actor in fragment"),
            actors: fragment
                .actors
                .iter()
                .map(CustomActorInfo::from)
                .sorted_by(|actor_a, actor_b| actor_a.actor_id.cmp(&actor_b.actor_id))
                .collect(),
        }
    }
}

impl CustomFragmentInfo {
    pub fn get_fragment_type_mask(&self) -> u32 {
        self.fragment_type_mask
    }

    pub fn distribution_type(&self) -> FragmentDistributionType {
        self.distribution_type
    }
}

use educe::Educe;

use crate::controller::id::IdCategory;

// The debug implementation is arbitrary. Just used in debug logs.
#[derive(Educe)]
#[educe(Debug)]
pub struct RescheduleContext {
    /// Meta information for all Actors
    #[educe(Debug(ignore))]
    actor_map: HashMap<ActorId, CustomActorInfo>,
    /// Status of all Actors, used to find the location of the `Actor`
    actor_status: BTreeMap<ActorId, WorkerId>,
    /// Meta information of all `Fragment`, used to find the `Fragment`'s `Actor`
    #[educe(Debug(ignore))]
    fragment_map: HashMap<FragmentId, CustomFragmentInfo>,
    /// Index of all `Actor` upstreams, specific to `Dispatcher`
    upstream_dispatchers: HashMap<ActorId, Vec<(FragmentId, DispatcherId, DispatcherType)>>,
    /// Fragments with `StreamSource`
    stream_source_fragment_ids: HashSet<FragmentId>,
    /// Fragments with `StreamSourceBackfill`
    stream_source_backfill_fragment_ids: HashSet<FragmentId>,
    /// Target fragments in `NoShuffle` relation
    no_shuffle_target_fragment_ids: HashSet<FragmentId>,
    /// Source fragments in `NoShuffle` relation
    no_shuffle_source_fragment_ids: HashSet<FragmentId>,
    // index for dispatcher type from upstream fragment to downstream fragment
    fragment_dispatcher_map: HashMap<FragmentId, HashMap<FragmentId, DispatcherType>>,
}

impl RescheduleContext {
    fn actor_id_to_worker_id(&self, actor_id: &ActorId) -> MetaResult<WorkerId> {
        self.actor_status
            .get(actor_id)
            .cloned()
            .ok_or_else(|| anyhow!("could not find worker for actor {}", actor_id).into())
    }
}

/// This function provides an simple balancing method
/// The specific process is as follows
///
/// 1. Calculate the number of target actors, and calculate the average value and the remainder, and
///    use the average value as expected.
///
/// 2. Filter out the actor to be removed and the actor to be retained, and sort them from largest
///    to smallest (according to the number of virtual nodes held).
///
/// 3. Calculate their balance, 1) For the actors to be removed, the number of virtual nodes per
///    actor is the balance. 2) For retained actors, the number of virtual nodes - expected is the
///    balance. 3) For newly created actors, -expected is the balance (always negative).
///
/// 4. Allocate the remainder, high priority to newly created nodes.
///
/// 5. After that, merge removed, retained and created into a queue, with the head of the queue
///    being the source, and move the virtual nodes to the destination at the end of the queue.
///
/// This can handle scale in, scale out, migration, and simultaneous scaling with as much affinity
/// as possible.
///
/// Note that this function can only rebalance actors whose `vnode_bitmap` is not `None`, in other
/// words, for `Fragment` of `FragmentDistributionType::Single`, using this function will cause
/// assert to fail and should be skipped from the upper level.
///
/// The return value is the bitmap distribution after scaling, which covers all virtual node indexes
pub fn rebalance_actor_vnode(
    actors: &[CustomActorInfo],
    actors_to_remove: &BTreeSet<ActorId>,
    actors_to_create: &BTreeSet<ActorId>,
) -> HashMap<ActorId, Bitmap> {
    let actor_ids: BTreeSet<_> = actors.iter().map(|actor| actor.actor_id).collect();

    assert_eq!(actors_to_remove.difference(&actor_ids).count(), 0);
    assert_eq!(actors_to_create.intersection(&actor_ids).count(), 0);

    assert!(actors.len() >= actors_to_remove.len());

    let target_actor_count = actors.len() - actors_to_remove.len() + actors_to_create.len();
    assert!(target_actor_count > 0);

    // `vnode_bitmap` must be set on distributed fragments.
    let vnode_count = actors[0]
        .vnode_bitmap
        .as_ref()
        .expect("vnode bitmap unset")
        .len();

    // represents the balance of each actor, used to sort later
    #[derive(Debug)]
    struct Balance {
        actor_id: ActorId,
        balance: i32,
        builder: BitmapBuilder,
    }
    let (expected, mut remain) = vnode_count.div_rem(&target_actor_count);

    tracing::debug!(
        "expected {}, remain {}, prev actors {}, target actors {}",
        expected,
        remain,
        actors.len(),
        target_actor_count,
    );

    let (mut removed, mut rest): (Vec<_>, Vec<_>) = actors
        .iter()
        .map(|actor| {
            (
                actor.actor_id as ActorId,
                actor.vnode_bitmap.clone().expect("vnode bitmap unset"),
            )
        })
        .partition(|(actor_id, _)| actors_to_remove.contains(actor_id));

    let order_by_bitmap_desc =
        |(id_a, bitmap_a): &(ActorId, Bitmap), (id_b, bitmap_b): &(ActorId, Bitmap)| -> Ordering {
            bitmap_a
                .count_ones()
                .cmp(&bitmap_b.count_ones())
                .reverse()
                .then(id_a.cmp(id_b))
        };

    let builder_from_bitmap = |bitmap: &Bitmap| -> BitmapBuilder {
        let mut builder = BitmapBuilder::default();
        builder.append_bitmap(bitmap);
        builder
    };

    let (prev_expected, _) = vnode_count.div_rem(&actors.len());

    let prev_remain = removed
        .iter()
        .map(|(_, bitmap)| {
            assert!(bitmap.count_ones() >= prev_expected);
            bitmap.count_ones() - prev_expected
        })
        .sum::<usize>();

    removed.sort_by(order_by_bitmap_desc);
    rest.sort_by(order_by_bitmap_desc);

    let removed_balances = removed.into_iter().map(|(actor_id, bitmap)| Balance {
        actor_id,
        balance: bitmap.count_ones() as i32,
        builder: builder_from_bitmap(&bitmap),
    });

    let mut rest_balances = rest
        .into_iter()
        .map(|(actor_id, bitmap)| Balance {
            actor_id,
            balance: bitmap.count_ones() as i32 - expected as i32,
            builder: builder_from_bitmap(&bitmap),
        })
        .collect_vec();

    let mut created_balances = actors_to_create
        .iter()
        .map(|actor_id| Balance {
            actor_id: *actor_id,
            balance: -(expected as i32),
            builder: BitmapBuilder::zeroed(vnode_count),
        })
        .collect_vec();

    for balance in created_balances
        .iter_mut()
        .rev()
        .take(prev_remain)
        .chain(rest_balances.iter_mut())
    {
        if remain > 0 {
            balance.balance -= 1;
            remain -= 1;
        }
    }

    // consume the rest `remain`
    for balance in &mut created_balances {
        if remain > 0 {
            balance.balance -= 1;
            remain -= 1;
        }
    }

    assert_eq!(remain, 0);

    let mut v: VecDeque<_> = removed_balances
        .chain(rest_balances)
        .chain(created_balances)
        .collect();

    // We will return the full bitmap here after rebalancing,
    // if we want to return only the changed actors, filter balance = 0 here
    let mut result = HashMap::with_capacity(target_actor_count);

    for balance in &v {
        tracing::debug!(
            "actor {:5}\tbalance {:5}\tR[{:5}]\tC[{:5}]",
            balance.actor_id,
            balance.balance,
            actors_to_remove.contains(&balance.actor_id),
            actors_to_create.contains(&balance.actor_id)
        );
    }

    while !v.is_empty() {
        if v.len() == 1 {
            let single = v.pop_front().unwrap();
            assert_eq!(single.balance, 0);
            if !actors_to_remove.contains(&single.actor_id) {
                result.insert(single.actor_id, single.builder.finish());
            }

            continue;
        }

        let mut src = v.pop_front().unwrap();
        let mut dst = v.pop_back().unwrap();

        let n = min(abs(src.balance), abs(dst.balance));

        let mut moved = 0;
        for idx in (0..vnode_count).rev() {
            if moved >= n {
                break;
            }

            if src.builder.is_set(idx) {
                src.builder.set(idx, false);
                assert!(!dst.builder.is_set(idx));
                dst.builder.set(idx, true);
                moved += 1;
            }
        }

        src.balance -= n;
        dst.balance += n;

        if src.balance != 0 {
            v.push_front(src);
        } else if !actors_to_remove.contains(&src.actor_id) {
            result.insert(src.actor_id, src.builder.finish());
        }

        if dst.balance != 0 {
            v.push_back(dst);
        } else {
            result.insert(dst.actor_id, dst.builder.finish());
        }
    }

    result
}

#[derive(Debug, Clone, Copy)]
pub struct RescheduleOptions {
    /// Whether to resolve the upstream of `NoShuffle` when scaling. It will check whether all the reschedules in the no shuffle dependency tree are corresponding, and rewrite them to the root of the no shuffle dependency tree.
    pub resolve_no_shuffle_upstream: bool,

    /// Whether to skip creating new actors. If it is true, the scaling-out actors will not be created.
    pub skip_create_new_actors: bool,
}

pub type ScaleControllerRef = Arc<ScaleController>;

pub struct ScaleController {
    pub metadata_manager: MetadataManager,

    pub source_manager: SourceManagerRef,

    pub env: MetaSrvEnv,

    /// We will acquire lock during DDL to prevent scaling operations on jobs that are in the creating state.
    /// e.g., a MV cannot be rescheduled during foreground backfill.
    pub reschedule_lock: RwLock<()>,
}

impl ScaleController {
    pub fn new(
        metadata_manager: &MetadataManager,
        source_manager: SourceManagerRef,
        env: MetaSrvEnv,
    ) -> Self {
        Self {
            metadata_manager: metadata_manager.clone(),
            source_manager,
            env,
            reschedule_lock: RwLock::new(()),
        }
    }

    pub async fn integrity_check(&self) -> MetaResult<()> {
        self.metadata_manager
            .catalog_controller
            .integrity_check()
            .await
    }

    /// Build the context for rescheduling and do some validation for the request.
    async fn build_reschedule_context(
        &self,
        reschedule: &mut HashMap<FragmentId, WorkerReschedule>,
        options: RescheduleOptions,
        table_parallelisms: Option<&mut HashMap<TableId, TableParallelism>>,
    ) -> MetaResult<RescheduleContext> {
        let worker_nodes: HashMap<WorkerId, WorkerNode> = self
            .metadata_manager
            .list_active_streaming_compute_nodes()
            .await?
            .into_iter()
            .map(|worker_node| (worker_node.id as _, worker_node))
            .collect();

        if worker_nodes.is_empty() {
            bail!("no available compute node in the cluster");
        }

        // Check if we are trying to move a fragment to a node marked as unschedulable
        let unschedulable_worker_ids: HashSet<_> = worker_nodes
            .values()
            .filter(|w| {
                w.property
                    .as_ref()
                    .map(|property| property.is_unschedulable)
                    .unwrap_or(false)
            })
            .map(|worker| worker.id as WorkerId)
            .collect();

        for (fragment_id, reschedule) in &*reschedule {
            for (worker_id, change) in &reschedule.worker_actor_diff {
                if unschedulable_worker_ids.contains(worker_id) && change.is_positive() {
                    bail!(
                        "unable to move fragment {} to unschedulable worker {}",
                        fragment_id,
                        worker_id
                    );
                }
            }
        }

        // FIXME: the same as anther place calling `list_table_fragments` in scaling.
        // Index for StreamActor
        let mut actor_map = HashMap::new();
        // Index for Fragment
        let mut fragment_map = HashMap::new();
        // Index for actor status, including actor's worker id
        let mut actor_status = BTreeMap::new();
        let mut fragment_state = HashMap::new();
        let mut fragment_to_table = HashMap::new();

        async fn fulfill_index_by_fragment_ids(
            actor_map: &mut HashMap<u32, CustomActorInfo>,
            fragment_map: &mut HashMap<FragmentId, CustomFragmentInfo>,
            actor_status: &mut BTreeMap<ActorId, WorkerId>,
            fragment_state: &mut HashMap<FragmentId, State>,
            fragment_to_table: &mut HashMap<FragmentId, TableId>,
            mgr: &MetadataManager,
            fragment_ids: Vec<risingwave_meta_model::FragmentId>,
        ) -> Result<(), MetaError> {
            let RescheduleWorkingSet {
                fragments,
                actors,
                mut actor_dispatchers,
                fragment_downstreams: _,
                fragment_upstreams: _,
                related_jobs,
            } = mgr
                .catalog_controller
                .resolve_working_set_for_reschedule_fragments(fragment_ids)
                .await?;

            let mut fragment_actors: HashMap<
                risingwave_meta_model::FragmentId,
                Vec<CustomActorInfo>,
            > = HashMap::new();

            let mut expr_contexts = HashMap::new();
            for (
                _,
                actor::Model {
                    actor_id,
                    fragment_id,
                    status: _,
                    splits: _,
                    worker_id,
                    upstream_actor_ids,
                    vnode_bitmap,
                    expr_context,
                },
            ) in actors
            {
                let dispatchers = actor_dispatchers
                    .remove(&actor_id)
                    .unwrap_or_default()
                    .into_iter()
                    .map(PbDispatcher::from)
                    .collect();

                let actor_info = CustomActorInfo {
                    actor_id: actor_id as _,
                    fragment_id: fragment_id as _,
                    dispatcher: dispatchers,
                    upstream_actor_id: upstream_actor_ids
                        .into_inner()
                        .values()
                        .flatten()
                        .map(|id| *id as _)
                        .collect(),
                    vnode_bitmap: vnode_bitmap.map(|b| Bitmap::from(&b.to_protobuf())),
                };

                actor_map.insert(actor_id as _, actor_info.clone());

                fragment_actors
                    .entry(fragment_id as _)
                    .or_default()
                    .push(actor_info);

                actor_status.insert(actor_id as _, worker_id as WorkerId);

                expr_contexts.insert(actor_id as u32, expr_context);
            }

            for (
                _,
                fragment::Model {
                    fragment_id,
                    job_id,
                    fragment_type_mask,
                    distribution_type,
                    stream_node,
                    state_table_ids,
                    upstream_fragment_id,
                    vnode_count: _,
                },
            ) in fragments
            {
                let actors = fragment_actors
                    .remove(&(fragment_id as _))
                    .unwrap_or_default();

                let CustomActorInfo {
                    actor_id,
                    fragment_id,
                    dispatcher,
                    upstream_actor_id,
                    vnode_bitmap,
                } = actors.first().unwrap().clone();

                let fragment = CustomFragmentInfo {
                    fragment_id: fragment_id as _,
                    fragment_type_mask: fragment_type_mask as _,
                    distribution_type: distribution_type.into(),
                    state_table_ids: state_table_ids.into_u32_array(),
                    upstream_fragment_ids: upstream_fragment_id.into_u32_array(),
                    actor_template: PbStreamActor {
                        nodes: Some(stream_node.to_protobuf()),
                        actor_id,
                        fragment_id: fragment_id as _,
                        dispatcher,
                        upstream_actor_id,
                        vnode_bitmap: vnode_bitmap.map(|b| b.to_protobuf()),
                        // todo, we need to fill this part
                        mview_definition: "".to_string(),
                        expr_context: expr_contexts
                            .get(&actor_id)
                            .cloned()
                            .map(|expr_context| expr_context.to_protobuf()),
                    },
                    actors,
                };

                fragment_map.insert(fragment_id as _, fragment);

                fragment_to_table.insert(fragment_id as _, TableId::from(job_id as u32));

                let related_job = related_jobs.get(&job_id).expect("job not found");

                fragment_state.insert(
                    fragment_id,
                    table_fragments::PbState::from(related_job.job_status),
                );
            }
            Ok(())
        }
        let fragment_ids = reschedule.keys().map(|id| *id as _).collect();

        fulfill_index_by_fragment_ids(
            &mut actor_map,
            &mut fragment_map,
            &mut actor_status,
            &mut fragment_state,
            &mut fragment_to_table,
            &self.metadata_manager,
            fragment_ids,
        )
        .await?;

        // NoShuffle relation index
        let mut no_shuffle_source_fragment_ids = HashSet::new();
        let mut no_shuffle_target_fragment_ids = HashSet::new();

        Self::build_no_shuffle_relation_index(
            &actor_map,
            &mut no_shuffle_source_fragment_ids,
            &mut no_shuffle_target_fragment_ids,
        );

        if options.resolve_no_shuffle_upstream {
            let original_reschedule_keys = reschedule.keys().cloned().collect();

            Self::resolve_no_shuffle_upstream_fragments(
                reschedule,
                &fragment_map,
                &no_shuffle_source_fragment_ids,
                &no_shuffle_target_fragment_ids,
            )?;

            if let Some(table_parallelisms) = table_parallelisms {
                // We need to reiterate through the NO_SHUFFLE dependencies in order to ascertain which downstream table the custom modifications of the table have been propagated from.
                Self::resolve_no_shuffle_upstream_tables(
                    original_reschedule_keys,
                    &fragment_map,
                    &no_shuffle_source_fragment_ids,
                    &no_shuffle_target_fragment_ids,
                    &fragment_to_table,
                    table_parallelisms,
                )?;
            }
        }

        let mut fragment_dispatcher_map = HashMap::new();
        Self::build_fragment_dispatcher_index(&actor_map, &mut fragment_dispatcher_map);

        // Then, we collect all available upstreams
        let mut upstream_dispatchers: HashMap<
            ActorId,
            Vec<(FragmentId, DispatcherId, DispatcherType)>,
        > = HashMap::new();
        for stream_actor in actor_map.values() {
            for dispatcher in &stream_actor.dispatcher {
                for downstream_actor_id in &dispatcher.downstream_actor_id {
                    upstream_dispatchers
                        .entry(*downstream_actor_id as ActorId)
                        .or_default()
                        .push((
                            stream_actor.fragment_id as FragmentId,
                            dispatcher.dispatcher_id as DispatcherId,
                            dispatcher.r#type(),
                        ));
                }
            }
        }

        let mut stream_source_fragment_ids = HashSet::new();
        let mut stream_source_backfill_fragment_ids = HashSet::new();
        let mut no_shuffle_reschedule = HashMap::new();
        for (fragment_id, WorkerReschedule { worker_actor_diff }) in &*reschedule {
            let fragment = fragment_map
                .get(fragment_id)
                .ok_or_else(|| anyhow!("fragment {fragment_id} does not exist"))?;

            // Check if the reschedule is supported.
            match fragment_state[fragment_id] {
                table_fragments::State::Unspecified => unreachable!(),
                state @ table_fragments::State::Initial
                | state @ table_fragments::State::Creating => {
                    bail!(
                        "the materialized view of fragment {fragment_id} is in state {}",
                        state.as_str_name()
                    )
                }
                table_fragments::State::Created => {}
            }

            if no_shuffle_target_fragment_ids.contains(fragment_id) {
                bail!("rescheduling NoShuffle downstream fragment (maybe Chain fragment) is forbidden, please use NoShuffle upstream fragment (like Materialized fragment) to scale");
            }

            // For the relation of NoShuffle (e.g. Materialize and Chain), we need a special
            // treatment because the upstream and downstream of NoShuffle are always 1-1
            // correspondence, so we need to clone the reschedule plan to the downstream of all
            // cascading relations.
            if no_shuffle_source_fragment_ids.contains(fragment_id) {
                // This fragment is a NoShuffle's upstream.
                let mut queue: VecDeque<_> = fragment_dispatcher_map
                    .get(fragment_id)
                    .unwrap()
                    .keys()
                    .cloned()
                    .collect();

                while let Some(downstream_id) = queue.pop_front() {
                    if !no_shuffle_target_fragment_ids.contains(&downstream_id) {
                        continue;
                    }

                    if let Some(downstream_fragments) = fragment_dispatcher_map.get(&downstream_id)
                    {
                        let no_shuffle_downstreams = downstream_fragments
                            .iter()
                            .filter(|(_, ty)| **ty == DispatcherType::NoShuffle)
                            .map(|(fragment_id, _)| fragment_id);

                        queue.extend(no_shuffle_downstreams.copied());
                    }

                    no_shuffle_reschedule.insert(
                        downstream_id,
                        WorkerReschedule {
                            worker_actor_diff: worker_actor_diff.clone(),
                        },
                    );
                }
            }

            if (fragment.get_fragment_type_mask() & FragmentTypeFlag::Source as u32) != 0 {
                let stream_node = fragment.actor_template.nodes.as_ref().unwrap();
                if stream_node.find_stream_source().is_some() {
                    stream_source_fragment_ids.insert(*fragment_id);
                }
            }

            // Check if the reschedule plan is valid.
            let current_worker_ids = fragment
                .actors
                .iter()
                .map(|a| actor_status.get(&a.actor_id).cloned().unwrap())
                .collect::<HashSet<_>>();

            for (removed, change) in worker_actor_diff {
                if !current_worker_ids.contains(removed) && change.is_negative() {
                    bail!(
                        "no actor on the worker {} of fragment {}",
                        removed,
                        fragment_id
                    );
                }
            }

            let added_actor_count: usize = worker_actor_diff
                .values()
                .filter(|change| change.is_positive())
                .cloned()
                .map(|change| change as usize)
                .sum();

            let removed_actor_count: usize = worker_actor_diff
                .values()
                .filter(|change| change.is_positive())
                .cloned()
                .map(|v| v.unsigned_abs())
                .sum();

            match fragment.distribution_type() {
                FragmentDistributionType::Hash => {
                    if fragment.actors.len() + added_actor_count <= removed_actor_count {
                        bail!("can't remove all actors from fragment {}", fragment_id);
                    }
                }
                FragmentDistributionType::Single => {
                    if added_actor_count != removed_actor_count {
                        bail!("single distribution fragment only support migration");
                    }
                }
                FragmentDistributionType::Unspecified => unreachable!(),
            }
        }

        if !no_shuffle_reschedule.is_empty() {
            tracing::info!(
                "reschedule plan rewritten with NoShuffle reschedule {:?}",
                no_shuffle_reschedule
            );

            for noshuffle_downstream in no_shuffle_reschedule.keys() {
                let fragment = fragment_map.get(noshuffle_downstream).unwrap();
                // SourceScan is always a NoShuffle downstream, rescheduled together with the upstream Source.
                if (fragment.get_fragment_type_mask() & FragmentTypeFlag::SourceScan as u32) != 0 {
                    let stream_node = fragment.actor_template.nodes.as_ref().unwrap();
                    if stream_node.find_source_backfill().is_some() {
                        stream_source_backfill_fragment_ids.insert(fragment.fragment_id);
                    }
                }
            }
        }

        // Modifications for NoShuffle downstream.
        reschedule.extend(no_shuffle_reschedule.into_iter());

        Ok(RescheduleContext {
            actor_map,
            actor_status,
            fragment_map,
            upstream_dispatchers,
            stream_source_fragment_ids,
            stream_source_backfill_fragment_ids,
            no_shuffle_target_fragment_ids,
            no_shuffle_source_fragment_ids,
            fragment_dispatcher_map,
        })
    }

    /// From the high-level [`WorkerReschedule`] to the low-level reschedule plan [`Reschedule`].
    ///
    /// Returns `(reschedule_fragment, applied_reschedules)`
    /// - `reschedule_fragment`: the generated reschedule plan
    /// - `applied_reschedules`: the changes that need to be updated to the meta store (`pre_apply_reschedules`, only for V1).
    ///
    /// In [normal process of scaling](`GlobalStreamManager::reschedule_actors`), we use the returned values to
    /// build a [`Command::RescheduleFragment`], which will then flows through the barrier mechanism to perform scaling.
    /// Meta store is updated after the barrier is collected.
    ///
    /// During recovery, we don't need the barrier mechanism, and can directly use the returned values to update meta.
    pub(crate) async fn analyze_reschedule_plan(
        &self,
        mut reschedules: HashMap<FragmentId, WorkerReschedule>,
        options: RescheduleOptions,
        table_parallelisms: Option<&mut HashMap<TableId, TableParallelism>>,
    ) -> MetaResult<HashMap<FragmentId, Reschedule>> {
        tracing::debug!("build_reschedule_context, reschedules: {:#?}", reschedules);
        let ctx = self
            .build_reschedule_context(&mut reschedules, options, table_parallelisms)
            .await?;
        tracing::debug!("reschedule context: {:#?}", ctx);
        let reschedules = reschedules;

        // Here, the plan for both upstream and downstream of the NO_SHUFFLE Fragment should already have been populated.

        // Index of actors to create/remove
        // Fragment Id => ( Actor Id => Worker Id )
        let (fragment_actors_to_remove, fragment_actors_to_create) =
            self.arrange_reschedules(&reschedules, &ctx)?;

        let mut fragment_actor_bitmap = HashMap::new();
        for fragment_id in reschedules.keys() {
            if ctx.no_shuffle_target_fragment_ids.contains(fragment_id) {
                // skipping chain fragment, we need to clone the upstream materialize fragment's
                // mapping later
                continue;
            }

            let actors_to_create = fragment_actors_to_create
                .get(fragment_id)
                .map(|map| map.iter().map(|(actor_id, _)| *actor_id).collect())
                .unwrap_or_default();

            let actors_to_remove = fragment_actors_to_remove
                .get(fragment_id)
                .map(|map| map.iter().map(|(actor_id, _)| *actor_id).collect())
                .unwrap_or_default();

            let fragment = ctx.fragment_map.get(fragment_id).unwrap();

            match fragment.distribution_type() {
                FragmentDistributionType::Single => {
                    // Skip re-balancing action for single distribution (always None)
                    fragment_actor_bitmap
                        .insert(fragment.fragment_id as FragmentId, Default::default());
                }
                FragmentDistributionType::Hash => {
                    let actor_vnode = rebalance_actor_vnode(
                        &fragment.actors,
                        &actors_to_remove,
                        &actors_to_create,
                    );

                    fragment_actor_bitmap.insert(fragment.fragment_id as FragmentId, actor_vnode);
                }

                FragmentDistributionType::Unspecified => unreachable!(),
            }
        }

        // Index for fragment -> { actor -> worker_id } after reschedule.
        // Since we need to organize the upstream and downstream relationships of NoShuffle,
        // we need to organize the actor distribution after a scaling.
        let mut fragment_actors_after_reschedule = HashMap::with_capacity(reschedules.len());
        for fragment_id in reschedules.keys() {
            let fragment = ctx.fragment_map.get(fragment_id).unwrap();
            let mut new_actor_ids = BTreeMap::new();
            for actor in &fragment.actors {
                if let Some(actors_to_remove) = fragment_actors_to_remove.get(fragment_id) {
                    if actors_to_remove.contains_key(&actor.actor_id) {
                        continue;
                    }
                }
                let worker_id = ctx.actor_id_to_worker_id(&actor.actor_id)?;
                new_actor_ids.insert(actor.actor_id as ActorId, worker_id);
            }

            if let Some(actors_to_create) = fragment_actors_to_create.get(fragment_id) {
                for (actor_id, worker_id) in actors_to_create {
                    new_actor_ids.insert(*actor_id, *worker_id);
                }
            }

            assert!(
                !new_actor_ids.is_empty(),
                "should be at least one actor in fragment {} after rescheduling",
                fragment_id
            );

            fragment_actors_after_reschedule.insert(*fragment_id, new_actor_ids);
        }

        let fragment_actors_after_reschedule = fragment_actors_after_reschedule;

        // In order to maintain consistency with the original structure, the upstream and downstream
        // actors of NoShuffle need to be in the same worker slot and hold the same virtual nodes,
        // so for the actors after the upstream re-balancing, since we have sorted the actors of the same fragment by id on all workers,
        // we can identify the corresponding upstream actor with NO_SHUFFLE.
        // NOTE: There should be more asserts here to ensure correctness.
        fn arrange_no_shuffle_relation(
            ctx: &RescheduleContext,
            fragment_id: &FragmentId,
            upstream_fragment_id: &FragmentId,
            fragment_actors_after_reschedule: &HashMap<FragmentId, BTreeMap<ActorId, WorkerId>>,
            actor_group_map: &mut HashMap<ActorId, (FragmentId, ActorId)>,
            fragment_updated_bitmap: &mut HashMap<FragmentId, HashMap<ActorId, Bitmap>>,
            no_shuffle_upstream_actor_map: &mut HashMap<ActorId, HashMap<FragmentId, ActorId>>,
            no_shuffle_downstream_actors_map: &mut HashMap<ActorId, HashMap<FragmentId, ActorId>>,
        ) {
            if !ctx.no_shuffle_target_fragment_ids.contains(fragment_id) {
                return;
            }

            let fragment = &ctx.fragment_map[fragment_id];

            let upstream_fragment = &ctx.fragment_map[upstream_fragment_id];

            // build actor group map
            for upstream_actor in &upstream_fragment.actors {
                for dispatcher in &upstream_actor.dispatcher {
                    if let DispatcherType::NoShuffle = dispatcher.get_type().unwrap() {
                        let downstream_actor_id =
                            *dispatcher.downstream_actor_id.iter().exactly_one().unwrap();

                        // upstream is root
                        if !ctx
                            .no_shuffle_target_fragment_ids
                            .contains(upstream_fragment_id)
                        {
                            actor_group_map.insert(
                                upstream_actor.actor_id,
                                (upstream_fragment.fragment_id, upstream_actor.actor_id),
                            );
                            actor_group_map.insert(
                                downstream_actor_id,
                                (upstream_fragment.fragment_id, upstream_actor.actor_id),
                            );
                        } else {
                            let root_actor_id = actor_group_map[&upstream_actor.actor_id];

                            actor_group_map.insert(downstream_actor_id, root_actor_id);
                        }
                    }
                }
            }

            // If the upstream is a Singleton Fragment, there will be no Bitmap changes
            let upstream_fragment_bitmap = fragment_updated_bitmap
                .get(upstream_fragment_id)
                .cloned()
                .unwrap_or_default();

            // Question: Is it possible to have Hash Distribution Fragment but the Actor's bitmap remains unchanged?
            if upstream_fragment.distribution_type() == FragmentDistributionType::Single {
                assert!(
                    upstream_fragment_bitmap.is_empty(),
                    "single fragment should have no bitmap updates"
                );
            }

            let upstream_fragment_actor_map = fragment_actors_after_reschedule
                .get(upstream_fragment_id)
                .cloned()
                .unwrap();

            let fragment_actor_map = fragment_actors_after_reschedule
                .get(fragment_id)
                .cloned()
                .unwrap();

            let mut worker_reverse_index: HashMap<WorkerId, BTreeSet<_>> = HashMap::new();

            // first, find existing actor bitmap, copy them
            let mut fragment_bitmap = HashMap::new();

            for (actor_id, worker_id) in &fragment_actor_map {
                if let Some((root_fragment, root_actor_id)) = actor_group_map.get(actor_id) {
                    let root_bitmap = fragment_updated_bitmap
                        .get(root_fragment)
                        .expect("root fragment bitmap not found")
                        .get(root_actor_id)
                        .cloned()
                        .expect("root actor bitmap not found");

                    // Copy the bitmap
                    fragment_bitmap.insert(*actor_id, root_bitmap);

                    no_shuffle_upstream_actor_map
                        .entry(*actor_id as ActorId)
                        .or_default()
                        .insert(*upstream_fragment_id, *root_actor_id);
                    no_shuffle_downstream_actors_map
                        .entry(*root_actor_id)
                        .or_default()
                        .insert(*fragment_id, *actor_id);
                } else {
                    worker_reverse_index
                        .entry(*worker_id)
                        .or_default()
                        .insert(*actor_id);
                }
            }

            let mut upstream_worker_reverse_index: HashMap<WorkerId, BTreeSet<_>> = HashMap::new();

            for (actor_id, worker_id) in &upstream_fragment_actor_map {
                if !actor_group_map.contains_key(actor_id) {
                    upstream_worker_reverse_index
                        .entry(*worker_id)
                        .or_default()
                        .insert(*actor_id);
                }
            }

            // then, find the rest of the actors and copy the bitmap
            for (worker_id, actor_ids) in worker_reverse_index {
                let upstream_actor_ids = upstream_worker_reverse_index
                    .get(&worker_id)
                    .unwrap()
                    .clone();

                assert_eq!(actor_ids.len(), upstream_actor_ids.len());

                for (actor_id, upstream_actor_id) in actor_ids
                    .into_iter()
                    .zip_eq_debug(upstream_actor_ids.into_iter())
                {
                    match upstream_fragment_bitmap.get(&upstream_actor_id).cloned() {
                        None => {
                            // single fragment should have no bitmap updates (same as upstream)
                            assert_eq!(
                                upstream_fragment.distribution_type(),
                                FragmentDistributionType::Single
                            );
                        }
                        Some(bitmap) => {
                            // Copy the bitmap
                            fragment_bitmap.insert(actor_id, bitmap);
                        }
                    }

                    no_shuffle_upstream_actor_map
                        .entry(actor_id as ActorId)
                        .or_default()
                        .insert(*upstream_fragment_id, upstream_actor_id);
                    no_shuffle_downstream_actors_map
                        .entry(upstream_actor_id)
                        .or_default()
                        .insert(*fragment_id, actor_id);
                }
            }

            match fragment.distribution_type() {
                FragmentDistributionType::Hash => {}
                FragmentDistributionType::Single => {
                    // single distribution should update nothing
                    assert!(fragment_bitmap.is_empty());
                }
                FragmentDistributionType::Unspecified => unreachable!(),
            }

            if let Err(e) = fragment_updated_bitmap.try_insert(*fragment_id, fragment_bitmap) {
                assert_eq!(
                    e.entry.get(),
                    &e.value,
                    "bitmaps derived from different no-shuffle upstreams mismatch"
                );
            }

            // Visit downstream fragments recursively.
            if let Some(downstream_fragments) = ctx.fragment_dispatcher_map.get(fragment_id) {
                let no_shuffle_downstreams = downstream_fragments
                    .iter()
                    .filter(|(_, ty)| **ty == DispatcherType::NoShuffle)
                    .map(|(fragment_id, _)| fragment_id);

                for downstream_fragment_id in no_shuffle_downstreams {
                    arrange_no_shuffle_relation(
                        ctx,
                        downstream_fragment_id,
                        fragment_id,
                        fragment_actors_after_reschedule,
                        actor_group_map,
                        fragment_updated_bitmap,
                        no_shuffle_upstream_actor_map,
                        no_shuffle_downstream_actors_map,
                    );
                }
            }
        }

        let mut no_shuffle_upstream_actor_map = HashMap::new();
        let mut no_shuffle_downstream_actors_map = HashMap::new();
        let mut actor_group_map = HashMap::new();
        // For all roots in the upstream and downstream dependency trees of NoShuffle, recursively
        // find all correspondences
        for fragment_id in reschedules.keys() {
            if ctx.no_shuffle_source_fragment_ids.contains(fragment_id)
                && !ctx.no_shuffle_target_fragment_ids.contains(fragment_id)
            {
                if let Some(downstream_fragments) = ctx.fragment_dispatcher_map.get(fragment_id) {
                    for downstream_fragment_id in downstream_fragments.keys() {
                        arrange_no_shuffle_relation(
                            &ctx,
                            downstream_fragment_id,
                            fragment_id,
                            &fragment_actors_after_reschedule,
                            &mut actor_group_map,
                            &mut fragment_actor_bitmap,
                            &mut no_shuffle_upstream_actor_map,
                            &mut no_shuffle_downstream_actors_map,
                        );
                    }
                }
            }
        }

        tracing::debug!("actor group map {:?}", actor_group_map);

        let mut new_created_actors = HashMap::new();
        for fragment_id in reschedules.keys() {
            let actors_to_create = fragment_actors_to_create
                .get(fragment_id)
                .cloned()
                .unwrap_or_default();

            let fragment = &ctx.fragment_map[fragment_id];

            assert!(!fragment.actors.is_empty());

            for (actor_to_create, sample_actor) in actors_to_create
                .iter()
                .zip_eq_debug(repeat(&fragment.actor_template).take(actors_to_create.len()))
            {
                let new_actor_id = actor_to_create.0;
                let mut new_actor = sample_actor.clone();

                // This should be assigned before the `modify_actor_upstream_and_downstream` call,
                // because we need to use the new actor id to find the upstream and
                // downstream in the NoShuffle relationship
                new_actor.actor_id = *new_actor_id;

                Self::modify_actor_upstream_and_downstream(
                    &ctx,
                    &fragment_actors_to_remove,
                    &fragment_actors_to_create,
                    &fragment_actor_bitmap,
                    &no_shuffle_upstream_actor_map,
                    &no_shuffle_downstream_actors_map,
                    &mut new_actor,
                )?;

                if let Some(bitmap) = fragment_actor_bitmap
                    .get(fragment_id)
                    .and_then(|actor_bitmaps| actor_bitmaps.get(new_actor_id))
                {
                    new_actor.vnode_bitmap = Some(bitmap.to_protobuf());
                }

                new_created_actors.insert(*new_actor_id, new_actor);
            }
        }

        // For stream source & source backfill fragments, we need to reallocate the splits.
        // Because we are in the Pause state, so it's no problem to reallocate
        let mut fragment_actor_splits = HashMap::new();
        for fragment_id in reschedules.keys() {
            let actors_after_reschedule = &fragment_actors_after_reschedule[fragment_id];

            if ctx.stream_source_fragment_ids.contains(fragment_id) {
                let fragment = &ctx.fragment_map[fragment_id];

                let prev_actor_ids = fragment
                    .actors
                    .iter()
                    .map(|actor| actor.actor_id)
                    .collect_vec();

                let curr_actor_ids = actors_after_reschedule.keys().cloned().collect_vec();

                let actor_splits = self
                    .source_manager
                    .migrate_splits_for_source_actors(
                        *fragment_id,
                        &prev_actor_ids,
                        &curr_actor_ids,
                    )
                    .await?;

                tracing::debug!(
                    "source actor splits: {:?}, fragment_id: {}",
                    actor_splits,
                    fragment_id
                );
                fragment_actor_splits.insert(*fragment_id, actor_splits);
            }
        }
        // We use 2 iterations to make sure source actors are migrated first, and then align backfill actors
        if !ctx.stream_source_backfill_fragment_ids.is_empty() {
            for fragment_id in reschedules.keys() {
                let actors_after_reschedule = &fragment_actors_after_reschedule[fragment_id];

                if ctx
                    .stream_source_backfill_fragment_ids
                    .contains(fragment_id)
                {
                    let fragment = &ctx.fragment_map[fragment_id];

                    let curr_actor_ids = actors_after_reschedule.keys().cloned().collect_vec();

                    let actor_splits = self.source_manager.migrate_splits_for_backfill_actors(
                        *fragment_id,
                        &fragment.upstream_fragment_ids,
                        &curr_actor_ids,
                        &fragment_actor_splits,
                        &no_shuffle_upstream_actor_map,
                    )?;
                    tracing::debug!(
                        "source backfill actor splits: {:?}, fragment_id: {}",
                        actor_splits,
                        fragment_id
                    );
                    fragment_actor_splits.insert(*fragment_id, actor_splits);
                }
            }
        }

        // Generate fragment reschedule plan
        let mut reschedule_fragment: HashMap<FragmentId, Reschedule> =
            HashMap::with_capacity(reschedules.len());

        for (fragment_id, _) in reschedules {
            let mut actors_to_create: HashMap<_, Vec<_>> = HashMap::new();

            if let Some(actor_worker_maps) = fragment_actors_to_create.get(&fragment_id).cloned() {
                for (actor_id, worker_id) in actor_worker_maps {
                    actors_to_create
                        .entry(worker_id)
                        .or_default()
                        .push(actor_id);
                }
            }

            let actors_to_remove = fragment_actors_to_remove
                .get(&fragment_id)
                .cloned()
                .unwrap_or_default()
                .into_keys()
                .collect();

            let actors_after_reschedule = &fragment_actors_after_reschedule[&fragment_id];

            assert!(!actors_after_reschedule.is_empty());

            let fragment = &ctx.fragment_map[&fragment_id];

            let in_degree_types: HashSet<_> = fragment
                .upstream_fragment_ids
                .iter()
                .flat_map(|upstream_fragment_id| {
                    ctx.fragment_dispatcher_map
                        .get(upstream_fragment_id)
                        .and_then(|dispatcher_map| {
                            dispatcher_map.get(&fragment.fragment_id).cloned()
                        })
                })
                .collect();

            let upstream_dispatcher_mapping = match fragment.distribution_type() {
                FragmentDistributionType::Hash => {
                    if !in_degree_types.contains(&DispatcherType::Hash) {
                        None
                    } else {
                        // Changes of the bitmap must occur in the case of HashDistribution
                        Some(ActorMapping::from_bitmaps(
                            &fragment_actor_bitmap[&fragment_id],
                        ))
                    }
                }

                FragmentDistributionType::Single => {
                    assert!(fragment_actor_bitmap.get(&fragment_id).unwrap().is_empty());
                    None
                }
                FragmentDistributionType::Unspecified => unreachable!(),
            };

            let mut upstream_fragment_dispatcher_set = BTreeSet::new();

            for actor in &fragment.actors {
                if let Some(upstream_actor_tuples) = ctx.upstream_dispatchers.get(&actor.actor_id) {
                    for (upstream_fragment_id, upstream_dispatcher_id, upstream_dispatcher_type) in
                        upstream_actor_tuples
                    {
                        match upstream_dispatcher_type {
                            DispatcherType::Unspecified => unreachable!(),
                            DispatcherType::NoShuffle => {}
                            _ => {
                                upstream_fragment_dispatcher_set
                                    .insert((*upstream_fragment_id, *upstream_dispatcher_id));
                            }
                        }
                    }
                }
            }

            let downstream_fragment_ids = if let Some(downstream_fragments) =
                ctx.fragment_dispatcher_map.get(&fragment_id)
            {
                // Skip fragments' no-shuffle downstream, as there's no need to update the merger
                // (receiver) of a no-shuffle downstream
                downstream_fragments
                    .iter()
                    .filter(|(_, dispatcher_type)| *dispatcher_type != &DispatcherType::NoShuffle)
                    .map(|(fragment_id, _)| *fragment_id)
                    .collect_vec()
            } else {
                vec![]
            };

            let vnode_bitmap_updates = match fragment.distribution_type() {
                FragmentDistributionType::Hash => {
                    let mut vnode_bitmap_updates =
                        fragment_actor_bitmap.remove(&fragment_id).unwrap();

                    // We need to keep the bitmaps from changed actors only,
                    // otherwise the barrier will become very large with many actors
                    for actor_id in actors_after_reschedule.keys() {
                        assert!(vnode_bitmap_updates.contains_key(actor_id));

                        // retain actor
                        if let Some(actor) = ctx.actor_map.get(actor_id) {
                            let bitmap = vnode_bitmap_updates.get(actor_id).unwrap();

                            if let Some(prev_bitmap) = actor.vnode_bitmap.as_ref() {
                                if prev_bitmap.eq(bitmap) {
                                    vnode_bitmap_updates.remove(actor_id);
                                }
                            }
                        }
                    }

                    vnode_bitmap_updates
                }
                FragmentDistributionType::Single => HashMap::new(),
                FragmentDistributionType::Unspecified => unreachable!(),
            };

            let upstream_fragment_dispatcher_ids =
                upstream_fragment_dispatcher_set.into_iter().collect_vec();

            let actor_splits = fragment_actor_splits
                .get(&fragment_id)
                .cloned()
                .unwrap_or_default();

            reschedule_fragment.insert(
                fragment_id,
                Reschedule {
                    added_actors: actors_to_create,
                    removed_actors: actors_to_remove,
                    vnode_bitmap_updates,
                    upstream_fragment_dispatcher_ids,
                    upstream_dispatcher_mapping,
                    downstream_fragment_ids,
                    actor_splits,
                    newly_created_actors: vec![],
                },
            );
        }

        let mut fragment_created_actors = HashMap::new();
        for (fragment_id, actors_to_create) in &fragment_actors_to_create {
            let mut created_actors = HashMap::new();
            for (actor_id, worker_id) in actors_to_create {
                let actor = new_created_actors.get(actor_id).cloned().unwrap();
                created_actors.insert(
                    *actor_id,
                    (
                        actor,
                        ActorStatus {
                            location: PbActorLocation::from_worker(*worker_id as _),
                            state: ActorState::Inactive as i32,
                        },
                    ),
                );
            }

            fragment_created_actors.insert(*fragment_id, created_actors);
        }

        for (fragment_id, to_create) in &fragment_created_actors {
            let reschedule = reschedule_fragment.get_mut(fragment_id).unwrap();
            reschedule.newly_created_actors = to_create.values().cloned().collect();
        }
        tracing::debug!("analyze_reschedule_plan result: {:#?}", reschedule_fragment);

        Ok(reschedule_fragment)
    }

    #[expect(clippy::type_complexity)]
    fn arrange_reschedules(
        &self,
        reschedule: &HashMap<FragmentId, WorkerReschedule>,
        ctx: &RescheduleContext,
    ) -> MetaResult<(
        HashMap<FragmentId, BTreeMap<ActorId, WorkerId>>,
        HashMap<FragmentId, BTreeMap<ActorId, WorkerId>>,
    )> {
        let mut fragment_actors_to_remove = HashMap::with_capacity(reschedule.len());
        let mut fragment_actors_to_create = HashMap::with_capacity(reschedule.len());

        for (fragment_id, WorkerReschedule { worker_actor_diff }) in reschedule {
            let fragment = ctx.fragment_map.get(fragment_id).unwrap();

            // Actor Id => Worker Id
            let mut actors_to_remove = BTreeMap::new();
            let mut actors_to_create = BTreeMap::new();

            // NOTE(important): The value needs to be a BTreeSet to ensure that the actors on the worker are sorted in ascending order.
            let mut worker_to_actors = HashMap::new();

            for actor in &fragment.actors {
                let worker_id = ctx.actor_id_to_worker_id(&actor.actor_id).unwrap();
                worker_to_actors
                    .entry(worker_id)
                    .or_insert(BTreeSet::new())
                    .insert(actor.actor_id as ActorId);
            }

            let decreased_actor_count = worker_actor_diff
                .iter()
                .filter(|(_, change)| change.is_negative())
                .map(|(worker_id, change)| (worker_id, change.unsigned_abs()));

            for (worker_id, n) in decreased_actor_count {
                if let Some(actor_ids) = worker_to_actors.get(worker_id) {
                    if actor_ids.len() < n {
                        bail!("plan illegal, for fragment {}, worker {} only has {} actors, but needs to reduce {}",fragment_id, worker_id, actor_ids.len(), n);
                    }

                    let removed_actors: Vec<_> = actor_ids
                        .iter()
                        .skip(actor_ids.len().saturating_sub(n))
                        .cloned()
                        .collect();

                    for actor in removed_actors {
                        actors_to_remove.insert(actor, *worker_id);
                    }
                }
            }

            let increased_actor_count = worker_actor_diff
                .iter()
                .filter(|(_, change)| change.is_positive());

            for (worker, n) in increased_actor_count {
                for _ in 0..*n {
                    let id = self
                        .env
                        .id_gen_manager()
                        .generate_interval::<{ IdCategory::Actor }>(1)
                        as ActorId;
                    actors_to_create.insert(id, *worker);
                }
            }

            if !actors_to_remove.is_empty() {
                fragment_actors_to_remove.insert(*fragment_id as FragmentId, actors_to_remove);
            }

            if !actors_to_create.is_empty() {
                fragment_actors_to_create.insert(*fragment_id as FragmentId, actors_to_create);
            }
        }

        // sanity checking
        for actors_to_remove in fragment_actors_to_remove.values() {
            for actor_id in actors_to_remove.keys() {
                let actor = ctx.actor_map.get(actor_id).unwrap();
                for dispatcher in &actor.dispatcher {
                    if DispatcherType::NoShuffle == dispatcher.get_type().unwrap() {
                        let downstream_actor_id = dispatcher.downstream_actor_id.iter().exactly_one().expect("there should be only one downstream actor id in NO_SHUFFLE dispatcher");

                        let _should_exists = fragment_actors_to_remove
                            .get(&(dispatcher.dispatcher_id as FragmentId))
                            .expect("downstream fragment of NO_SHUFFLE relation should be in the removing map")
                            .get(downstream_actor_id)
                            .expect("downstream actor of NO_SHUFFLE relation should be in the removing map");
                    }
                }
            }
        }

        Ok((fragment_actors_to_remove, fragment_actors_to_create))
    }

    /// Modifies the upstream and downstream actors of the new created actor according to the
    /// overall changes, and is used to handle cascading updates
    fn modify_actor_upstream_and_downstream(
        ctx: &RescheduleContext,
        fragment_actors_to_remove: &HashMap<FragmentId, BTreeMap<ActorId, WorkerId>>,
        fragment_actors_to_create: &HashMap<FragmentId, BTreeMap<ActorId, WorkerId>>,
        fragment_actor_bitmap: &HashMap<FragmentId, HashMap<ActorId, Bitmap>>,
        no_shuffle_upstream_actor_map: &HashMap<ActorId, HashMap<FragmentId, ActorId>>,
        no_shuffle_downstream_actors_map: &HashMap<ActorId, HashMap<FragmentId, ActorId>>,
        new_actor: &mut PbStreamActor,
    ) -> MetaResult<()> {
        let fragment = &ctx.fragment_map[&new_actor.fragment_id];
        let mut applied_upstream_fragment_actor_ids = HashMap::new();

        for upstream_fragment_id in &fragment.upstream_fragment_ids {
            let upstream_dispatch_type = &ctx
                .fragment_dispatcher_map
                .get(upstream_fragment_id)
                .and_then(|map| map.get(&fragment.fragment_id))
                .unwrap();

            match upstream_dispatch_type {
                DispatcherType::Unspecified => unreachable!(),
                DispatcherType::Hash | DispatcherType::Broadcast | DispatcherType::Simple => {
                    let upstream_fragment = &ctx.fragment_map[upstream_fragment_id];
                    let mut upstream_actor_ids = upstream_fragment
                        .actors
                        .iter()
                        .map(|actor| actor.actor_id as ActorId)
                        .collect_vec();

                    if let Some(upstream_actors_to_remove) =
                        fragment_actors_to_remove.get(upstream_fragment_id)
                    {
                        upstream_actor_ids
                            .retain(|actor_id| !upstream_actors_to_remove.contains_key(actor_id));
                    }

                    if let Some(upstream_actors_to_create) =
                        fragment_actors_to_create.get(upstream_fragment_id)
                    {
                        upstream_actor_ids.extend(upstream_actors_to_create.keys().cloned());
                    }

                    applied_upstream_fragment_actor_ids.insert(
                        *upstream_fragment_id as FragmentId,
                        upstream_actor_ids.clone(),
                    );
                }
                DispatcherType::NoShuffle => {
                    let no_shuffle_upstream_actor_id = *no_shuffle_upstream_actor_map
                        .get(&new_actor.actor_id)
                        .and_then(|map| map.get(upstream_fragment_id))
                        .unwrap();

                    applied_upstream_fragment_actor_ids.insert(
                        *upstream_fragment_id as FragmentId,
                        vec![no_shuffle_upstream_actor_id as ActorId],
                    );
                }
            }
        }

        new_actor.upstream_actor_id = applied_upstream_fragment_actor_ids
            .values()
            .flatten()
            .cloned()
            .collect_vec();

        fn replace_merge_node_upstream(
            stream_node: &mut StreamNode,
            applied_upstream_fragment_actor_ids: &HashMap<FragmentId, Vec<ActorId>>,
        ) {
            if let Some(NodeBody::Merge(s)) = stream_node.node_body.as_mut() {
                s.upstream_actor_id = applied_upstream_fragment_actor_ids
                    .get(&s.upstream_fragment_id)
                    .cloned()
                    .unwrap();
            }

            for child in &mut stream_node.input {
                replace_merge_node_upstream(child, applied_upstream_fragment_actor_ids);
            }
        }

        if let Some(node) = new_actor.nodes.as_mut() {
            replace_merge_node_upstream(node, &applied_upstream_fragment_actor_ids);
        }

        // Update downstream actor ids
        for dispatcher in &mut new_actor.dispatcher {
            let downstream_fragment_id = dispatcher
                .downstream_actor_id
                .iter()
                .filter_map(|actor_id| ctx.actor_map.get(actor_id).map(|actor| actor.fragment_id))
                .dedup()
                .exactly_one()
                .unwrap() as FragmentId;

            let downstream_fragment_actors_to_remove =
                fragment_actors_to_remove.get(&downstream_fragment_id);
            let downstream_fragment_actors_to_create =
                fragment_actors_to_create.get(&downstream_fragment_id);

            match dispatcher.r#type() {
                d @ (DispatcherType::Hash | DispatcherType::Simple | DispatcherType::Broadcast) => {
                    if let Some(downstream_actors_to_remove) = downstream_fragment_actors_to_remove
                    {
                        dispatcher
                            .downstream_actor_id
                            .retain(|id| !downstream_actors_to_remove.contains_key(id));
                    }

                    if let Some(downstream_actors_to_create) = downstream_fragment_actors_to_create
                    {
                        dispatcher
                            .downstream_actor_id
                            .extend(downstream_actors_to_create.keys().cloned())
                    }

                    // There should be still exactly one downstream actor
                    if d == DispatcherType::Simple {
                        assert_eq!(dispatcher.downstream_actor_id.len(), 1);
                    }
                }
                DispatcherType::NoShuffle => {
                    assert_eq!(dispatcher.downstream_actor_id.len(), 1);
                    let downstream_actor_id = no_shuffle_downstream_actors_map
                        .get(&new_actor.actor_id)
                        .and_then(|map| map.get(&downstream_fragment_id))
                        .unwrap();
                    dispatcher.downstream_actor_id = vec![*downstream_actor_id as ActorId];
                }
                DispatcherType::Unspecified => unreachable!(),
            }

            if let Some(mapping) = dispatcher.hash_mapping.as_mut() {
                if let Some(downstream_updated_bitmap) =
                    fragment_actor_bitmap.get(&downstream_fragment_id)
                {
                    // If downstream scale in/out
                    *mapping = ActorMapping::from_bitmaps(downstream_updated_bitmap).to_protobuf();
                }
            }
        }

        Ok(())
    }

    pub async fn post_apply_reschedule(
        &self,
        reschedules: &HashMap<FragmentId, Reschedule>,
        table_parallelism: &HashMap<TableId, TableParallelism>,
    ) -> MetaResult<()> {
        // Update fragment info after rescheduling in meta store.
        self.metadata_manager
            .post_apply_reschedules(reschedules.clone(), table_parallelism.clone())
            .await?;

        // Update serving fragment info after rescheduling in meta store.
        if !reschedules.is_empty() {
            let workers = self
                .metadata_manager
                .list_active_serving_compute_nodes()
                .await?;
            let streaming_parallelisms = self
                .metadata_manager
                .running_fragment_parallelisms(Some(reschedules.keys().cloned().collect()))
                .await?;
            let serving_worker_slot_mapping = Arc::new(ServingVnodeMapping::default());
            let (upserted, failed) =
                serving_worker_slot_mapping.upsert(streaming_parallelisms, &workers);
            if !upserted.is_empty() {
                tracing::debug!(
                    "Update serving vnode mapping for fragments {:?}.",
                    upserted.keys()
                );
                self.env
                    .notification_manager()
                    .notify_frontend_without_version(
                        Operation::Update,
                        Info::ServingWorkerSlotMappings(FragmentWorkerSlotMappings {
                            mappings: to_fragment_worker_slot_mapping(&upserted),
                        }),
                    );
            }
            if !failed.is_empty() {
                tracing::debug!(
                    "Fail to update serving vnode mapping for fragments {:?}.",
                    failed
                );
                self.env
                    .notification_manager()
                    .notify_frontend_without_version(
                        Operation::Delete,
                        Info::ServingWorkerSlotMappings(FragmentWorkerSlotMappings {
                            mappings: to_deleted_fragment_worker_slot_mapping(&failed),
                        }),
                    );
            }
        }

        let mut stream_source_actor_splits = HashMap::new();
        let mut stream_source_dropped_actors = HashSet::new();

        for (fragment_id, reschedule) in reschedules {
            if !reschedule.actor_splits.is_empty() {
                stream_source_actor_splits
                    .insert(*fragment_id as FragmentId, reschedule.actor_splits.clone());
                stream_source_dropped_actors.extend(reschedule.removed_actors.clone());
            }
        }

        if !stream_source_actor_splits.is_empty() {
            self.source_manager
                .apply_source_change(
                    None,
                    None,
                    Some(stream_source_actor_splits),
                    Some(stream_source_dropped_actors),
                )
                .await;
        }

        Ok(())
    }

    pub async fn generate_table_resize_plan(
        &self,
        policy: TableResizePolicy,
    ) -> MetaResult<HashMap<FragmentId, WorkerReschedule>> {
        type VnodeCount = usize;

        let TableResizePolicy {
            worker_ids,
            table_parallelisms,
        } = policy;

        let workers = self
            .metadata_manager
            .list_active_streaming_compute_nodes()
            .await?;

        let unschedulable_worker_ids = Self::filter_unschedulable_workers(&workers);

        for worker_id in &worker_ids {
            if unschedulable_worker_ids.contains(worker_id) {
                bail!("Cannot include unschedulable worker {}", worker_id)
            }
        }

        let workers = workers
            .into_iter()
            .filter(|worker| worker_ids.contains(&(worker.id as _)))
            .collect::<Vec<_>>();

        let workers: HashMap<_, _> = workers
            .into_iter()
            .map(|worker| (worker.id, worker))
            .collect();

        let schedulable_worker_slots = workers
            .values()
            .map(|worker| (worker.id as WorkerId, worker.parallelism as usize))
            .collect::<BTreeMap<_, _>>();

        // index for no shuffle relation
        let mut no_shuffle_source_fragment_ids = HashSet::new();
        let mut no_shuffle_target_fragment_ids = HashSet::new();

        // index for fragment_id -> (distribution_type, vnode_count)
        let mut fragment_distribution_map = HashMap::new();
        // index for actor -> worker id
        let mut actor_location = HashMap::new();
        // index for table_id -> [fragment_id]
        let mut table_fragment_id_map = HashMap::new();
        // index for fragment_id -> [actor_id]
        let mut fragment_actor_id_map = HashMap::new();

        async fn build_index(
            no_shuffle_source_fragment_ids: &mut HashSet<FragmentId>,
            no_shuffle_target_fragment_ids: &mut HashSet<FragmentId>,
            fragment_distribution_map: &mut HashMap<
                FragmentId,
                (FragmentDistributionType, VnodeCount),
            >,
            actor_location: &mut HashMap<ActorId, WorkerId>,
            table_fragment_id_map: &mut HashMap<u32, HashSet<FragmentId>>,
            fragment_actor_id_map: &mut HashMap<FragmentId, HashSet<u32>>,
            mgr: &MetadataManager,
            table_ids: Vec<ObjectId>,
        ) -> Result<(), MetaError> {
            let RescheduleWorkingSet {
                fragments,
                actors,
                actor_dispatchers: _actor_dispatchers,
                fragment_downstreams,
                fragment_upstreams: _fragment_upstreams,
                related_jobs: _related_jobs,
            } = mgr
                .catalog_controller
                .resolve_working_set_for_reschedule_tables(table_ids)
                .await?;

            for (fragment_id, downstreams) in fragment_downstreams {
                for (downstream_fragment_id, dispatcher_type) in downstreams {
                    if let risingwave_meta_model::actor_dispatcher::DispatcherType::NoShuffle =
                        dispatcher_type
                    {
                        no_shuffle_source_fragment_ids.insert(fragment_id as FragmentId);
                        no_shuffle_target_fragment_ids.insert(downstream_fragment_id as FragmentId);
                    }
                }
            }

            for (fragment_id, fragment) in fragments {
                fragment_distribution_map.insert(
                    fragment_id as FragmentId,
                    (
                        FragmentDistributionType::from(fragment.distribution_type),
                        fragment.vnode_count as _,
                    ),
                );

                table_fragment_id_map
                    .entry(fragment.job_id as u32)
                    .or_default()
                    .insert(fragment_id as FragmentId);
            }

            for (actor_id, actor) in actors {
                actor_location.insert(actor_id as ActorId, actor.worker_id as WorkerId);
                fragment_actor_id_map
                    .entry(actor.fragment_id as FragmentId)
                    .or_default()
                    .insert(actor_id as ActorId);
            }

            Ok(())
        }

        let table_ids = table_parallelisms
            .keys()
            .map(|id| *id as ObjectId)
            .collect();

        build_index(
            &mut no_shuffle_source_fragment_ids,
            &mut no_shuffle_target_fragment_ids,
            &mut fragment_distribution_map,
            &mut actor_location,
            &mut table_fragment_id_map,
            &mut fragment_actor_id_map,
            &self.metadata_manager,
            table_ids,
        )
        .await?;
        tracing::debug!(
            ?worker_ids,
            ?table_parallelisms,
            ?no_shuffle_source_fragment_ids,
            ?no_shuffle_target_fragment_ids,
            ?fragment_distribution_map,
            ?actor_location,
            ?table_fragment_id_map,
            ?fragment_actor_id_map,
            "generate_table_resize_plan, after build_index"
        );

        let mut target_plan = HashMap::new();

        for (table_id, parallelism) in table_parallelisms {
            let fragment_map = table_fragment_id_map.remove(&table_id).unwrap();

            for fragment_id in fragment_map {
                // Currently, all of our NO_SHUFFLE relation propagations are only transmitted from upstream to downstream.
                if no_shuffle_target_fragment_ids.contains(&fragment_id) {
                    continue;
                }

                let mut fragment_slots: BTreeMap<WorkerId, usize> = BTreeMap::new();

                for actor_id in &fragment_actor_id_map[&fragment_id] {
                    let worker_id = actor_location[actor_id];
                    *fragment_slots.entry(worker_id).or_default() += 1;
                }

                let all_available_slots: usize = schedulable_worker_slots.values().cloned().sum();

                if all_available_slots == 0 {
                    bail!(
                        "No schedulable slots available for fragment {}",
                        fragment_id
                    );
                }

                let (dist, vnode_count) = fragment_distribution_map[&fragment_id];
                let max_parallelism = vnode_count;

                match dist {
                    FragmentDistributionType::Unspecified => unreachable!(),
                    FragmentDistributionType::Single => {
                        let (single_worker_id, should_be_one) = fragment_slots
                            .iter()
                            .exactly_one()
                            .expect("single fragment should have only one worker slot");

                        assert_eq!(*should_be_one, 1);

                        let units =
                            schedule_units_for_slots(&schedulable_worker_slots, 1, table_id)?;

                        let (chosen_target_worker_id, should_be_one) =
                            units.iter().exactly_one().ok().with_context(|| {
                                format!(
                                    "Cannot find a single target worker for fragment {fragment_id}"
                                )
                            })?;

                        assert_eq!(*should_be_one, 1);

                        if *chosen_target_worker_id == *single_worker_id {
                            tracing::debug!("single fragment {fragment_id} already on target worker {chosen_target_worker_id}");
                            continue;
                        }

                        target_plan.insert(
                            fragment_id,
                            WorkerReschedule {
                                worker_actor_diff: BTreeMap::from_iter(vec![
                                    (*chosen_target_worker_id, 1),
                                    (*single_worker_id, -1),
                                ]),
                            },
                        );
                    }
                    FragmentDistributionType::Hash => match parallelism {
                        TableParallelism::Adaptive => {
                            if all_available_slots > max_parallelism {
                                tracing::warn!("available parallelism for table {table_id} is larger than max parallelism, force limit to {max_parallelism}");
                                // force limit to `max_parallelism`
                                let target_worker_slots = schedule_units_for_slots(
                                    &schedulable_worker_slots,
                                    max_parallelism,
                                    table_id,
                                )?;

                                target_plan.insert(
                                    fragment_id,
                                    Self::diff_worker_slot_changes(
                                        &fragment_slots,
                                        &target_worker_slots,
                                    ),
                                );
                            } else {
                                target_plan.insert(
                                    fragment_id,
                                    Self::diff_worker_slot_changes(
                                        &fragment_slots,
                                        &schedulable_worker_slots,
                                    ),
                                );
                            }
                        }
                        TableParallelism::Fixed(mut n) => {
                            if n > max_parallelism {
                                tracing::warn!("specified parallelism {n} for table {table_id} is larger than max parallelism, force limit to {max_parallelism}");
                                n = max_parallelism
                            }

                            let target_worker_slots =
                                schedule_units_for_slots(&schedulable_worker_slots, n, table_id)?;

                            target_plan.insert(
                                fragment_id,
                                Self::diff_worker_slot_changes(
                                    &fragment_slots,
                                    &target_worker_slots,
                                ),
                            );
                        }
                        TableParallelism::Custom => {
                            // skipping for custom
                        }
                    },
                }
            }
        }

        target_plan.retain(|_, plan| !plan.worker_actor_diff.is_empty());
        tracing::debug!(
            ?target_plan,
            "generate_table_resize_plan finished target_plan"
        );
        Ok(target_plan)
    }

    pub(crate) fn filter_unschedulable_workers(workers: &[WorkerNode]) -> HashSet<WorkerId> {
        workers
            .iter()
            .filter(|worker| {
                worker
                    .property
                    .as_ref()
                    .map(|p| p.is_unschedulable)
                    .unwrap_or(false)
            })
            .map(|worker| worker.id as WorkerId)
            .collect()
    }

    fn diff_worker_slot_changes(
        fragment_worker_slots: &BTreeMap<WorkerId, usize>,
        target_worker_slots: &BTreeMap<WorkerId, usize>,
    ) -> WorkerReschedule {
        let mut increased_actor_count: BTreeMap<WorkerId, usize> = BTreeMap::new();
        let mut decreased_actor_count: BTreeMap<WorkerId, usize> = BTreeMap::new();

        for (&worker_id, &target_slots) in target_worker_slots {
            let &current_slots = fragment_worker_slots.get(&worker_id).unwrap_or(&0);

            if target_slots > current_slots {
                increased_actor_count.insert(worker_id, target_slots - current_slots);
            }
        }

        for (&worker_id, &current_slots) in fragment_worker_slots {
            let &target_slots = target_worker_slots.get(&worker_id).unwrap_or(&0);

            if current_slots > target_slots {
                decreased_actor_count.insert(worker_id, current_slots - target_slots);
            }
        }

        let worker_ids: HashSet<_> = increased_actor_count
            .keys()
            .chain(decreased_actor_count.keys())
            .cloned()
            .collect();

        let mut worker_actor_diff = BTreeMap::new();

        for worker_id in worker_ids {
            let increased = increased_actor_count.remove(&worker_id).unwrap_or(0) as isize;
            let decreased = decreased_actor_count.remove(&worker_id).unwrap_or(0) as isize;
            let change = increased - decreased;

            assert_ne!(change, 0);

            worker_actor_diff.insert(worker_id, change);
        }

        WorkerReschedule { worker_actor_diff }
    }

    fn build_no_shuffle_relation_index(
        actor_map: &HashMap<ActorId, CustomActorInfo>,
        no_shuffle_source_fragment_ids: &mut HashSet<FragmentId>,
        no_shuffle_target_fragment_ids: &mut HashSet<FragmentId>,
    ) {
        let mut fragment_cache = HashSet::new();
        for actor in actor_map.values() {
            if fragment_cache.contains(&actor.fragment_id) {
                continue;
            }

            for dispatcher in &actor.dispatcher {
                for downstream_actor_id in &dispatcher.downstream_actor_id {
                    if let Some(downstream_actor) = actor_map.get(downstream_actor_id) {
                        // Checking for no shuffle dispatchers
                        if dispatcher.r#type() == DispatcherType::NoShuffle {
                            no_shuffle_source_fragment_ids.insert(actor.fragment_id as FragmentId);
                            no_shuffle_target_fragment_ids
                                .insert(downstream_actor.fragment_id as FragmentId);
                        }
                    }
                }
            }

            fragment_cache.insert(actor.fragment_id);
        }
    }

    fn build_fragment_dispatcher_index(
        actor_map: &HashMap<ActorId, CustomActorInfo>,
        fragment_dispatcher_map: &mut HashMap<FragmentId, HashMap<FragmentId, DispatcherType>>,
    ) {
        for actor in actor_map.values() {
            for dispatcher in &actor.dispatcher {
                for downstream_actor_id in &dispatcher.downstream_actor_id {
                    if let Some(downstream_actor) = actor_map.get(downstream_actor_id) {
                        fragment_dispatcher_map
                            .entry(actor.fragment_id as FragmentId)
                            .or_default()
                            .insert(
                                downstream_actor.fragment_id as FragmentId,
                                dispatcher.r#type(),
                            );
                    }
                }
            }
        }
    }

    pub fn resolve_no_shuffle_upstream_tables(
        fragment_ids: HashSet<FragmentId>,
        fragment_map: &HashMap<FragmentId, CustomFragmentInfo>,
        no_shuffle_source_fragment_ids: &HashSet<FragmentId>,
        no_shuffle_target_fragment_ids: &HashSet<FragmentId>,
        fragment_to_table: &HashMap<FragmentId, TableId>,
        table_parallelisms: &mut HashMap<TableId, TableParallelism>,
    ) -> MetaResult<()> {
        let mut queue: VecDeque<FragmentId> = fragment_ids.iter().cloned().collect();

        let mut fragment_ids = fragment_ids;

        // We trace the upstreams of each downstream under the hierarchy until we reach the top
        // for every no_shuffle relation.
        while let Some(fragment_id) = queue.pop_front() {
            if !no_shuffle_target_fragment_ids.contains(&fragment_id) {
                continue;
            }

            // for upstream
            for upstream_fragment_id in &fragment_map[&fragment_id].upstream_fragment_ids {
                if !no_shuffle_source_fragment_ids.contains(upstream_fragment_id) {
                    continue;
                }

                let table_id = &fragment_to_table[&fragment_id];
                let upstream_table_id = &fragment_to_table[upstream_fragment_id];

                // Only custom parallelism will be propagated to the no shuffle upstream.
                if let Some(TableParallelism::Custom) = table_parallelisms.get(table_id) {
                    if let Some(upstream_table_parallelism) =
                        table_parallelisms.get(upstream_table_id)
                    {
                        if upstream_table_parallelism != &TableParallelism::Custom {
                            bail!(
                                "Cannot change upstream table {} from {:?} to {:?}",
                                upstream_table_id,
                                upstream_table_parallelism,
                                TableParallelism::Custom
                            )
                        }
                    } else {
                        table_parallelisms.insert(*upstream_table_id, TableParallelism::Custom);
                    }
                }

                fragment_ids.insert(*upstream_fragment_id);
                queue.push_back(*upstream_fragment_id);
            }
        }

        let downstream_fragment_ids = fragment_ids
            .iter()
            .filter(|fragment_id| no_shuffle_target_fragment_ids.contains(fragment_id));

        let downstream_table_ids = downstream_fragment_ids
            .map(|fragment_id| fragment_to_table.get(fragment_id).unwrap())
            .collect::<HashSet<_>>();

        table_parallelisms.retain(|table_id, _| !downstream_table_ids.contains(table_id));

        Ok(())
    }

    pub fn resolve_no_shuffle_upstream_fragments<T>(
        reschedule: &mut HashMap<FragmentId, T>,
        fragment_map: &HashMap<FragmentId, CustomFragmentInfo>,
        no_shuffle_source_fragment_ids: &HashSet<FragmentId>,
        no_shuffle_target_fragment_ids: &HashSet<FragmentId>,
    ) -> MetaResult<()>
    where
        T: Clone + Eq,
    {
        let mut queue: VecDeque<FragmentId> = reschedule.keys().cloned().collect();

        // We trace the upstreams of each downstream under the hierarchy until we reach the top
        // for every no_shuffle relation.
        while let Some(fragment_id) = queue.pop_front() {
            if !no_shuffle_target_fragment_ids.contains(&fragment_id) {
                continue;
            }

            // for upstream
            for upstream_fragment_id in &fragment_map[&fragment_id].upstream_fragment_ids {
                if !no_shuffle_source_fragment_ids.contains(upstream_fragment_id) {
                    continue;
                }

                let reschedule_plan = &reschedule[&fragment_id];

                if let Some(upstream_reschedule_plan) = reschedule.get(upstream_fragment_id) {
                    if upstream_reschedule_plan != reschedule_plan {
                        bail!("Inconsistent NO_SHUFFLE plan, check target worker ids of fragment {} and {}", fragment_id, upstream_fragment_id);
                    }

                    continue;
                }

                reschedule.insert(*upstream_fragment_id, reschedule_plan.clone());

                queue.push_back(*upstream_fragment_id);
            }
        }

        reschedule.retain(|fragment_id, _| !no_shuffle_target_fragment_ids.contains(fragment_id));

        Ok(())
    }
}

/// At present, for table level scaling, we use the strategy `TableResizePolicy`.
/// Currently, this is used as an internal interface, so it won’t be included in Protobuf.
#[derive(Debug)]
pub struct TableResizePolicy {
    pub(crate) worker_ids: BTreeSet<WorkerId>,
    pub(crate) table_parallelisms: HashMap<u32, TableParallelism>,
}

impl GlobalStreamManager {
    pub async fn reschedule_lock_read_guard(&self) -> RwLockReadGuard<'_, ()> {
        self.scale_controller.reschedule_lock.read().await
    }

    pub async fn reschedule_lock_write_guard(&self) -> RwLockWriteGuard<'_, ()> {
        self.scale_controller.reschedule_lock.write().await
    }

    /// The entrypoint of rescheduling actors.
    ///
    /// Used by:
    /// - The directly exposed low-level API `risingwave_meta_service::scale_service::ScaleService` (`risectl meta reschedule`)
    /// - High-level parallelism control API
    ///     * manual `ALTER [TABLE | INDEX | MATERIALIZED VIEW | SINK] SET PARALLELISM`
    ///     * automatic parallelism control for [`TableParallelism::Adaptive`] when worker nodes changed
    pub async fn reschedule_actors(
        &self,
        database_id: DatabaseId,
        reschedules: HashMap<FragmentId, WorkerReschedule>,
        options: RescheduleOptions,
        table_parallelism: Option<HashMap<TableId, TableParallelism>>,
    ) -> MetaResult<()> {
        let mut table_parallelism = table_parallelism;

        let reschedule_fragment = self
            .scale_controller
            .analyze_reschedule_plan(reschedules, options, table_parallelism.as_mut())
            .await?;

        tracing::debug!("reschedule plan: {:?}", reschedule_fragment);

        let up_down_stream_fragment: HashSet<_> = reschedule_fragment
            .iter()
            .flat_map(|(_, reschedule)| {
                reschedule
                    .upstream_fragment_dispatcher_ids
                    .iter()
                    .map(|(fragment_id, _)| *fragment_id)
                    .chain(reschedule.downstream_fragment_ids.iter().cloned())
            })
            .collect();

        let fragment_actors =
            try_join_all(up_down_stream_fragment.iter().map(|fragment_id| async {
                let actor_ids = self
                    .metadata_manager
                    .get_running_actors_of_fragment(*fragment_id)
                    .await?;
                Result::<_, MetaError>::Ok((*fragment_id, actor_ids))
            }))
            .await?
            .into_iter()
            .collect();

        let command = Command::RescheduleFragment {
            reschedules: reschedule_fragment,
            table_parallelism: table_parallelism.unwrap_or_default(),
            fragment_actors,
        };

        tracing::debug!("pausing tick lock in source manager");
        let _source_pause_guard = self.source_manager.paused.lock().await;

        self.barrier_scheduler
            .run_config_change_command_with_pause(database_id, command)
            .await?;

        tracing::info!("reschedule done");

        Ok(())
    }

    /// When new worker nodes joined, or the parallelism of existing worker nodes changed,
    /// examines if there are any jobs can be scaled, and scales them if found.
    ///
    /// This method will iterate over all `CREATED` jobs, and can be repeatedly called.
    ///
    /// Returns
    /// - `Ok(false)` if no jobs can be scaled;
    /// - `Ok(true)` if some jobs are scaled, and it is possible that there are more jobs can be scaled.
    async fn trigger_parallelism_control(&self) -> MetaResult<bool> {
        let background_streaming_jobs = self
            .metadata_manager
            .list_background_creating_jobs()
            .await?;

        if !background_streaming_jobs.is_empty() {
            tracing::debug!(
                "skipping parallelism control due to background jobs {:?}",
                background_streaming_jobs
            );
            // skip if there are background creating jobs
            return Ok(true);
        }

        tracing::info!("trigger parallelism control");

        let _reschedule_job_lock = self.reschedule_lock_write_guard().await;

        let table_parallelisms: HashMap<_, _> = {
            let streaming_parallelisms = self
                .metadata_manager
                .catalog_controller
                .get_all_created_streaming_parallelisms()
                .await?;

            streaming_parallelisms
                .into_iter()
                .map(|(table_id, parallelism)| {
                    let table_parallelism = match parallelism {
                        StreamingParallelism::Adaptive => TableParallelism::Adaptive,
                        StreamingParallelism::Fixed(n) => TableParallelism::Fixed(n),
                        StreamingParallelism::Custom => TableParallelism::Custom,
                    };

                    (table_id, table_parallelism)
                })
                .collect()
        };

        let workers = self
            .metadata_manager
            .cluster_controller
            .list_active_streaming_workers()
            .await?;

        let schedulable_worker_ids: BTreeSet<_> = workers
            .iter()
            .filter(|worker| {
                !worker
                    .property
                    .as_ref()
                    .map(|p| p.is_unschedulable)
                    .unwrap_or(false)
            })
            .map(|worker| worker.id as WorkerId)
            .collect();

        if table_parallelisms.is_empty() {
            tracing::info!("no streaming jobs for scaling, maybe an empty cluster");
            return Ok(false);
        }

        let batch_size = match self.env.opts.parallelism_control_batch_size {
            0 => table_parallelisms.len(),
            n => n,
        };

        tracing::info!(
            "total {} streaming jobs, batch size {}, schedulable worker ids: {:?}",
            table_parallelisms.len(),
            batch_size,
            schedulable_worker_ids
        );

        let batches: Vec<_> = table_parallelisms
            .into_iter()
            .chunks(batch_size)
            .into_iter()
            .map(|chunk| chunk.collect_vec())
            .collect();

        let mut reschedules = None;

        for batch in batches {
            let parallelisms: HashMap<_, _> =
                batch.into_iter().map(|(x, p)| (x as u32, p)).collect();
            // `table_parallelisms` contains ALL created jobs.
            // We rely on `generate_table_resize_plan` to check if there are
            // any jobs that can be scaled.
            let plan = self
                .scale_controller
                .generate_table_resize_plan(TableResizePolicy {
                    worker_ids: schedulable_worker_ids.clone(),
                    table_parallelisms: parallelisms.clone(),
                })
                .await?;

            if !plan.is_empty() {
                tracing::info!(
                    "reschedule plan generated for streaming jobs {:?}",
                    parallelisms
                );
                reschedules = Some(plan);
                break;
            }
        }

        let Some(reschedules) = reschedules else {
            tracing::info!("no reschedule plan generated");
            return Ok(false);
        };

        for (database_id, reschedules) in self
            .metadata_manager
            .split_fragment_map_by_database(reschedules)
            .await?
        {
            self.reschedule_actors(
                database_id,
                reschedules,
                RescheduleOptions {
                    resolve_no_shuffle_upstream: false,
                    skip_create_new_actors: false,
                },
                None,
            )
            .await?;
        }

        Ok(true)
    }

    /// Handles notification of worker node activation and deletion, and triggers parallelism control.
    async fn run(&self, mut shutdown_rx: Receiver<()>) {
        tracing::info!("starting automatic parallelism control monitor");

        let check_period =
            Duration::from_secs(self.env.opts.parallelism_control_trigger_period_sec);

        let mut ticker = tokio::time::interval_at(
            Instant::now()
                + Duration::from_secs(self.env.opts.parallelism_control_trigger_first_delay_sec),
            check_period,
        );
        ticker.set_missed_tick_behavior(MissedTickBehavior::Skip);

        // waiting for the first tick
        ticker.tick().await;

        let (local_notification_tx, mut local_notification_rx) =
            tokio::sync::mpsc::unbounded_channel();

        self.env
            .notification_manager()
            .insert_local_sender(local_notification_tx)
            .await;

        let worker_nodes = self
            .metadata_manager
            .list_active_streaming_compute_nodes()
            .await
            .expect("list active streaming compute nodes");

        let mut worker_cache: BTreeMap<_, _> = worker_nodes
            .into_iter()
            .map(|worker| (worker.id, worker))
            .collect();

        let mut should_trigger = false;

        loop {
            tokio::select! {
                biased;

                _ = &mut shutdown_rx => {
                    tracing::info!("Stream manager is stopped");
                    break;
                }

                _ = ticker.tick(), if should_trigger => {
                    let include_workers = worker_cache.keys().copied().collect_vec();

                    if include_workers.is_empty() {
                        tracing::debug!("no available worker nodes");
                        should_trigger = false;
                        continue;
                    }

                    match self.trigger_parallelism_control().await {
                        Ok(cont) => {
                            should_trigger = cont;
                        }
                        Err(e) => {
                            tracing::warn!(error = %e.as_report(), "Failed to trigger scale out, waiting for next tick to retry after {}s", ticker.period().as_secs());
                            ticker.reset();
                        }
                    }
                }

                notification = local_notification_rx.recv() => {
                    let notification = notification.expect("local notification channel closed in loop of stream manager");

                    // Only maintain the cache for streaming compute nodes.
                    let worker_is_streaming_compute = |worker: &WorkerNode| {
                        worker.get_type() == Ok(WorkerType::ComputeNode)
                            && worker.property.as_ref().unwrap().is_streaming
                    };

                    match notification {
                        LocalNotification::WorkerNodeActivated(worker) => {
                            if !worker_is_streaming_compute(&worker) {
                                continue;
                            }

                            tracing::info!(worker = worker.id, "worker activated notification received");

                            let prev_worker = worker_cache.insert(worker.id, worker.clone());

                            match prev_worker {
                                Some(prev_worker) if prev_worker.get_parallelism() != worker.get_parallelism()  => {
                                    tracing::info!(worker = worker.id, "worker parallelism changed");
                                    should_trigger = true;
                                }
                                None => {
                                    tracing::info!(worker = worker.id, "new worker joined");
                                    should_trigger = true;
                                }
                                _ => {}
                            }
                        }

                        // Since our logic for handling passive scale-in is within the barrier manager,
                        // there’s not much we can do here. All we can do is proactively remove the entries from our cache.
                        LocalNotification::WorkerNodeDeleted(worker) => {
                            if !worker_is_streaming_compute(&worker) {
                                continue;
                            }

                            match worker_cache.remove(&worker.id) {
                                Some(prev_worker) => {
                                    tracing::info!(worker = prev_worker.id, "worker removed from stream manager cache");
                                }
                                None => {
                                    tracing::warn!(worker = worker.id, "worker not found in stream manager cache, but it was removed");
                                }
                            }
                        }

                        _ => {}
                    }
                }
            }
        }
    }

    pub fn start_auto_parallelism_monitor(
        self: Arc<Self>,
    ) -> (JoinHandle<()>, oneshot::Sender<()>) {
        tracing::info!("Automatic parallelism scale-out is enabled for streaming jobs");
        let (shutdown_tx, shutdown_rx) = tokio::sync::oneshot::channel();
        let join_handle = tokio::spawn(async move {
            self.run(shutdown_rx).await;
        });

        (join_handle, shutdown_tx)
    }
}

pub fn schedule_units_for_slots(
    slots: &BTreeMap<WorkerId, usize>,
    total_unit_size: usize,
    salt: u32,
) -> MetaResult<BTreeMap<WorkerId, usize>> {
    let mut ch = ConsistentHashRing::new(salt);

    for (worker_id, parallelism) in slots {
        ch.add_worker(*worker_id as _, *parallelism as u32);
    }

    let target_distribution = ch.distribute_tasks(total_unit_size as u32)?;

    Ok(target_distribution
        .into_iter()
        .map(|(worker_id, task_count)| (worker_id as WorkerId, task_count as usize))
        .collect())
}

pub struct ConsistentHashRing {
    ring: BTreeMap<u64, u32>,
    weights: BTreeMap<u32, u32>,
    virtual_nodes: u32,
    salt: u32,
}

impl ConsistentHashRing {
    fn new(salt: u32) -> Self {
        ConsistentHashRing {
            ring: BTreeMap::new(),
            weights: BTreeMap::new(),
            virtual_nodes: 1024,
            salt,
        }
    }

    fn hash<T: Hash, S: Hash>(key: T, salt: S) -> u64 {
        let mut hasher = DefaultHasher::new();
        salt.hash(&mut hasher);
        key.hash(&mut hasher);
        hasher.finish()
    }

    fn add_worker(&mut self, id: u32, weight: u32) {
        let virtual_nodes_count = self.virtual_nodes;

        for i in 0..virtual_nodes_count {
            let virtual_node_key = (id, i);
            let hash = Self::hash(virtual_node_key, self.salt);
            self.ring.insert(hash, id);
        }

        self.weights.insert(id, weight);
    }

    fn distribute_tasks(&self, total_tasks: u32) -> MetaResult<BTreeMap<u32, u32>> {
        let total_weight = self.weights.values().sum::<u32>();

        let mut soft_limits = HashMap::new();
        for (worker_id, worker_capacity) in &self.weights {
            soft_limits.insert(
                *worker_id,
                (total_tasks as f64 * (*worker_capacity as f64 / total_weight as f64)).ceil()
                    as u32,
            );
        }

        let mut task_distribution: BTreeMap<u32, u32> = BTreeMap::new();
        let mut task_hashes = (0..total_tasks)
            .map(|task_idx| Self::hash(task_idx, self.salt))
            .collect_vec();

        // Sort task hashes to disperse them around the hash ring
        task_hashes.sort();

        for task_hash in task_hashes {
            let mut assigned = false;

            // Iterator that starts from the current task_hash or the next node in the ring
            let ring_range = self.ring.range(task_hash..).chain(self.ring.iter());

            for (_, &worker_id) in ring_range {
                let task_limit = soft_limits[&worker_id];

                let worker_task_count = task_distribution.entry(worker_id).or_insert(0);

                if *worker_task_count < task_limit {
                    *worker_task_count += 1;
                    assigned = true;
                    break;
                }
            }

            if !assigned {
                bail!("Could not distribute tasks due to capacity constraints.");
            }
        }

        Ok(task_distribution)
    }
}

#[cfg(test)]
mod tests {
    use super::*;

    const DEFAULT_SALT: u32 = 42;

    #[test]
    fn test_single_worker_capacity() {
        let mut ch = ConsistentHashRing::new(DEFAULT_SALT);
        ch.add_worker(1, 10);

        let total_tasks = 5;
        let task_distribution = ch.distribute_tasks(total_tasks).unwrap();

        assert_eq!(task_distribution.get(&1).cloned().unwrap_or(0), 5);
    }

    #[test]
    fn test_multiple_workers_even_distribution() {
        let mut ch = ConsistentHashRing::new(DEFAULT_SALT);

        ch.add_worker(1, 1);
        ch.add_worker(2, 1);
        ch.add_worker(3, 1);

        let total_tasks = 3;
        let task_distribution = ch.distribute_tasks(total_tasks).unwrap();

        for id in 1..=3 {
            assert_eq!(task_distribution.get(&id).cloned().unwrap_or(0), 1);
        }
    }

    #[test]
    fn test_weighted_distribution() {
        let mut ch = ConsistentHashRing::new(DEFAULT_SALT);

        ch.add_worker(1, 2);
        ch.add_worker(2, 3);
        ch.add_worker(3, 5);

        let total_tasks = 10;
        let task_distribution = ch.distribute_tasks(total_tasks).unwrap();

        assert_eq!(task_distribution.get(&1).cloned().unwrap_or(0), 2);
        assert_eq!(task_distribution.get(&2).cloned().unwrap_or(0), 3);
        assert_eq!(task_distribution.get(&3).cloned().unwrap_or(0), 5);
    }

    #[test]
    fn test_over_capacity() {
        let mut ch = ConsistentHashRing::new(DEFAULT_SALT);

        ch.add_worker(1, 1);
        ch.add_worker(2, 2);
        ch.add_worker(3, 3);

        let total_tasks = 10; // More tasks than the total weight
        let task_distribution = ch.distribute_tasks(total_tasks);

        assert!(task_distribution.is_ok());
    }

    #[test]
    fn test_balance_distribution() {
        for mut worker_capacity in 1..10 {
            for workers in 3..10 {
                let mut ring = ConsistentHashRing::new(DEFAULT_SALT);

                for worker_id in 0..workers {
                    ring.add_worker(worker_id, worker_capacity);
                }

                // Here we simulate a real situation where the actual parallelism cannot fill all the capacity.
                // This is to ensure an average distribution, for example, when three workers with 6 parallelism are assigned 9 tasks,
                // they should ideally get an exact distribution of 3, 3, 3 respectively.
                if worker_capacity % 2 == 0 {
                    worker_capacity /= 2;
                }

                let total_tasks = worker_capacity * workers;

                let task_distribution = ring.distribute_tasks(total_tasks).unwrap();

                for (_, v) in task_distribution {
                    assert_eq!(v, worker_capacity);
                }
            }
        }
    }
}