referrerpolicy=no-referrer-when-downgrade

sc_consensus_grandpa/
import.rs

1// This file is part of Substrate.
2
3// Copyright (C) Parity Technologies (UK) Ltd.
4// SPDX-License-Identifier: GPL-3.0-or-later WITH Classpath-exception-2.0
5
6// This program is free software: you can redistribute it and/or modify
7// it under the terms of the GNU General Public License as published by
8// the Free Software Foundation, either version 3 of the License, or
9// (at your option) any later version.
10
11// This program is distributed in the hope that it will be useful,
12// but WITHOUT ANY WARRANTY; without even the implied warranty of
13// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
14// GNU General Public License for more details.
15
16// You should have received a copy of the GNU General Public License
17// along with this program. If not, see <https://www.gnu.org/licenses/>.
18
19use std::{collections::HashMap, marker::PhantomData, sync::Arc};
20
21use codec::Decode;
22use log::debug;
23use parking_lot::Mutex;
24
25use sc_client_api::{backend::Backend, utils::is_descendent_of};
26use sc_consensus::{
27	shared_data::{SharedDataLocked, SharedDataLockedUpgradable},
28	BlockCheckParams, BlockImport, BlockImportParams, ImportResult, JustificationImport,
29};
30use sc_telemetry::TelemetryHandle;
31use sc_utils::mpsc::TracingUnboundedSender;
32use sp_api::{Core, RuntimeApiInfo};
33use sp_blockchain::BlockStatus;
34use sp_consensus::{BlockOrigin, Error as ConsensusError, SelectChain};
35use sp_consensus_grandpa::{ConsensusLog, GrandpaApi, ScheduledChange, SetId, GRANDPA_ENGINE_ID};
36use sp_runtime::{
37	generic::OpaqueDigestItemId,
38	traits::{Block as BlockT, Header as HeaderT, NumberFor, Zero},
39	Justification,
40};
41
42use crate::{
43	authorities::{AuthoritySet, DelayKind, PendingChange, SharedAuthoritySet},
44	environment,
45	justification::GrandpaJustification,
46	notification::GrandpaJustificationSender,
47	AuthoritySetChanges, ClientForGrandpa, CommandOrError, Error, NewAuthoritySet, VoterCommand,
48	LOG_TARGET,
49};
50
51/// A block-import handler for GRANDPA.
52///
53/// This scans each imported block for signals of changing authority set.
54/// If the block being imported enacts an authority set change then:
55/// - If the current authority set is still live: we import the block
56/// - Otherwise, the block must include a valid justification.
57///
58/// When using GRANDPA, the block import worker should be using this block import
59/// object.
60pub struct GrandpaBlockImport<Backend, Block: BlockT, Client, SC> {
61	inner: Arc<Client>,
62	justification_import_period: u32,
63	select_chain: SC,
64	authority_set: SharedAuthoritySet<Block::Hash, NumberFor<Block>>,
65	send_voter_commands: TracingUnboundedSender<VoterCommand<Block::Hash, NumberFor<Block>>>,
66	authority_set_hard_forks:
67		Mutex<HashMap<Block::Hash, PendingChange<Block::Hash, NumberFor<Block>>>>,
68	justification_sender: GrandpaJustificationSender<Block>,
69	telemetry: Option<TelemetryHandle>,
70	_phantom: PhantomData<Backend>,
71}
72
73impl<Backend, Block: BlockT, Client, SC: Clone> Clone
74	for GrandpaBlockImport<Backend, Block, Client, SC>
75{
76	fn clone(&self) -> Self {
77		GrandpaBlockImport {
78			inner: self.inner.clone(),
79			justification_import_period: self.justification_import_period,
80			select_chain: self.select_chain.clone(),
81			authority_set: self.authority_set.clone(),
82			send_voter_commands: self.send_voter_commands.clone(),
83			authority_set_hard_forks: Mutex::new(self.authority_set_hard_forks.lock().clone()),
84			justification_sender: self.justification_sender.clone(),
85			telemetry: self.telemetry.clone(),
86			_phantom: PhantomData,
87		}
88	}
89}
90
91#[async_trait::async_trait]
92impl<BE, Block: BlockT, Client, SC> JustificationImport<Block>
93	for GrandpaBlockImport<BE, Block, Client, SC>
94where
95	NumberFor<Block>: finality_grandpa::BlockNumberOps,
96	BE: Backend<Block>,
97	Client: ClientForGrandpa<Block, BE>,
98	SC: SelectChain<Block>,
99{
100	type Error = ConsensusError;
101
102	async fn on_start(&mut self) -> Vec<(Block::Hash, NumberFor<Block>)> {
103		let mut out = Vec::new();
104		let chain_info = self.inner.info();
105
106		// request justifications for all pending changes for which change blocks have already been
107		// imported
108		let pending_changes: Vec<_> =
109			self.authority_set.inner().pending_changes().cloned().collect();
110
111		for pending_change in pending_changes {
112			if pending_change.delay_kind == DelayKind::Finalized &&
113				pending_change.effective_number() > chain_info.finalized_number &&
114				pending_change.effective_number() <= chain_info.best_number
115			{
116				let effective_block_hash = if !pending_change.delay.is_zero() {
117					self.select_chain
118						.finality_target(
119							pending_change.canon_hash,
120							Some(pending_change.effective_number()),
121						)
122						.await
123				} else {
124					Ok(pending_change.canon_hash)
125				};
126
127				if let Ok(hash) = effective_block_hash {
128					if let Ok(Some(header)) = self.inner.header(hash) {
129						if *header.number() == pending_change.effective_number() {
130							out.push((header.hash(), *header.number()));
131						}
132					}
133				}
134			}
135		}
136
137		out
138	}
139
140	async fn import_justification(
141		&mut self,
142		hash: Block::Hash,
143		number: NumberFor<Block>,
144		justification: Justification,
145	) -> Result<(), Self::Error> {
146		// this justification was requested by the sync service, therefore we
147		// are not sure if it should enact a change or not. it could have been a
148		// request made as part of initial sync but that means the justification
149		// wasn't part of the block and was requested asynchronously, probably
150		// makes sense to log in that case.
151		GrandpaBlockImport::import_justification(self, hash, number, justification, false, false)
152	}
153}
154
155enum AppliedChanges<H, N> {
156	Standard(bool), // true if the change is ready to be applied (i.e. it's a root)
157	Forced(NewAuthoritySet<H, N>),
158	None,
159}
160
161impl<H, N> AppliedChanges<H, N> {
162	fn needs_justification(&self) -> bool {
163		match *self {
164			AppliedChanges::Standard(_) => true,
165			AppliedChanges::Forced(_) | AppliedChanges::None => false,
166		}
167	}
168}
169
170struct PendingSetChanges<Block: BlockT> {
171	just_in_case: Option<(
172		AuthoritySet<Block::Hash, NumberFor<Block>>,
173		SharedDataLockedUpgradable<AuthoritySet<Block::Hash, NumberFor<Block>>>,
174	)>,
175	applied_changes: AppliedChanges<Block::Hash, NumberFor<Block>>,
176	do_pause: bool,
177}
178
179impl<Block: BlockT> PendingSetChanges<Block> {
180	// revert the pending set change explicitly.
181	fn revert(self) {}
182
183	fn defuse(mut self) -> (AppliedChanges<Block::Hash, NumberFor<Block>>, bool) {
184		self.just_in_case = None;
185		let applied_changes = std::mem::replace(&mut self.applied_changes, AppliedChanges::None);
186		(applied_changes, self.do_pause)
187	}
188}
189
190impl<Block: BlockT> Drop for PendingSetChanges<Block> {
191	fn drop(&mut self) {
192		if let Some((old_set, mut authorities)) = self.just_in_case.take() {
193			*authorities.upgrade() = old_set;
194		}
195	}
196}
197
198/// Checks the given header for a consensus digest signalling a **standard** scheduled change and
199/// extracts it.
200pub fn find_scheduled_change<B: BlockT>(
201	header: &B::Header,
202) -> Option<ScheduledChange<NumberFor<B>>> {
203	let id = OpaqueDigestItemId::Consensus(&GRANDPA_ENGINE_ID);
204
205	let filter_log = |log: ConsensusLog<NumberFor<B>>| match log {
206		ConsensusLog::ScheduledChange(change) => Some(change),
207		_ => None,
208	};
209
210	// find the first consensus digest with the right ID which converts to
211	// the right kind of consensus log.
212	header.digest().convert_first(|l| l.try_to(id).and_then(filter_log))
213}
214
215/// Checks the given header for a consensus digest signalling a **forced** scheduled change and
216/// extracts it.
217pub fn find_forced_change<B: BlockT>(
218	header: &B::Header,
219) -> Option<(NumberFor<B>, ScheduledChange<NumberFor<B>>)> {
220	let id = OpaqueDigestItemId::Consensus(&GRANDPA_ENGINE_ID);
221
222	let filter_log = |log: ConsensusLog<NumberFor<B>>| match log {
223		ConsensusLog::ForcedChange(delay, change) => Some((delay, change)),
224		_ => None,
225	};
226
227	// find the first consensus digest with the right ID which converts to
228	// the right kind of consensus log.
229	header.digest().convert_first(|l| l.try_to(id).and_then(filter_log))
230}
231
232impl<BE, Block: BlockT, Client, SC> GrandpaBlockImport<BE, Block, Client, SC>
233where
234	NumberFor<Block>: finality_grandpa::BlockNumberOps,
235	BE: Backend<Block>,
236	Client: ClientForGrandpa<Block, BE>,
237	Client::Api: GrandpaApi<Block>,
238	for<'a> &'a Client: BlockImport<Block, Error = ConsensusError>,
239{
240	// check for a new authority set change.
241	fn check_new_change(
242		&self,
243		header: &Block::Header,
244		hash: Block::Hash,
245	) -> Option<PendingChange<Block::Hash, NumberFor<Block>>> {
246		// check for forced authority set hard forks
247		if let Some(change) = self.authority_set_hard_forks.lock().get(&hash) {
248			return Some(change.clone())
249		}
250
251		// check for forced change.
252		if let Some((median_last_finalized, change)) = find_forced_change::<Block>(header) {
253			return Some(PendingChange {
254				next_authorities: change.next_authorities,
255				delay: change.delay,
256				canon_height: *header.number(),
257				canon_hash: hash,
258				delay_kind: DelayKind::Best { median_last_finalized },
259			})
260		}
261
262		// check normal scheduled change.
263		let change = find_scheduled_change::<Block>(header)?;
264		Some(PendingChange {
265			next_authorities: change.next_authorities,
266			delay: change.delay,
267			canon_height: *header.number(),
268			canon_hash: hash,
269			delay_kind: DelayKind::Finalized,
270		})
271	}
272
273	fn make_authorities_changes(
274		&self,
275		block: &mut BlockImportParams<Block>,
276		hash: Block::Hash,
277		initial_sync: bool,
278	) -> Result<PendingSetChanges<Block>, ConsensusError> {
279		// when we update the authorities, we need to hold the lock
280		// until the block is written to prevent a race if we need to restore
281		// the old authority set on error or panic.
282		struct InnerGuard<'a, H, N> {
283			old: Option<AuthoritySet<H, N>>,
284			guard: Option<SharedDataLocked<'a, AuthoritySet<H, N>>>,
285		}
286
287		impl<'a, H, N> InnerGuard<'a, H, N> {
288			fn as_mut(&mut self) -> &mut AuthoritySet<H, N> {
289				self.guard.as_mut().expect("only taken on deconstruction; qed")
290			}
291
292			fn set_old(&mut self, old: AuthoritySet<H, N>) {
293				if self.old.is_none() {
294					// ignore "newer" old changes.
295					self.old = Some(old);
296				}
297			}
298
299			fn consume(
300				mut self,
301			) -> Option<(AuthoritySet<H, N>, SharedDataLocked<'a, AuthoritySet<H, N>>)> {
302				self.old
303					.take()
304					.map(|old| (old, self.guard.take().expect("only taken on deconstruction; qed")))
305			}
306		}
307
308		impl<'a, H, N> Drop for InnerGuard<'a, H, N> {
309			fn drop(&mut self) {
310				if let (Some(mut guard), Some(old)) = (self.guard.take(), self.old.take()) {
311					*guard = old;
312				}
313			}
314		}
315
316		let number = *(block.header.number());
317		let maybe_change = self.check_new_change(&block.header, hash);
318
319		// returns a function for checking whether a block is a descendent of another
320		// consistent with querying client directly after importing the block.
321		let parent_hash = *block.header.parent_hash();
322		let is_descendent_of = is_descendent_of(&*self.inner, Some((hash, parent_hash)));
323
324		let mut guard = InnerGuard { guard: Some(self.authority_set.inner_locked()), old: None };
325
326		// whether to pause the old authority set -- happens after import
327		// of a forced change block.
328		let mut do_pause = false;
329
330		// add any pending changes.
331		if let Some(change) = maybe_change {
332			let old = guard.as_mut().clone();
333			guard.set_old(old);
334
335			if let DelayKind::Best { .. } = change.delay_kind {
336				do_pause = true;
337			}
338
339			guard
340				.as_mut()
341				.add_pending_change(change, &is_descendent_of)
342				.map_err(|e| ConsensusError::ClientImport(e.to_string()))?;
343		}
344
345		let applied_changes = {
346			let forced_change_set = guard
347				.as_mut()
348				.apply_forced_changes(
349					hash,
350					number,
351					&is_descendent_of,
352					initial_sync,
353					self.telemetry.clone(),
354				)
355				.map_err(|e| ConsensusError::ClientImport(e.to_string()))
356				.map_err(ConsensusError::from)?;
357
358			if let Some((median_last_finalized_number, new_set)) = forced_change_set {
359				let new_authorities = {
360					let (set_id, new_authorities) = new_set.current();
361
362					// we will use the median last finalized number as a hint
363					// for the canon block the new authority set should start
364					// with. we use the minimum between the median and the local
365					// best finalized block.
366					let best_finalized_number = self.inner.info().finalized_number;
367					let canon_number = best_finalized_number.min(median_last_finalized_number);
368					let canon_hash = self.inner.hash(canon_number)
369							.map_err(|e| ConsensusError::ClientImport(e.to_string()))?
370							.expect(
371								"the given block number is less or equal than the current best finalized number; \
372								 current best finalized number must exist in chain; qed."
373							);
374
375					NewAuthoritySet {
376						canon_number,
377						canon_hash,
378						set_id,
379						authorities: new_authorities.to_vec(),
380					}
381				};
382				let old = ::std::mem::replace(guard.as_mut(), new_set);
383				guard.set_old(old);
384
385				AppliedChanges::Forced(new_authorities)
386			} else {
387				let did_standard = guard
388					.as_mut()
389					.enacts_standard_change(hash, number, &is_descendent_of)
390					.map_err(|e| ConsensusError::ClientImport(e.to_string()))
391					.map_err(ConsensusError::from)?;
392
393				if let Some(root) = did_standard {
394					AppliedChanges::Standard(root)
395				} else {
396					AppliedChanges::None
397				}
398			}
399		};
400
401		// consume the guard safely and write necessary changes.
402		let just_in_case = guard.consume();
403		if let Some((_, ref authorities)) = just_in_case {
404			let authorities_change = match applied_changes {
405				AppliedChanges::Forced(ref new) => Some(new),
406				AppliedChanges::Standard(_) => None, // the change isn't actually applied yet.
407				AppliedChanges::None => None,
408			};
409
410			crate::aux_schema::update_authority_set::<Block, _, _>(
411				authorities,
412				authorities_change,
413				|insert| {
414					block
415						.auxiliary
416						.extend(insert.iter().map(|(k, v)| (k.to_vec(), Some(v.to_vec()))))
417				},
418			);
419		}
420
421		let just_in_case = just_in_case.map(|(o, i)| (o, i.release_mutex()));
422
423		Ok(PendingSetChanges { just_in_case, applied_changes, do_pause })
424	}
425
426	/// Read current set id form a given state.
427	fn current_set_id(&self, hash: Block::Hash) -> Result<SetId, ConsensusError> {
428		let runtime_version = self.inner.runtime_api().version(hash).map_err(|e| {
429			ConsensusError::ClientImport(format!(
430				"Unable to retrieve current runtime version. {}",
431				e
432			))
433		})?;
434
435		if runtime_version
436			.api_version(&<dyn GrandpaApi<Block>>::ID)
437			.map_or(false, |v| v < 3)
438		{
439			// The new API is not supported in this runtime. Try reading directly from storage.
440			// This code may be removed once warp sync to an old runtime is no longer needed.
441			for prefix in ["GrandpaFinality", "Grandpa"] {
442				let k = [
443					sp_crypto_hashing::twox_128(prefix.as_bytes()),
444					sp_crypto_hashing::twox_128(b"CurrentSetId"),
445				]
446				.concat();
447				if let Ok(Some(id)) =
448					self.inner.storage(hash, &sc_client_api::StorageKey(k.to_vec()))
449				{
450					if let Ok(id) = SetId::decode(&mut id.0.as_ref()) {
451						return Ok(id)
452					}
453				}
454			}
455			Err(ConsensusError::ClientImport("Unable to retrieve current set id.".into()))
456		} else {
457			self.inner
458				.runtime_api()
459				.current_set_id(hash)
460				.map_err(|e| ConsensusError::ClientImport(e.to_string()))
461		}
462	}
463
464	/// Import whole new state and reset authority set.
465	async fn import_state(
466		&self,
467		mut block: BlockImportParams<Block>,
468	) -> Result<ImportResult, ConsensusError> {
469		let hash = block.post_hash();
470		let number = *block.header.number();
471		// Force imported state finality.
472		block.finalized = true;
473		let import_result = (&*self.inner).import_block(block).await;
474		match import_result {
475			Ok(ImportResult::Imported(aux)) => {
476				// We've just imported a new state. We trust the sync module has verified
477				// finality proofs and that the state is correct and final.
478				// So we can read the authority list and set id from the state.
479				self.authority_set_hard_forks.lock().clear();
480				let authorities = self
481					.inner
482					.runtime_api()
483					.grandpa_authorities(hash)
484					.map_err(|e| ConsensusError::ClientImport(e.to_string()))?;
485				let set_id = self.current_set_id(hash)?;
486				let authority_set = AuthoritySet::new(
487					authorities.clone(),
488					set_id,
489					fork_tree::ForkTree::new(),
490					Vec::new(),
491					AuthoritySetChanges::empty(),
492				)
493				.ok_or_else(|| ConsensusError::ClientImport("Invalid authority list".into()))?;
494				*self.authority_set.inner_locked() = authority_set.clone();
495
496				crate::aux_schema::update_authority_set::<Block, _, _>(
497					&authority_set,
498					None,
499					|insert| self.inner.insert_aux(insert, []),
500				)
501				.map_err(|e| ConsensusError::ClientImport(e.to_string()))?;
502				let new_set =
503					NewAuthoritySet { canon_number: number, canon_hash: hash, set_id, authorities };
504				let _ = self
505					.send_voter_commands
506					.unbounded_send(VoterCommand::ChangeAuthorities(new_set));
507				Ok(ImportResult::Imported(aux))
508			},
509			Ok(r) => Ok(r),
510			Err(e) => Err(ConsensusError::ClientImport(e.to_string())),
511		}
512	}
513}
514
515#[async_trait::async_trait]
516impl<BE, Block: BlockT, Client, SC> BlockImport<Block> for GrandpaBlockImport<BE, Block, Client, SC>
517where
518	NumberFor<Block>: finality_grandpa::BlockNumberOps,
519	BE: Backend<Block>,
520	Client: ClientForGrandpa<Block, BE>,
521	Client::Api: GrandpaApi<Block>,
522	for<'a> &'a Client: BlockImport<Block, Error = ConsensusError>,
523	SC: Send + Sync,
524{
525	type Error = ConsensusError;
526
527	async fn import_block(
528		&self,
529		mut block: BlockImportParams<Block>,
530	) -> Result<ImportResult, Self::Error> {
531		let hash = block.post_hash();
532		let number = *block.header.number();
533
534		// early exit if block already in chain, otherwise the check for
535		// authority changes will error when trying to re-import a change block
536		match self.inner.status(hash) {
537			Ok(BlockStatus::InChain) => {
538				// Strip justifications when re-importing an existing block.
539				let _justifications = block.justifications.take();
540				return (&*self.inner).import_block(block).await
541			},
542			Ok(BlockStatus::Unknown) => {},
543			Err(e) => return Err(ConsensusError::ClientImport(e.to_string())),
544		}
545
546		if block.with_state() {
547			return self.import_state(block).await
548		}
549
550		if number <= self.inner.info().finalized_number {
551			// Importing an old block. Just save justifications and authority set changes
552			if self.check_new_change(&block.header, hash).is_some() {
553				if block.justifications.is_none() {
554					return Err(ConsensusError::ClientImport(
555						"Justification required when importing \
556							an old block with authority set change."
557							.into(),
558					))
559				}
560				let mut authority_set = self.authority_set.inner_locked();
561				authority_set.authority_set_changes.insert(number);
562				crate::aux_schema::update_authority_set::<Block, _, _>(
563					&authority_set,
564					None,
565					|insert| {
566						block
567							.auxiliary
568							.extend(insert.iter().map(|(k, v)| (k.to_vec(), Some(v.to_vec()))))
569					},
570				);
571			}
572			return (&*self.inner).import_block(block).await
573		}
574
575		// on initial sync we will restrict logging under info to avoid spam.
576		let initial_sync = block.origin == BlockOrigin::NetworkInitialSync;
577
578		let pending_changes = self.make_authorities_changes(&mut block, hash, initial_sync)?;
579
580		// we don't want to finalize on `inner.import_block`
581		let mut justifications = block.justifications.take();
582		let import_result = (&*self.inner).import_block(block).await;
583
584		let mut imported_aux = {
585			match import_result {
586				Ok(ImportResult::Imported(aux)) => aux,
587				Ok(r) => {
588					debug!(
589						target: LOG_TARGET,
590						"Restoring old authority set after block import result: {:?}", r,
591					);
592					pending_changes.revert();
593					return Ok(r)
594				},
595				Err(e) => {
596					debug!(
597						target: LOG_TARGET,
598						"Restoring old authority set after block import error: {}", e,
599					);
600					pending_changes.revert();
601					return Err(ConsensusError::ClientImport(e.to_string()))
602				},
603			}
604		};
605
606		let (applied_changes, do_pause) = pending_changes.defuse();
607
608		// Send the pause signal after import but BEFORE sending a `ChangeAuthorities` message.
609		if do_pause {
610			let _ = self.send_voter_commands.unbounded_send(VoterCommand::Pause(
611				"Forced change scheduled after inactivity".to_string(),
612			));
613		}
614
615		let needs_justification = applied_changes.needs_justification();
616
617		match applied_changes {
618			AppliedChanges::Forced(new) => {
619				// NOTE: when we do a force change we are "discrediting" the old set so we
620				// ignore any justifications from them. this block may contain a justification
621				// which should be checked and imported below against the new authority
622				// triggered by this forced change. the new grandpa voter will start at the
623				// last median finalized block (which is before the block that enacts the
624				// change), full nodes syncing the chain will not be able to successfully
625				// import justifications for those blocks since their local authority set view
626				// is still of the set before the forced change was enacted, still after #1867
627				// they should import the block and discard the justification, and they will
628				// then request a justification from sync if it's necessary (which they should
629				// then be able to successfully validate).
630				let _ =
631					self.send_voter_commands.unbounded_send(VoterCommand::ChangeAuthorities(new));
632
633				// we must clear all pending justifications requests, presumably they won't be
634				// finalized hence why this forced changes was triggered
635				imported_aux.clear_justification_requests = true;
636			},
637			AppliedChanges::Standard(false) => {
638				// we can't apply this change yet since there are other dependent changes that we
639				// need to apply first, drop any justification that might have been provided with
640				// the block to make sure we request them from `sync` which will ensure they'll be
641				// applied in-order.
642				justifications.take();
643			},
644			_ => {},
645		}
646
647		let grandpa_justification =
648			justifications.and_then(|just| just.into_justification(GRANDPA_ENGINE_ID));
649
650		match grandpa_justification {
651			Some(justification) => {
652				if environment::should_process_justification(
653					&*self.inner,
654					self.justification_import_period,
655					number,
656					needs_justification,
657				) {
658					let import_res = self.import_justification(
659						hash,
660						number,
661						(GRANDPA_ENGINE_ID, justification),
662						needs_justification,
663						initial_sync,
664					);
665
666					import_res.unwrap_or_else(|err| {
667						if needs_justification {
668							debug!(
669								target: LOG_TARGET,
670								"Requesting justification from peers due to imported block #{} that enacts authority set change with invalid justification: {}",
671								number,
672								err
673							);
674							imported_aux.bad_justification = true;
675							imported_aux.needs_justification = true;
676						}
677					});
678				} else {
679					debug!(
680						target: LOG_TARGET,
681						"Ignoring unnecessary justification for block #{}",
682						number,
683					);
684				}
685			},
686			None =>
687				if needs_justification {
688					debug!(
689						target: LOG_TARGET,
690						"Imported unjustified block #{} that enacts authority set change, waiting for finality for enactment.",
691						number,
692					);
693
694					imported_aux.needs_justification = true;
695				},
696		}
697
698		Ok(ImportResult::Imported(imported_aux))
699	}
700
701	async fn check_block(
702		&self,
703		block: BlockCheckParams<Block>,
704	) -> Result<ImportResult, Self::Error> {
705		self.inner.check_block(block).await
706	}
707}
708
709impl<Backend, Block: BlockT, Client, SC> GrandpaBlockImport<Backend, Block, Client, SC> {
710	pub(crate) fn new(
711		inner: Arc<Client>,
712		justification_import_period: u32,
713		select_chain: SC,
714		authority_set: SharedAuthoritySet<Block::Hash, NumberFor<Block>>,
715		send_voter_commands: TracingUnboundedSender<VoterCommand<Block::Hash, NumberFor<Block>>>,
716		authority_set_hard_forks: Vec<(SetId, PendingChange<Block::Hash, NumberFor<Block>>)>,
717		justification_sender: GrandpaJustificationSender<Block>,
718		telemetry: Option<TelemetryHandle>,
719	) -> GrandpaBlockImport<Backend, Block, Client, SC> {
720		// check for and apply any forced authority set hard fork that applies
721		// to the *current* authority set.
722		if let Some((_, change)) = authority_set_hard_forks
723			.iter()
724			.find(|(set_id, _)| *set_id == authority_set.set_id())
725		{
726			authority_set.inner().current_authorities = change.next_authorities.clone();
727		}
728
729		// index authority set hard forks by block hash so that they can be used
730		// by any node syncing the chain and importing a block hard fork
731		// authority set changes.
732		let authority_set_hard_forks = authority_set_hard_forks
733			.into_iter()
734			.map(|(_, change)| (change.canon_hash, change))
735			.collect::<HashMap<_, _>>();
736
737		// check for and apply any forced authority set hard fork that apply to
738		// any *pending* standard changes, checking by the block hash at which
739		// they were announced.
740		{
741			let mut authority_set = authority_set.inner();
742
743			authority_set.pending_standard_changes =
744				authority_set.pending_standard_changes.clone().map(&mut |hash, _, original| {
745					authority_set_hard_forks.get(hash).cloned().unwrap_or(original)
746				});
747		}
748
749		GrandpaBlockImport {
750			inner,
751			justification_import_period,
752			select_chain,
753			authority_set,
754			send_voter_commands,
755			authority_set_hard_forks: Mutex::new(authority_set_hard_forks),
756			justification_sender,
757			telemetry,
758			_phantom: PhantomData,
759		}
760	}
761}
762
763impl<BE, Block: BlockT, Client, SC> GrandpaBlockImport<BE, Block, Client, SC>
764where
765	BE: Backend<Block>,
766	Client: ClientForGrandpa<Block, BE>,
767	NumberFor<Block>: finality_grandpa::BlockNumberOps,
768{
769	/// Import a block justification and finalize the block.
770	///
771	/// If `enacts_change` is set to true, then finalizing this block *must*
772	/// enact an authority set change, the function will panic otherwise.
773	fn import_justification(
774		&self,
775		hash: Block::Hash,
776		number: NumberFor<Block>,
777		justification: Justification,
778		enacts_change: bool,
779		initial_sync: bool,
780	) -> Result<(), ConsensusError> {
781		if justification.0 != GRANDPA_ENGINE_ID {
782			// TODO: the import queue needs to be refactored to be able dispatch to the correct
783			// `JustificationImport` instance based on `ConsensusEngineId`, or we need to build a
784			// justification import pipeline similar to what we do for `BlockImport`. In the
785			// meantime we'll just drop the justification, since this is only used for BEEFY which
786			// is still WIP.
787			return Ok(())
788		}
789
790		let justification = GrandpaJustification::decode_and_verify_finalizes(
791			&justification.1,
792			(hash, number),
793			self.authority_set.set_id(),
794			&self.authority_set.current_authorities(),
795		);
796
797		let justification = match justification {
798			Err(e) => {
799				return match e {
800					sp_blockchain::Error::OutdatedJustification =>
801						Err(ConsensusError::OutdatedJustification),
802					_ => Err(ConsensusError::ClientImport(e.to_string())),
803				};
804			},
805			Ok(justification) => justification,
806		};
807
808		let result = environment::finalize_block(
809			self.inner.clone(),
810			&self.authority_set,
811			None,
812			hash,
813			number,
814			justification.into(),
815			initial_sync,
816			Some(&self.justification_sender),
817			self.telemetry.clone(),
818		);
819
820		match result {
821			Err(CommandOrError::VoterCommand(command)) => {
822				grandpa_log!(
823					initial_sync,
824					"👴 Imported justification for block #{} that triggers \
825					command {}, signaling voter.",
826					number,
827					command,
828				);
829
830				// send the command to the voter
831				let _ = self.send_voter_commands.unbounded_send(command);
832			},
833			Err(CommandOrError::Error(e)) =>
834				return Err(match e {
835					Error::Grandpa(error) => ConsensusError::ClientImport(error.to_string()),
836					Error::Network(error) => ConsensusError::ClientImport(error),
837					Error::Blockchain(error) => ConsensusError::ClientImport(error),
838					Error::Client(error) => ConsensusError::ClientImport(error.to_string()),
839					Error::Safety(error) => ConsensusError::ClientImport(error),
840					Error::Signing(error) => ConsensusError::ClientImport(error),
841					Error::Timer(error) => ConsensusError::ClientImport(error.to_string()),
842					Error::RuntimeApi(error) => ConsensusError::ClientImport(error.to_string()),
843				}),
844			Ok(_) => {
845				assert!(
846					!enacts_change,
847					"returns Ok when no authority set change should be enacted; qed;"
848				);
849			},
850		}
851
852		Ok(())
853	}
854}