forest/rpc/methods/
chain.rs

1// Copyright 2019-2025 ChainSafe Systems
2// SPDX-License-Identifier: Apache-2.0, MIT
3
4mod types;
5use enumflags2::{BitFlags, make_bitflags};
6use types::*;
7
8#[cfg(test)]
9use crate::blocks::RawBlockHeader;
10use crate::blocks::{Block, CachingBlockHeader, Tipset, TipsetKey};
11use crate::chain::index::ResolveNullTipset;
12use crate::chain::{ChainStore, ExportOptions, FilecoinSnapshotVersion, HeadChange};
13use crate::cid_collections::CidHashSet;
14use crate::ipld::DfsIter;
15use crate::ipld::{CHAIN_EXPORT_STATUS, cancel_export, end_export, start_export};
16use crate::lotus_json::{HasLotusJson, LotusJson, lotus_json_with_self};
17#[cfg(test)]
18use crate::lotus_json::{assert_all_snapshots, assert_unchanged_via_json};
19use crate::message::{ChainMessage, SignedMessage};
20use crate::rpc::eth::{EthLog, eth_logs_with_filter, types::ApiHeaders, types::EthFilterSpec};
21use crate::rpc::f3::F3ExportLatestSnapshot;
22use crate::rpc::types::{ApiExportResult, ApiExportStatus, ApiTipsetKey, Event};
23use crate::rpc::{ApiPaths, Ctx, EthEventHandler, Permission, RpcMethod, ServerError};
24use crate::shim::clock::ChainEpoch;
25use crate::shim::error::ExitCode;
26use crate::shim::executor::Receipt;
27use crate::shim::message::Message;
28use crate::utils::db::CborStoreExt as _;
29use crate::utils::io::VoidAsyncWriter;
30use anyhow::{Context as _, Result};
31use cid::Cid;
32use fvm_ipld_blockstore::Blockstore;
33use fvm_ipld_encoding::{CborStore, RawBytes};
34use hex::ToHex;
35use ipld_core::ipld::Ipld;
36use jsonrpsee::types::Params;
37use jsonrpsee::types::error::ErrorObjectOwned;
38use num::BigInt;
39use schemars::JsonSchema;
40use serde::{Deserialize, Serialize};
41use sha2::Sha256;
42use std::fs::File;
43use std::{
44    collections::VecDeque,
45    path::PathBuf,
46    sync::{Arc, LazyLock},
47};
48use tokio::sync::{
49    Mutex,
50    broadcast::{self, Receiver as Subscriber},
51};
52use tokio::task::JoinHandle;
53use tokio_util::sync::CancellationToken;
54
55const HEAD_CHANNEL_CAPACITY: usize = 10;
56
57static CHAIN_EXPORT_LOCK: LazyLock<Mutex<Option<CancellationToken>>> =
58    LazyLock::new(|| Mutex::new(None));
59
60/// Subscribes to head changes from the chain store and broadcasts new blocks.
61///
62/// # Notes
63///
64/// Spawns an internal `tokio` task that can be aborted anytime via the returned `JoinHandle`,
65/// allowing manual cleanup if needed.
66pub(crate) fn new_heads<DB: Blockstore>(
67    data: &crate::rpc::RPCState<DB>,
68) -> (Subscriber<ApiHeaders>, JoinHandle<()>) {
69    let (sender, receiver) = broadcast::channel(HEAD_CHANNEL_CAPACITY);
70
71    let mut subscriber = data.chain_store().publisher().subscribe();
72
73    let handle = tokio::spawn(async move {
74        while let Ok(v) = subscriber.recv().await {
75            let headers = match v {
76                HeadChange::Apply(ts) => ApiHeaders(ts.block_headers().clone().into()),
77            };
78            if let Err(e) = sender.send(headers) {
79                tracing::error!("Failed to send headers: {}", e);
80                break;
81            }
82        }
83    });
84
85    (receiver, handle)
86}
87
88/// Subscribes to head changes from the chain store and broadcasts new `Ethereum` logs.
89///
90/// # Notes
91///
92/// Spawns an internal `tokio` task that can be aborted anytime via the returned `JoinHandle`,
93/// allowing manual cleanup if needed.
94pub(crate) fn logs<DB: Blockstore + Sync + Send + 'static>(
95    ctx: &Ctx<DB>,
96    filter: Option<EthFilterSpec>,
97) -> (Subscriber<Vec<EthLog>>, JoinHandle<()>) {
98    let (sender, receiver) = broadcast::channel(HEAD_CHANNEL_CAPACITY);
99
100    let mut subscriber = ctx.chain_store().publisher().subscribe();
101
102    let ctx = ctx.clone();
103
104    let handle = tokio::spawn(async move {
105        while let Ok(v) = subscriber.recv().await {
106            match v {
107                HeadChange::Apply(ts) => {
108                    match eth_logs_with_filter(&ctx, &ts, filter.clone(), None).await {
109                        Ok(logs) => {
110                            if !logs.is_empty()
111                                && let Err(e) = sender.send(logs)
112                            {
113                                tracing::error!(
114                                    "Failed to send logs for tipset {}: {}",
115                                    ts.key(),
116                                    e
117                                );
118                                break;
119                            }
120                        }
121                        Err(e) => {
122                            tracing::error!("Failed to fetch logs for tipset {}: {}", ts.key(), e);
123                        }
124                    }
125                }
126            }
127        }
128    });
129
130    (receiver, handle)
131}
132
133pub enum ChainGetFinalizedTipset {}
134impl RpcMethod<0> for ChainGetFinalizedTipset {
135    const NAME: &'static str = "Filecoin.ChainGetFinalizedTipSet";
136    const PARAM_NAMES: [&'static str; 0] = [];
137    const API_PATHS: BitFlags<ApiPaths> = make_bitflags!(ApiPaths::V1);
138    const PERMISSION: Permission = Permission::Read;
139    const DESCRIPTION: Option<&'static str> = Some(
140        "Returns the latest F3 finalized tipset, or falls back to EC finality if F3 is not operational on the node or if the F3 finalized tipset is further back than EC finalized tipset.",
141    );
142
143    type Params = ();
144    type Ok = Arc<Tipset>;
145
146    async fn handle(
147        ctx: Ctx<impl Blockstore + Send + Sync + 'static>,
148        (): Self::Params,
149    ) -> Result<Self::Ok, ServerError> {
150        let head = ctx.chain_store().heaviest_tipset();
151        let ec_finality_epoch = (head.epoch() - ctx.chain_config().policy.chain_finality).max(0);
152
153        // Either get the f3 finalized tipset or the ec finalized tipset
154        match get_f3_finality_tipset(&ctx, ec_finality_epoch).await {
155            Ok(f3_tipset) => {
156                tracing::debug!("Using F3 finalized tipset at epoch {}", f3_tipset.epoch());
157                Ok(f3_tipset)
158            }
159            Err(_) => {
160                // fallback to ec finality
161                tracing::warn!("F3 finalization unavailable, falling back to EC finality");
162                let ec_tipset = ctx.chain_index().tipset_by_height(
163                    ec_finality_epoch,
164                    head,
165                    ResolveNullTipset::TakeOlder,
166                )?;
167                Ok(ec_tipset)
168            }
169        }
170    }
171}
172
173// get f3 finalized tipset based on ec finality epoch
174async fn get_f3_finality_tipset<DB: Blockstore + Sync + Send + 'static>(
175    ctx: &Ctx<DB>,
176    ec_finality_epoch: ChainEpoch,
177) -> Result<Arc<Tipset>> {
178    let f3_finalized_cert = crate::rpc::f3::F3GetLatestCertificate::handle(ctx.clone(), ())
179        .await
180        .map_err(|e| anyhow::anyhow!("Failed to get F3 certificate: {}", e))?;
181
182    let f3_finalized_head = f3_finalized_cert.chain_head();
183    if f3_finalized_head.epoch < ec_finality_epoch {
184        return Err(anyhow::anyhow!(
185            "F3 finalized tipset epoch {} is further back than EC finalized tipset epoch {}",
186            f3_finalized_head.epoch,
187            ec_finality_epoch
188        ));
189    }
190
191    ctx.chain_index()
192        .load_required_tipset(&f3_finalized_head.key)
193        .map_err(|e| {
194            anyhow::anyhow!(
195                "Failed to load F3 finalized tipset at epoch {}: {}",
196                f3_finalized_head.epoch,
197                e
198            )
199        })
200}
201
202pub enum ChainGetMessage {}
203impl RpcMethod<1> for ChainGetMessage {
204    const NAME: &'static str = "Filecoin.ChainGetMessage";
205    const PARAM_NAMES: [&'static str; 1] = ["messageCid"];
206    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
207    const PERMISSION: Permission = Permission::Read;
208    const DESCRIPTION: Option<&'static str> = Some("Returns the message with the specified CID.");
209
210    type Params = (Cid,);
211    type Ok = Message;
212
213    async fn handle(
214        ctx: Ctx<impl Blockstore>,
215        (message_cid,): Self::Params,
216    ) -> Result<Self::Ok, ServerError> {
217        let chain_message: ChainMessage = ctx
218            .store()
219            .get_cbor(&message_cid)?
220            .with_context(|| format!("can't find message with cid {message_cid}"))?;
221        Ok(match chain_message {
222            ChainMessage::Signed(m) => m.into_message(),
223            ChainMessage::Unsigned(m) => m,
224        })
225    }
226}
227
228pub enum ChainGetEvents {}
229impl RpcMethod<1> for ChainGetEvents {
230    const NAME: &'static str = "Filecoin.ChainGetEvents";
231    const PARAM_NAMES: [&'static str; 1] = ["rootCid"];
232    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
233    const PERMISSION: Permission = Permission::Read;
234    const DESCRIPTION: Option<&'static str> =
235        Some("Returns the events under the given event AMT root CID.");
236
237    type Params = (Cid,);
238    type Ok = Vec<Event>;
239    async fn handle(
240        ctx: Ctx<impl Blockstore + Send + Sync + 'static>,
241        (root_cid,): Self::Params,
242    ) -> Result<Self::Ok, ServerError> {
243        let tsk = ctx
244            .state_manager
245            .chain_store()
246            .get_tipset_key(&root_cid)?
247            .with_context(|| format!("can't find events with cid {root_cid}"))?;
248
249        let ts = ctx.chain_store().load_required_tipset_or_heaviest(&tsk)?;
250
251        let events = EthEventHandler::collect_chain_events(&ctx, &ts, &root_cid).await?;
252
253        Ok(events)
254    }
255}
256
257pub enum ChainGetParentMessages {}
258impl RpcMethod<1> for ChainGetParentMessages {
259    const NAME: &'static str = "Filecoin.ChainGetParentMessages";
260    const PARAM_NAMES: [&'static str; 1] = ["blockCid"];
261    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
262    const PERMISSION: Permission = Permission::Read;
263    const DESCRIPTION: Option<&'static str> =
264        Some("Returns the messages included in the blocks of the parent tipset.");
265
266    type Params = (Cid,);
267    type Ok = Vec<ApiMessage>;
268
269    async fn handle(
270        ctx: Ctx<impl Blockstore>,
271        (block_cid,): Self::Params,
272    ) -> Result<Self::Ok, ServerError> {
273        let store = ctx.store();
274        let block_header: CachingBlockHeader = store
275            .get_cbor(&block_cid)?
276            .with_context(|| format!("can't find block header with cid {block_cid}"))?;
277        if block_header.epoch == 0 {
278            Ok(vec![])
279        } else {
280            let parent_tipset = Tipset::load_required(store, &block_header.parents)?;
281            load_api_messages_from_tipset(store, &parent_tipset)
282        }
283    }
284}
285
286pub enum ChainGetParentReceipts {}
287impl RpcMethod<1> for ChainGetParentReceipts {
288    const NAME: &'static str = "Filecoin.ChainGetParentReceipts";
289    const PARAM_NAMES: [&'static str; 1] = ["blockCid"];
290    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
291    const PERMISSION: Permission = Permission::Read;
292    const DESCRIPTION: Option<&'static str> =
293        Some("Returns the message receipts included in the blocks of the parent tipset.");
294
295    type Params = (Cid,);
296    type Ok = Vec<ApiReceipt>;
297
298    async fn handle(
299        ctx: Ctx<impl Blockstore>,
300        (block_cid,): Self::Params,
301    ) -> Result<Self::Ok, ServerError> {
302        let store = ctx.store();
303        let block_header: CachingBlockHeader = store
304            .get_cbor(&block_cid)?
305            .with_context(|| format!("can't find block header with cid {block_cid}"))?;
306        if block_header.epoch == 0 {
307            return Ok(vec![]);
308        }
309        let receipts = Receipt::get_receipts(store, block_header.message_receipts)
310            .map_err(|_| {
311                ErrorObjectOwned::owned::<()>(
312                    1,
313                    format!(
314                        "failed to root: ipld: could not find {}",
315                        block_header.message_receipts
316                    ),
317                    None,
318                )
319            })?
320            .iter()
321            .map(|r| ApiReceipt {
322                exit_code: r.exit_code().into(),
323                return_data: r.return_data(),
324                gas_used: r.gas_used(),
325                events_root: r.events_root(),
326            })
327            .collect();
328
329        Ok(receipts)
330    }
331}
332
333pub enum ChainGetMessagesInTipset {}
334impl RpcMethod<1> for ChainGetMessagesInTipset {
335    const NAME: &'static str = "Filecoin.ChainGetMessagesInTipset";
336    const PARAM_NAMES: [&'static str; 1] = ["tipsetKey"];
337    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
338    const PERMISSION: Permission = Permission::Read;
339
340    type Params = (ApiTipsetKey,);
341    type Ok = Vec<ApiMessage>;
342
343    async fn handle(
344        ctx: Ctx<impl Blockstore>,
345        (ApiTipsetKey(tipset_key),): Self::Params,
346    ) -> Result<Self::Ok, ServerError> {
347        let tipset = ctx
348            .chain_store()
349            .load_required_tipset_or_heaviest(&tipset_key)?;
350        load_api_messages_from_tipset(ctx.store(), &tipset)
351    }
352}
353
354pub enum ChainPruneSnapshot {}
355impl RpcMethod<1> for ChainPruneSnapshot {
356    const NAME: &'static str = "Forest.SnapshotGC";
357    const PARAM_NAMES: [&'static str; 1] = ["blocking"];
358    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
359    const PERMISSION: Permission = Permission::Admin;
360
361    type Params = (bool,);
362    type Ok = ();
363
364    async fn handle(
365        _ctx: Ctx<impl Blockstore + Send + Sync + 'static>,
366        (blocking,): Self::Params,
367    ) -> Result<Self::Ok, ServerError> {
368        if let Some(gc) = crate::daemon::GLOBAL_SNAPSHOT_GC.get() {
369            let progress_rx = gc.trigger()?;
370            while blocking && progress_rx.recv_async().await.is_ok() {}
371            Ok(())
372        } else {
373            Err(anyhow::anyhow!("snapshot gc is not enabled").into())
374        }
375    }
376}
377
378pub enum ForestChainExport {}
379impl RpcMethod<1> for ForestChainExport {
380    const NAME: &'static str = "Forest.ChainExport";
381    const PARAM_NAMES: [&'static str; 1] = ["params"];
382    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
383    const PERMISSION: Permission = Permission::Read;
384
385    type Params = (ForestChainExportParams,);
386    type Ok = ApiExportResult;
387
388    async fn handle(
389        ctx: Ctx<impl Blockstore + Send + Sync + 'static>,
390        (params,): Self::Params,
391    ) -> Result<Self::Ok, ServerError> {
392        let ForestChainExportParams {
393            version,
394            epoch,
395            recent_roots,
396            output_path,
397            tipset_keys: ApiTipsetKey(tsk),
398            skip_checksum,
399            dry_run,
400        } = params;
401
402        let token = CancellationToken::new();
403        {
404            let mut guard = CHAIN_EXPORT_LOCK.lock().await;
405            if guard.is_some() {
406                return Err(
407                    anyhow::anyhow!("A chain export is still in progress. Cancel it with the export-cancel subcommand if needed.").into(),
408                );
409            }
410            *guard = Some(token.clone());
411        }
412        start_export();
413
414        let head = ctx.chain_store().load_required_tipset_or_heaviest(&tsk)?;
415        let start_ts =
416            ctx.chain_index()
417                .tipset_by_height(epoch, head, ResolveNullTipset::TakeOlder)?;
418
419        let options = Some(ExportOptions {
420            skip_checksum,
421            seen: Default::default(),
422        });
423        let writer = if dry_run {
424            tokio_util::either::Either::Left(VoidAsyncWriter)
425        } else {
426            tokio_util::either::Either::Right(tokio::fs::File::create(&output_path).await?)
427        };
428        let result = match version {
429            FilecoinSnapshotVersion::V1 => {
430                let db = ctx.store_owned();
431
432                let chain_export =
433                    crate::chain::export::<Sha256>(&db, &start_ts, recent_roots, writer, options);
434
435                tokio::select! {
436                    result = chain_export => {
437                        result.map(|checksum_opt| ApiExportResult::Done(checksum_opt.map(|hash| hash.encode_hex())))
438                    },
439                    _ = token.cancelled() => {
440                        cancel_export();
441                        tracing::warn!("Snapshot export was cancelled");
442                        Ok(ApiExportResult::Cancelled)
443                    },
444                }
445            }
446            FilecoinSnapshotVersion::V2 => {
447                let db = ctx.store_owned();
448
449                let f3_snap_tmp_path = {
450                    let mut f3_snap_dir = output_path.clone();
451                    let mut builder = tempfile::Builder::new();
452                    let with_suffix = builder.suffix(".f3snap.bin");
453                    if f3_snap_dir.pop() {
454                        with_suffix.tempfile_in(&f3_snap_dir)
455                    } else {
456                        with_suffix.tempfile_in(".")
457                    }?
458                    .into_temp_path()
459                };
460                let f3_snap = {
461                    match F3ExportLatestSnapshot::run(f3_snap_tmp_path.display().to_string()).await
462                    {
463                        Ok(cid) => Some((cid, File::open(&f3_snap_tmp_path)?)),
464                        Err(e) => {
465                            tracing::error!("Failed to export F3 snapshot: {e}");
466                            None
467                        }
468                    }
469                };
470
471                let chain_export = crate::chain::export_v2::<Sha256, _>(
472                    &db,
473                    f3_snap,
474                    &start_ts,
475                    recent_roots,
476                    writer,
477                    options,
478                );
479
480                tokio::select! {
481                    result = chain_export => {
482                        result.map(|checksum_opt| ApiExportResult::Done(checksum_opt.map(|hash| hash.encode_hex())))
483                    },
484                    _ = token.cancelled() => {
485                        cancel_export();
486                        tracing::warn!("Snapshot export was cancelled");
487                        Ok(ApiExportResult::Cancelled)
488                    },
489                }
490            }
491        };
492        end_export();
493        // Clean up token
494        let mut guard = CHAIN_EXPORT_LOCK.lock().await;
495        *guard = None;
496        match result {
497            Ok(export_result) => Ok(export_result),
498            Err(e) => Err(anyhow::anyhow!(e).into()),
499        }
500    }
501}
502
503pub enum ForestChainExportStatus {}
504impl RpcMethod<0> for ForestChainExportStatus {
505    const NAME: &'static str = "Forest.ChainExportStatus";
506    const PARAM_NAMES: [&'static str; 0] = [];
507    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
508    const PERMISSION: Permission = Permission::Read;
509
510    type Params = ();
511    type Ok = ApiExportStatus;
512
513    async fn handle(_ctx: Ctx<impl Blockstore>, (): Self::Params) -> Result<Self::Ok, ServerError> {
514        let mutex = CHAIN_EXPORT_STATUS.lock();
515
516        let progress = if mutex.initial_epoch == 0 {
517            0.0
518        } else {
519            let p = 1.0 - ((mutex.epoch as f64) / (mutex.initial_epoch as f64));
520            if p.is_finite() {
521                p.clamp(0.0, 1.0)
522            } else {
523                0.0
524            }
525        };
526        // only two decimal places
527        let progress = (progress * 100.0).round() / 100.0;
528
529        let status = ApiExportStatus {
530            progress,
531            exporting: mutex.exporting,
532            cancelled: mutex.cancelled,
533            start_time: mutex.start_time,
534        };
535
536        Ok(status)
537    }
538}
539
540pub enum ForestChainExportCancel {}
541impl RpcMethod<0> for ForestChainExportCancel {
542    const NAME: &'static str = "Forest.ChainExportCancel";
543    const PARAM_NAMES: [&'static str; 0] = [];
544    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
545    const PERMISSION: Permission = Permission::Read;
546
547    type Params = ();
548    type Ok = bool;
549
550    async fn handle(_ctx: Ctx<impl Blockstore>, (): Self::Params) -> Result<Self::Ok, ServerError> {
551        if let Some(token) = CHAIN_EXPORT_LOCK.lock().await.as_ref() {
552            token.cancel();
553            return Ok(true);
554        }
555
556        Ok(false)
557    }
558}
559
560pub enum ForestChainExportDiff {}
561impl RpcMethod<1> for ForestChainExportDiff {
562    const NAME: &'static str = "Forest.ChainExportDiff";
563    const PARAM_NAMES: [&'static str; 1] = ["params"];
564    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
565    const PERMISSION: Permission = Permission::Read;
566
567    type Params = (ForestChainExportDiffParams,);
568    type Ok = ();
569
570    async fn handle(
571        ctx: Ctx<impl Blockstore + Send + Sync + 'static>,
572        (params,): Self::Params,
573    ) -> Result<Self::Ok, ServerError> {
574        let ForestChainExportDiffParams {
575            from,
576            to,
577            depth,
578            output_path,
579        } = params;
580
581        let _locked = CHAIN_EXPORT_LOCK.try_lock();
582        if _locked.is_err() {
583            return Err(
584                anyhow::anyhow!("Another chain export diff job is still in progress").into(),
585            );
586        }
587
588        let chain_finality = ctx.chain_config().policy.chain_finality;
589        if depth < chain_finality {
590            return Err(
591                anyhow::anyhow!(format!("depth must be greater than {chain_finality}")).into(),
592            );
593        }
594
595        let head = ctx.chain_store().heaviest_tipset();
596        let start_ts =
597            ctx.chain_index()
598                .tipset_by_height(from, head, ResolveNullTipset::TakeOlder)?;
599
600        crate::tool::subcommands::archive_cmd::do_export(
601            &ctx.store_owned(),
602            start_ts,
603            output_path,
604            None,
605            depth,
606            Some(to),
607            Some(chain_finality),
608            true,
609        )
610        .await?;
611
612        Ok(())
613    }
614}
615
616pub enum ChainExport {}
617impl RpcMethod<1> for ChainExport {
618    const NAME: &'static str = "Filecoin.ChainExport";
619    const PARAM_NAMES: [&'static str; 1] = ["params"];
620    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
621    const PERMISSION: Permission = Permission::Read;
622
623    type Params = (ChainExportParams,);
624    type Ok = ApiExportResult;
625
626    async fn handle(
627        ctx: Ctx<impl Blockstore + Send + Sync + 'static>,
628        (ChainExportParams {
629            epoch,
630            recent_roots,
631            output_path,
632            tipset_keys,
633            skip_checksum,
634            dry_run,
635        },): Self::Params,
636    ) -> Result<Self::Ok, ServerError> {
637        ForestChainExport::handle(
638            ctx,
639            (ForestChainExportParams {
640                version: FilecoinSnapshotVersion::V1,
641                epoch,
642                recent_roots,
643                output_path,
644                tipset_keys,
645                skip_checksum,
646                dry_run,
647            },),
648        )
649        .await
650    }
651}
652
653pub enum ChainReadObj {}
654impl RpcMethod<1> for ChainReadObj {
655    const NAME: &'static str = "Filecoin.ChainReadObj";
656    const PARAM_NAMES: [&'static str; 1] = ["cid"];
657    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
658    const PERMISSION: Permission = Permission::Read;
659    const DESCRIPTION: Option<&'static str> = Some(
660        "Reads IPLD nodes referenced by the specified CID from the chain blockstore and returns raw bytes.",
661    );
662
663    type Params = (Cid,);
664    type Ok = Vec<u8>;
665
666    async fn handle(
667        ctx: Ctx<impl Blockstore>,
668        (cid,): Self::Params,
669    ) -> Result<Self::Ok, ServerError> {
670        let bytes = ctx
671            .store()
672            .get(&cid)?
673            .with_context(|| format!("can't find object with cid={cid}"))?;
674        Ok(bytes)
675    }
676}
677
678pub enum ChainHasObj {}
679impl RpcMethod<1> for ChainHasObj {
680    const NAME: &'static str = "Filecoin.ChainHasObj";
681    const PARAM_NAMES: [&'static str; 1] = ["cid"];
682    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
683    const PERMISSION: Permission = Permission::Read;
684    const DESCRIPTION: Option<&'static str> =
685        Some("Checks if a given CID exists in the chain blockstore.");
686
687    type Params = (Cid,);
688    type Ok = bool;
689
690    async fn handle(
691        ctx: Ctx<impl Blockstore>,
692        (cid,): Self::Params,
693    ) -> Result<Self::Ok, ServerError> {
694        Ok(ctx.store().get(&cid)?.is_some())
695    }
696}
697
698/// Returns statistics about the graph referenced by 'obj'.
699/// If 'base' is also specified, then the returned stat will be a diff between the two objects.
700pub enum ChainStatObj {}
701impl RpcMethod<2> for ChainStatObj {
702    const NAME: &'static str = "Filecoin.ChainStatObj";
703    const PARAM_NAMES: [&'static str; 2] = ["obj_cid", "base_cid"];
704    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
705    const PERMISSION: Permission = Permission::Read;
706
707    type Params = (Cid, Option<Cid>);
708    type Ok = ObjStat;
709
710    async fn handle(
711        ctx: Ctx<impl Blockstore>,
712        (obj_cid, base_cid): Self::Params,
713    ) -> Result<Self::Ok, ServerError> {
714        let mut stats = ObjStat::default();
715        let mut seen = CidHashSet::default();
716        let mut walk = |cid, collect| {
717            let mut queue = VecDeque::new();
718            queue.push_back(cid);
719            while let Some(link_cid) = queue.pop_front() {
720                if !seen.insert(link_cid) {
721                    continue;
722                }
723                let data = ctx.store().get(&link_cid)?;
724                if let Some(data) = data {
725                    if collect {
726                        stats.links += 1;
727                        stats.size += data.len();
728                    }
729                    if matches!(link_cid.codec(), fvm_ipld_encoding::DAG_CBOR)
730                        && let Ok(ipld) =
731                            crate::utils::encoding::from_slice_with_fallback::<Ipld>(&data)
732                    {
733                        for ipld in DfsIter::new(ipld) {
734                            if let Ipld::Link(cid) = ipld {
735                                queue.push_back(cid);
736                            }
737                        }
738                    }
739                }
740            }
741            anyhow::Ok(())
742        };
743        if let Some(base_cid) = base_cid {
744            walk(base_cid, false)?;
745        }
746        walk(obj_cid, true)?;
747        Ok(stats)
748    }
749}
750
751pub enum ChainGetBlockMessages {}
752impl RpcMethod<1> for ChainGetBlockMessages {
753    const NAME: &'static str = "Filecoin.ChainGetBlockMessages";
754    const PARAM_NAMES: [&'static str; 1] = ["blockCid"];
755    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
756    const PERMISSION: Permission = Permission::Read;
757    const DESCRIPTION: Option<&'static str> =
758        Some("Returns all messages from the specified block.");
759
760    type Params = (Cid,);
761    type Ok = BlockMessages;
762
763    async fn handle(
764        ctx: Ctx<impl Blockstore>,
765        (block_cid,): Self::Params,
766    ) -> Result<Self::Ok, ServerError> {
767        let blk: CachingBlockHeader = ctx.store().get_cbor_required(&block_cid)?;
768        let (unsigned_cids, signed_cids) = crate::chain::read_msg_cids(ctx.store(), &blk)?;
769        let (bls_msg, secp_msg) =
770            crate::chain::block_messages_from_cids(ctx.store(), &unsigned_cids, &signed_cids)?;
771        let cids = unsigned_cids.into_iter().chain(signed_cids).collect();
772
773        let ret = BlockMessages {
774            bls_msg,
775            secp_msg,
776            cids,
777        };
778        Ok(ret)
779    }
780}
781
782pub enum ChainGetPath {}
783impl RpcMethod<2> for ChainGetPath {
784    const NAME: &'static str = "Filecoin.ChainGetPath";
785    const PARAM_NAMES: [&'static str; 2] = ["from", "to"];
786    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
787    const PERMISSION: Permission = Permission::Read;
788    const DESCRIPTION: Option<&'static str> =
789        Some("Returns the path between the two specified tipsets.");
790
791    type Params = (TipsetKey, TipsetKey);
792    type Ok = Vec<PathChange>;
793
794    async fn handle(
795        ctx: Ctx<impl Blockstore>,
796        (from, to): Self::Params,
797    ) -> Result<Self::Ok, ServerError> {
798        impl_chain_get_path(ctx.chain_store(), &from, &to).map_err(Into::into)
799    }
800}
801
802/// Find the path between two tipsets, as a series of [`PathChange`]s.
803///
804/// ```text
805/// 0 - A - B - C - D
806///     ^~~~~~~~> apply B, C
807///
808/// 0 - A - B - C - D
809///     <~~~~~~~^ revert C, B
810///
811///     <~~~~~~~~ revert C, B
812/// 0 - A - B  - C
813///     |
814///      -- B' - C'
815///      ~~~~~~~~> then apply B', C'
816/// ```
817///
818/// Exposes errors from the [`Blockstore`], and returns an error if there is no common ancestor.
819fn impl_chain_get_path(
820    chain_store: &ChainStore<impl Blockstore>,
821    from: &TipsetKey,
822    to: &TipsetKey,
823) -> anyhow::Result<Vec<PathChange>> {
824    let mut to_revert = chain_store
825        .load_required_tipset_or_heaviest(from)
826        .context("couldn't load `from`")?;
827    let mut to_apply = chain_store
828        .load_required_tipset_or_heaviest(to)
829        .context("couldn't load `to`")?;
830
831    let mut all_reverts = vec![];
832    let mut all_applies = vec![];
833
834    // This loop is guaranteed to terminate if the blockstore contain no cycles.
835    // This is currently computationally infeasible.
836    while to_revert != to_apply {
837        if to_revert.epoch() > to_apply.epoch() {
838            let next = chain_store
839                .load_required_tipset_or_heaviest(to_revert.parents())
840                .context("couldn't load ancestor of `from`")?;
841            all_reverts.push(to_revert);
842            to_revert = next;
843        } else {
844            let next = chain_store
845                .load_required_tipset_or_heaviest(to_apply.parents())
846                .context("couldn't load ancestor of `to`")?;
847            all_applies.push(to_apply);
848            to_apply = next;
849        }
850    }
851    Ok(all_reverts
852        .into_iter()
853        .map(PathChange::Revert)
854        .chain(all_applies.into_iter().rev().map(PathChange::Apply))
855        .collect())
856}
857
858/// Get tipset at epoch. Pick younger tipset if epoch points to a
859/// null-tipset. Only tipsets below the given `head` are searched. If `head`
860/// is null, the node will use the heaviest tipset.
861pub enum ChainGetTipSetByHeight {}
862impl RpcMethod<2> for ChainGetTipSetByHeight {
863    const NAME: &'static str = "Filecoin.ChainGetTipSetByHeight";
864    const PARAM_NAMES: [&'static str; 2] = ["height", "tipsetKey"];
865    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
866    const PERMISSION: Permission = Permission::Read;
867    const DESCRIPTION: Option<&'static str> = Some("Returns the tipset at the specified height.");
868
869    type Params = (ChainEpoch, ApiTipsetKey);
870    type Ok = Arc<Tipset>;
871
872    async fn handle(
873        ctx: Ctx<impl Blockstore>,
874        (height, ApiTipsetKey(tipset_key)): Self::Params,
875    ) -> Result<Self::Ok, ServerError> {
876        let ts = ctx
877            .chain_store()
878            .load_required_tipset_or_heaviest(&tipset_key)?;
879        let tss = ctx
880            .chain_index()
881            .tipset_by_height(height, ts, ResolveNullTipset::TakeOlder)?;
882        Ok(tss)
883    }
884}
885
886pub enum ChainGetTipSetAfterHeight {}
887impl RpcMethod<2> for ChainGetTipSetAfterHeight {
888    const NAME: &'static str = "Filecoin.ChainGetTipSetAfterHeight";
889    const PARAM_NAMES: [&'static str; 2] = ["height", "tipsetKey"];
890    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
891    const PERMISSION: Permission = Permission::Read;
892    const DESCRIPTION: Option<&'static str> = Some(
893        "Looks back and returns the tipset at the specified epoch.
894    If there are no blocks at the given epoch,
895    returns the first non-nil tipset at a later epoch.",
896    );
897
898    type Params = (ChainEpoch, ApiTipsetKey);
899    type Ok = Arc<Tipset>;
900
901    async fn handle(
902        ctx: Ctx<impl Blockstore>,
903        (height, ApiTipsetKey(tipset_key)): Self::Params,
904    ) -> Result<Self::Ok, ServerError> {
905        let ts = ctx
906            .chain_store()
907            .load_required_tipset_or_heaviest(&tipset_key)?;
908        let tss = ctx
909            .chain_index()
910            .tipset_by_height(height, ts, ResolveNullTipset::TakeNewer)?;
911        Ok(tss)
912    }
913}
914
915pub enum ChainGetGenesis {}
916impl RpcMethod<0> for ChainGetGenesis {
917    const NAME: &'static str = "Filecoin.ChainGetGenesis";
918    const PARAM_NAMES: [&'static str; 0] = [];
919    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
920    const PERMISSION: Permission = Permission::Read;
921
922    type Params = ();
923    type Ok = Option<Tipset>;
924
925    async fn handle(ctx: Ctx<impl Blockstore>, (): Self::Params) -> Result<Self::Ok, ServerError> {
926        let genesis = ctx.chain_store().genesis_block_header();
927        Ok(Some(Tipset::from(genesis)))
928    }
929}
930
931pub enum ChainHead {}
932impl RpcMethod<0> for ChainHead {
933    const NAME: &'static str = "Filecoin.ChainHead";
934    const PARAM_NAMES: [&'static str; 0] = [];
935    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
936    const PERMISSION: Permission = Permission::Read;
937    const DESCRIPTION: Option<&'static str> = Some("Returns the chain head (heaviest tipset).");
938
939    type Params = ();
940    type Ok = Arc<Tipset>;
941
942    async fn handle(ctx: Ctx<impl Blockstore>, (): Self::Params) -> Result<Self::Ok, ServerError> {
943        let heaviest = ctx.chain_store().heaviest_tipset();
944        Ok(heaviest)
945    }
946}
947
948pub enum ChainGetBlock {}
949impl RpcMethod<1> for ChainGetBlock {
950    const NAME: &'static str = "Filecoin.ChainGetBlock";
951    const PARAM_NAMES: [&'static str; 1] = ["blockCid"];
952    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
953    const PERMISSION: Permission = Permission::Read;
954    const DESCRIPTION: Option<&'static str> = Some("Returns the block with the specified CID.");
955
956    type Params = (Cid,);
957    type Ok = CachingBlockHeader;
958
959    async fn handle(
960        ctx: Ctx<impl Blockstore>,
961        (block_cid,): Self::Params,
962    ) -> Result<Self::Ok, ServerError> {
963        let blk: CachingBlockHeader = ctx.store().get_cbor_required(&block_cid)?;
964        Ok(blk)
965    }
966}
967
968pub enum ChainGetTipSet {}
969impl RpcMethod<1> for ChainGetTipSet {
970    const NAME: &'static str = "Filecoin.ChainGetTipSet";
971    const PARAM_NAMES: [&'static str; 1] = ["tipsetKey"];
972    const API_PATHS: BitFlags<ApiPaths> = make_bitflags!(ApiPaths::{ V0 | V1 });
973    const PERMISSION: Permission = Permission::Read;
974    const DESCRIPTION: Option<&'static str> = Some("Returns the tipset with the specified CID.");
975
976    type Params = (ApiTipsetKey,);
977    type Ok = Arc<Tipset>;
978
979    async fn handle(
980        ctx: Ctx<impl Blockstore>,
981        (ApiTipsetKey(tipset_key),): Self::Params,
982    ) -> Result<Self::Ok, ServerError> {
983        let ts = ctx
984            .chain_store()
985            .load_required_tipset_or_heaviest(&tipset_key)?;
986        Ok(ts)
987    }
988}
989
990pub enum ChainGetTipSetV2 {}
991impl RpcMethod<1> for ChainGetTipSetV2 {
992    const NAME: &'static str = "Filecoin.ChainGetTipSet";
993    const PARAM_NAMES: [&'static str; 1] = ["tipsetSelector"];
994    const API_PATHS: BitFlags<ApiPaths> = make_bitflags!(ApiPaths::{ V2 });
995    const PERMISSION: Permission = Permission::Read;
996    const DESCRIPTION: Option<&'static str> = Some("Returns the tipset with the specified CID.");
997
998    type Params = (ApiTipsetKey,);
999    type Ok = Arc<Tipset>;
1000
1001    async fn handle(_: Ctx<impl Blockstore>, _: Self::Params) -> Result<Self::Ok, ServerError> {
1002        Err(ServerError::unsupported_method())
1003    }
1004}
1005
1006pub enum ChainSetHead {}
1007impl RpcMethod<1> for ChainSetHead {
1008    const NAME: &'static str = "Filecoin.ChainSetHead";
1009    const PARAM_NAMES: [&'static str; 1] = ["tsk"];
1010    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
1011    const PERMISSION: Permission = Permission::Admin;
1012
1013    type Params = (TipsetKey,);
1014    type Ok = ();
1015
1016    async fn handle(
1017        ctx: Ctx<impl Blockstore>,
1018        (tsk,): Self::Params,
1019    ) -> Result<Self::Ok, ServerError> {
1020        // This is basically a port of the reference implementation at
1021        // https://github.com/filecoin-project/lotus/blob/v1.23.0/node/impl/full/chain.go#L321
1022
1023        let new_head = ctx.chain_index().load_required_tipset(&tsk)?;
1024        let mut current = ctx.chain_store().heaviest_tipset();
1025        while current.epoch() >= new_head.epoch() {
1026            for cid in current.key().to_cids() {
1027                ctx.chain_store().unmark_block_as_validated(&cid);
1028            }
1029            let parents = &current.block_headers().first().parents;
1030            current = ctx.chain_index().load_required_tipset(parents)?;
1031        }
1032        ctx.chain_store()
1033            .set_heaviest_tipset(new_head)
1034            .map_err(Into::into)
1035    }
1036}
1037
1038pub enum ChainGetMinBaseFee {}
1039impl RpcMethod<1> for ChainGetMinBaseFee {
1040    const NAME: &'static str = "Forest.ChainGetMinBaseFee";
1041    const PARAM_NAMES: [&'static str; 1] = ["lookback"];
1042    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
1043    const PERMISSION: Permission = Permission::Read;
1044
1045    type Params = (u32,);
1046    type Ok = String;
1047
1048    async fn handle(
1049        ctx: Ctx<impl Blockstore>,
1050        (lookback,): Self::Params,
1051    ) -> Result<Self::Ok, ServerError> {
1052        let mut current = ctx.chain_store().heaviest_tipset();
1053        let mut min_base_fee = current.block_headers().first().parent_base_fee.clone();
1054
1055        for _ in 0..lookback {
1056            let parents = &current.block_headers().first().parents;
1057            current = ctx.chain_index().load_required_tipset(parents)?;
1058
1059            min_base_fee =
1060                min_base_fee.min(current.block_headers().first().parent_base_fee.to_owned());
1061        }
1062
1063        Ok(min_base_fee.atto().to_string())
1064    }
1065}
1066
1067pub enum ChainTipSetWeight {}
1068impl RpcMethod<1> for ChainTipSetWeight {
1069    const NAME: &'static str = "Filecoin.ChainTipSetWeight";
1070    const PARAM_NAMES: [&'static str; 1] = ["tipsetKey"];
1071    const API_PATHS: BitFlags<ApiPaths> = ApiPaths::all();
1072    const PERMISSION: Permission = Permission::Read;
1073    const DESCRIPTION: Option<&'static str> = Some("Returns the weight of the specified tipset.");
1074
1075    type Params = (ApiTipsetKey,);
1076    type Ok = BigInt;
1077
1078    async fn handle(
1079        ctx: Ctx<impl Blockstore>,
1080        (ApiTipsetKey(tipset_key),): Self::Params,
1081    ) -> Result<Self::Ok, ServerError> {
1082        let ts = ctx
1083            .chain_store()
1084            .load_required_tipset_or_heaviest(&tipset_key)?;
1085        let weight = crate::fil_cns::weight(ctx.store(), &ts)?;
1086        Ok(weight)
1087    }
1088}
1089
1090pub const CHAIN_NOTIFY: &str = "Filecoin.ChainNotify";
1091pub(crate) fn chain_notify<DB: Blockstore>(
1092    _params: Params<'_>,
1093    data: &crate::rpc::RPCState<DB>,
1094) -> Subscriber<Vec<ApiHeadChange>> {
1095    let (sender, receiver) = broadcast::channel(HEAD_CHANNEL_CAPACITY);
1096
1097    // As soon as the channel is created, send the current tipset
1098    let current = data.chain_store().heaviest_tipset();
1099    let (change, tipset) = ("current".into(), current);
1100    sender
1101        .send(vec![ApiHeadChange { change, tipset }])
1102        .expect("receiver is not dropped");
1103
1104    let mut subscriber = data.chain_store().publisher().subscribe();
1105
1106    tokio::spawn(async move {
1107        // Skip first message
1108        let _ = subscriber.recv().await;
1109
1110        while let Ok(v) = subscriber.recv().await {
1111            let (change, tipset) = match v {
1112                HeadChange::Apply(ts) => ("apply".into(), ts),
1113            };
1114
1115            if sender.send(vec![ApiHeadChange { change, tipset }]).is_err() {
1116                break;
1117            }
1118        }
1119    });
1120    receiver
1121}
1122
1123fn load_api_messages_from_tipset(
1124    store: &impl Blockstore,
1125    tipset: &Tipset,
1126) -> Result<Vec<ApiMessage>, ServerError> {
1127    let full_tipset = tipset
1128        .fill_from_blockstore(store)
1129        .context("Failed to load full tipset")?;
1130    let blocks = full_tipset.into_blocks();
1131    let mut messages = vec![];
1132    let mut seen = CidHashSet::default();
1133    for Block {
1134        bls_messages,
1135        secp_messages,
1136        ..
1137    } in blocks
1138    {
1139        for message in bls_messages {
1140            let cid = message.cid();
1141            if seen.insert(cid) {
1142                messages.push(ApiMessage { cid, message });
1143            }
1144        }
1145
1146        for msg in secp_messages {
1147            let cid = msg.cid();
1148            if seen.insert(cid) {
1149                messages.push(ApiMessage {
1150                    cid,
1151                    message: msg.message,
1152                });
1153            }
1154        }
1155    }
1156
1157    Ok(messages)
1158}
1159
1160#[derive(Debug, Clone, PartialEq, Serialize, Deserialize, JsonSchema)]
1161pub struct BlockMessages {
1162    #[serde(rename = "BlsMessages", with = "crate::lotus_json")]
1163    #[schemars(with = "LotusJson<Vec<Message>>")]
1164    pub bls_msg: Vec<Message>,
1165    #[serde(rename = "SecpkMessages", with = "crate::lotus_json")]
1166    #[schemars(with = "LotusJson<Vec<SignedMessage>>")]
1167    pub secp_msg: Vec<SignedMessage>,
1168    #[serde(rename = "Cids", with = "crate::lotus_json")]
1169    #[schemars(with = "LotusJson<Vec<Cid>>")]
1170    pub cids: Vec<Cid>,
1171}
1172lotus_json_with_self!(BlockMessages);
1173
1174#[derive(Debug, Clone, Serialize, Deserialize, Eq, PartialEq, JsonSchema)]
1175#[serde(rename_all = "PascalCase")]
1176pub struct ApiReceipt {
1177    // Exit status of message execution
1178    pub exit_code: ExitCode,
1179    // `Return` value if the exit code is zero
1180    #[serde(rename = "Return", with = "crate::lotus_json")]
1181    #[schemars(with = "LotusJson<RawBytes>")]
1182    pub return_data: RawBytes,
1183    // Non-negative value of GasUsed
1184    pub gas_used: u64,
1185    #[serde(with = "crate::lotus_json")]
1186    #[schemars(with = "LotusJson<Option<Cid>>")]
1187    pub events_root: Option<Cid>,
1188}
1189
1190lotus_json_with_self!(ApiReceipt);
1191
1192#[derive(Serialize, Deserialize, JsonSchema, Clone, Debug, Eq, PartialEq)]
1193#[serde(rename_all = "PascalCase")]
1194pub struct ApiMessage {
1195    #[serde(with = "crate::lotus_json")]
1196    #[schemars(with = "LotusJson<Cid>")]
1197    pub cid: Cid,
1198    #[serde(with = "crate::lotus_json")]
1199    #[schemars(with = "LotusJson<Message>")]
1200    pub message: Message,
1201}
1202
1203lotus_json_with_self!(ApiMessage);
1204
1205#[derive(Debug, Clone, Serialize, Deserialize, JsonSchema)]
1206pub struct ForestChainExportParams {
1207    pub version: FilecoinSnapshotVersion,
1208    pub epoch: ChainEpoch,
1209    pub recent_roots: i64,
1210    pub output_path: PathBuf,
1211    #[schemars(with = "LotusJson<ApiTipsetKey>")]
1212    #[serde(with = "crate::lotus_json")]
1213    pub tipset_keys: ApiTipsetKey,
1214    pub skip_checksum: bool,
1215    pub dry_run: bool,
1216}
1217lotus_json_with_self!(ForestChainExportParams);
1218
1219#[derive(Debug, Clone, Serialize, Deserialize, JsonSchema)]
1220pub struct ForestChainExportDiffParams {
1221    pub from: ChainEpoch,
1222    pub to: ChainEpoch,
1223    pub depth: i64,
1224    pub output_path: PathBuf,
1225}
1226lotus_json_with_self!(ForestChainExportDiffParams);
1227
1228#[derive(Debug, Clone, Serialize, Deserialize, JsonSchema)]
1229pub struct ChainExportParams {
1230    pub epoch: ChainEpoch,
1231    pub recent_roots: i64,
1232    pub output_path: PathBuf,
1233    #[schemars(with = "LotusJson<ApiTipsetKey>")]
1234    #[serde(with = "crate::lotus_json")]
1235    pub tipset_keys: ApiTipsetKey,
1236    pub skip_checksum: bool,
1237    pub dry_run: bool,
1238}
1239lotus_json_with_self!(ChainExportParams);
1240
1241#[derive(PartialEq, Debug, Serialize, Deserialize, Clone, JsonSchema)]
1242#[serde(rename_all = "PascalCase")]
1243pub struct ApiHeadChange {
1244    #[serde(rename = "Type")]
1245    pub change: String,
1246    #[serde(rename = "Val", with = "crate::lotus_json")]
1247    #[schemars(with = "LotusJson<Tipset>")]
1248    pub tipset: Arc<Tipset>,
1249}
1250lotus_json_with_self!(ApiHeadChange);
1251
1252#[derive(PartialEq, Debug, Serialize, Deserialize, Clone, JsonSchema)]
1253#[serde(tag = "Type", content = "Val", rename_all = "snake_case")]
1254pub enum PathChange<T = Arc<Tipset>> {
1255    Revert(T),
1256    Apply(T),
1257}
1258impl HasLotusJson for PathChange {
1259    type LotusJson = PathChange<<Tipset as HasLotusJson>::LotusJson>;
1260
1261    #[cfg(test)]
1262    fn snapshots() -> Vec<(serde_json::Value, Self)> {
1263        use serde_json::json;
1264        vec![(
1265            json!({
1266                "Type": "revert",
1267                "Val": {
1268                    "Blocks": [
1269                        {
1270                            "BeaconEntries": null,
1271                            "ForkSignaling": 0,
1272                            "Height": 0,
1273                            "Messages": { "/": "baeaaaaa" },
1274                            "Miner": "f00",
1275                            "ParentBaseFee": "0",
1276                            "ParentMessageReceipts": { "/": "baeaaaaa" },
1277                            "ParentStateRoot": { "/":"baeaaaaa" },
1278                            "ParentWeight": "0",
1279                            "Parents": [{"/":"bafyreiaqpwbbyjo4a42saasj36kkrpv4tsherf2e7bvezkert2a7dhonoi"}],
1280                            "Timestamp": 0,
1281                            "WinPoStProof": null
1282                        }
1283                    ],
1284                    "Cids": [
1285                        { "/": "bafy2bzaceag62hjj3o43lf6oyeox3fvg5aqkgl5zagbwpjje3ajwg6yw4iixk" }
1286                    ],
1287                    "Height": 0
1288                }
1289            }),
1290            Self::Revert(Arc::new(Tipset::from(RawBlockHeader::default()))),
1291        )]
1292    }
1293
1294    fn into_lotus_json(self) -> Self::LotusJson {
1295        match self {
1296            PathChange::Revert(it) => {
1297                PathChange::Revert(Arc::unwrap_or_clone(it).into_lotus_json())
1298            }
1299            PathChange::Apply(it) => PathChange::Apply(Arc::unwrap_or_clone(it).into_lotus_json()),
1300        }
1301    }
1302
1303    fn from_lotus_json(lotus_json: Self::LotusJson) -> Self {
1304        match lotus_json {
1305            PathChange::Revert(it) => PathChange::Revert(Tipset::from_lotus_json(it).into()),
1306            PathChange::Apply(it) => PathChange::Apply(Tipset::from_lotus_json(it).into()),
1307        }
1308    }
1309}
1310
1311#[cfg(test)]
1312impl<T> quickcheck::Arbitrary for PathChange<T>
1313where
1314    T: quickcheck::Arbitrary,
1315{
1316    fn arbitrary(g: &mut quickcheck::Gen) -> Self {
1317        let inner = T::arbitrary(g);
1318        g.choose(&[PathChange::Apply(inner.clone()), PathChange::Revert(inner)])
1319            .unwrap()
1320            .clone()
1321    }
1322}
1323
1324#[test]
1325fn snapshots() {
1326    assert_all_snapshots::<PathChange>()
1327}
1328
1329#[cfg(test)]
1330quickcheck::quickcheck! {
1331    fn quickcheck(val: PathChange) -> () {
1332        assert_unchanged_via_json(val)
1333    }
1334}
1335
1336#[cfg(test)]
1337mod tests {
1338    use super::*;
1339    use PathChange::{Apply, Revert};
1340
1341    use crate::{
1342        blocks::{Chain4U, RawBlockHeader, chain4u},
1343        db::{
1344            MemoryDB,
1345            car::{AnyCar, ManyCar},
1346        },
1347        networks::{self, ChainConfig},
1348    };
1349
1350    #[test]
1351    fn revert_to_ancestor_linear() {
1352        let store = ChainStore::calibnet();
1353        chain4u! {
1354            in store.blockstore();
1355            [_genesis = store.genesis_block_header()]
1356            -> [a] -> [b] -> [c, d] -> [e]
1357        };
1358
1359        // simple
1360        assert_path_change(&store, b, a, [Revert(&[b])]);
1361
1362        // from multi-member tipset
1363        assert_path_change(&store, [c, d], a, [Revert(&[c, d][..]), Revert(&[b])]);
1364
1365        // to multi-member tipset
1366        assert_path_change(&store, e, [c, d], [Revert(e)]);
1367
1368        // over multi-member tipset
1369        assert_path_change(&store, e, b, [Revert(&[e][..]), Revert(&[c, d])]);
1370    }
1371
1372    /// Mirror how lotus handles passing an incomplete `TipsetKey`s.
1373    /// Tested on lotus `1.23.2`
1374    #[test]
1375    fn incomplete_tipsets() {
1376        let store = ChainStore::calibnet();
1377        chain4u! {
1378            in store.blockstore();
1379            [_genesis = store.genesis_block_header()]
1380            -> [a, b] -> [c] -> [d, _e] // this pattern 2 -> 1 -> 2 can be found at calibnet epoch 1369126
1381        };
1382
1383        // apply to descendant with incomplete `from`
1384        assert_path_change(
1385            &store,
1386            a,
1387            c,
1388            [
1389                Revert(&[a][..]), // revert the incomplete tipset
1390                Apply(&[a, b]),   // apply the complete one
1391                Apply(&[c]),      // apply the destination
1392            ],
1393        );
1394
1395        // apply to descendant with incomplete `to`
1396        assert_path_change(&store, c, d, [Apply(d)]);
1397
1398        // revert to ancestor with incomplete `from`
1399        assert_path_change(&store, d, c, [Revert(d)]);
1400
1401        // revert to ancestor with incomplete `to`
1402        assert_path_change(
1403            &store,
1404            c,
1405            a,
1406            [
1407                Revert(&[c][..]),
1408                Revert(&[a, b]), // revert the complete tipset
1409                Apply(&[a]),     // apply the incomplete one
1410            ],
1411        );
1412    }
1413
1414    #[test]
1415    fn apply_to_descendant_linear() {
1416        let store = ChainStore::calibnet();
1417        chain4u! {
1418            in store.blockstore();
1419            [_genesis = store.genesis_block_header()]
1420            -> [a] -> [b] -> [c, d] -> [e]
1421        };
1422
1423        // simple
1424        assert_path_change(&store, a, b, [Apply(&[b])]);
1425
1426        // from multi-member tipset
1427        assert_path_change(&store, [c, d], e, [Apply(e)]);
1428
1429        // to multi-member tipset
1430        assert_path_change(&store, b, [c, d], [Apply([c, d])]);
1431
1432        // over multi-member tipset
1433        assert_path_change(&store, b, e, [Apply(&[c, d][..]), Apply(&[e])]);
1434    }
1435
1436    #[test]
1437    fn cross_fork_simple() {
1438        let store = ChainStore::calibnet();
1439        chain4u! {
1440            in store.blockstore();
1441            [_genesis = store.genesis_block_header()]
1442            -> [a] -> [b1] -> [c1]
1443        };
1444        chain4u! {
1445            from [a] in store.blockstore();
1446            [b2] -> [c2]
1447        };
1448
1449        // same height
1450        assert_path_change(&store, b1, b2, [Revert(b1), Apply(b2)]);
1451
1452        // different height
1453        assert_path_change(&store, b1, c2, [Revert(b1), Apply(b2), Apply(c2)]);
1454
1455        let _ = (a, c1);
1456    }
1457
1458    impl ChainStore<Chain4U<ManyCar>> {
1459        fn _load(genesis_car: &'static [u8], genesis_cid: Cid) -> Self {
1460            let db = Arc::new(Chain4U::with_blockstore(
1461                ManyCar::new(MemoryDB::default())
1462                    .with_read_only(AnyCar::new(genesis_car).unwrap())
1463                    .unwrap(),
1464            ));
1465            let genesis_block_header = db.get_cbor(&genesis_cid).unwrap().unwrap();
1466            ChainStore::new(
1467                db,
1468                Arc::new(MemoryDB::default()),
1469                Arc::new(MemoryDB::default()),
1470                Arc::new(MemoryDB::default()),
1471                Arc::new(ChainConfig::calibnet()),
1472                genesis_block_header,
1473            )
1474            .unwrap()
1475        }
1476        pub fn calibnet() -> Self {
1477            Self::_load(
1478                networks::calibnet::DEFAULT_GENESIS,
1479                *networks::calibnet::GENESIS_CID,
1480            )
1481        }
1482    }
1483
1484    /// Utility for writing ergonomic tests
1485    trait MakeTipset {
1486        fn make_tipset(self) -> Tipset;
1487    }
1488
1489    impl MakeTipset for &RawBlockHeader {
1490        fn make_tipset(self) -> Tipset {
1491            Tipset::from(CachingBlockHeader::new(self.clone()))
1492        }
1493    }
1494
1495    impl<const N: usize> MakeTipset for [&RawBlockHeader; N] {
1496        fn make_tipset(self) -> Tipset {
1497            self.as_slice().make_tipset()
1498        }
1499    }
1500
1501    impl<const N: usize> MakeTipset for &[&RawBlockHeader; N] {
1502        fn make_tipset(self) -> Tipset {
1503            self.as_slice().make_tipset()
1504        }
1505    }
1506
1507    impl MakeTipset for &[&RawBlockHeader] {
1508        fn make_tipset(self) -> Tipset {
1509            Tipset::new(self.iter().cloned().cloned()).unwrap()
1510        }
1511    }
1512
1513    #[track_caller]
1514    fn assert_path_change<T: MakeTipset>(
1515        store: &ChainStore<impl Blockstore>,
1516        from: impl MakeTipset,
1517        to: impl MakeTipset,
1518        expected: impl IntoIterator<Item = PathChange<T>>,
1519    ) {
1520        fn print(path_change: &PathChange) {
1521            let it = match path_change {
1522                Revert(it) => {
1523                    print!("Revert(");
1524                    it
1525                }
1526                Apply(it) => {
1527                    print!(" Apply(");
1528                    it
1529                }
1530            };
1531            println!(
1532                "epoch = {}, key.cid = {})",
1533                it.epoch(),
1534                it.key().cid().unwrap()
1535            )
1536        }
1537
1538        let actual =
1539            impl_chain_get_path(store, from.make_tipset().key(), to.make_tipset().key()).unwrap();
1540        let expected = expected
1541            .into_iter()
1542            .map(|change| match change {
1543                PathChange::Revert(it) => PathChange::Revert(Arc::new(it.make_tipset())),
1544                PathChange::Apply(it) => PathChange::Apply(Arc::new(it.make_tipset())),
1545            })
1546            .collect::<Vec<_>>();
1547        if expected != actual {
1548            println!("SUMMARY");
1549            println!("=======");
1550            println!("expected:");
1551            for it in &expected {
1552                print(it)
1553            }
1554            println!();
1555            println!("actual:");
1556            for it in &actual {
1557                print(it)
1558            }
1559            println!("=======\n")
1560        }
1561        assert_eq!(
1562            expected, actual,
1563            "expected change (left) does not match actual change (right)"
1564        )
1565    }
1566}