New contract PoA sync fixes (#5991)

* generate proofs on newly-created state

* report only missed steps after first block

* dont report skipped if not signer

* test

* finality tracing and passing valid header to `commit_block`

* avoid proposing multiple times on the same step when validator set changes

* limit benign reports

* Ordering -> AtomicOrdering

* reinstate warning now that spam is reduced

* flush pending transition changes when necessary

* ensure epochs aren't re-zoomed on every block
This commit is contained in:
keorn 2017-07-13 09:48:00 +02:00 committed by Gav Wood
parent 40e29c92bc
commit 22261bc2d1
8 changed files with 227 additions and 86 deletions

View File

@ -200,7 +200,7 @@ pub trait IsBlock {
/// Trait for a object that has a state database.
pub trait Drain {
/// Drop this object and return the underlieing database.
/// Drop this object and return the underlying database.
fn drain(self) -> StateDB;
}
@ -370,6 +370,22 @@ impl<'x> OpenBlock<'x> {
}
}
/// Push transactions onto the block.
pub fn push_transactions(&mut self, transactions: &[SignedTransaction]) -> Result<(), Error> {
push_transactions(self, transactions)
}
/// Populate self from a header.
pub fn populate_from(&mut self, header: &Header) {
self.set_difficulty(*header.difficulty());
self.set_gas_limit(*header.gas_limit());
self.set_timestamp(header.timestamp());
self.set_author(header.author().clone());
self.set_extra_data(header.extra_data().clone()).unwrap_or_else(|e| warn!("Couldn't set extradata: {}. Ignoring.", e));
self.set_uncles_hash(header.uncles_hash().clone());
self.set_transactions_root(header.transactions_root().clone());
}
/// Turn this into a `ClosedBlock`.
pub fn close(self) -> ClosedBlock {
let mut s = self;
@ -579,18 +595,13 @@ pub fn enact(
is_epoch_begin,
)?;
b.set_difficulty(*header.difficulty());
b.set_gas_limit(*header.gas_limit());
b.set_timestamp(header.timestamp());
b.set_author(header.author().clone());
b.set_extra_data(header.extra_data().clone()).unwrap_or_else(|e| warn!("Couldn't set extradata: {}. Ignoring.", e));
b.set_uncles_hash(header.uncles_hash().clone());
b.set_transactions_root(header.transactions_root().clone());
b.populate_from(header);
b.push_transactions(transactions)?;
push_transactions(&mut b, transactions)?;
for u in uncles {
b.push_uncle(u.clone())?;
}
Ok(b.close_and_lock())
}
@ -681,7 +692,36 @@ mod tests {
let header = block.header();
let transactions: Result<Vec<_>, Error> = block.transactions().into_iter().map(SignedTransaction::new).collect();
let transactions = transactions?;
enact(&header, &transactions, &block.uncles(), engine, tracing, db, parent, last_hashes, factories, false)
{
if ::log::max_log_level() >= ::log::LogLevel::Trace {
let s = State::from_existing(db.boxed_clone(), parent.state_root().clone(), engine.account_start_nonce(parent.number() + 1), factories.clone())?;
trace!(target: "enact", "num={}, root={}, author={}, author_balance={}\n",
header.number(), s.root(), header.author(), s.balance(&header.author())?);
}
}
let mut b = OpenBlock::new(
engine,
factories,
tracing,
db,
parent,
last_hashes,
Address::new(),
(3141562.into(), 31415620.into()),
vec![],
false,
)?;
b.populate_from(&header);
b.push_transactions(&transactions)?;
for u in &block.uncles() {
b.push_uncle(u.clone())?;
}
Ok(b.close_and_lock())
}
/// Enact the block given by `block_bytes` using `engine` on the database `db` with given `parent` block header. Seal the block aferwards

View File

@ -530,7 +530,7 @@ impl Client {
} else {
imported_blocks.push(header.hash());
let route = self.commit_block(closed_block, &header.hash(), &block.bytes);
let route = self.commit_block(closed_block, &header, &block.bytes);
import_results.push(route);
self.report.write().accrue_block(&block);
@ -635,10 +635,14 @@ impl Client {
Ok(hash)
}
fn commit_block<B>(&self, block: B, hash: &H256, block_data: &[u8]) -> ImportRoute where B: IsBlock + Drain {
let number = block.header().number();
let parent = block.header().parent_hash().clone();
let header = block.header().clone(); // TODO: optimize and avoid copy.
// NOTE: the header of the block passed here is not necessarily sealed, as
// it is for reconstructing the state transition.
//
// The header passed is from the original block data and is sealed.
fn commit_block<B>(&self, block: B, header: &Header, block_data: &[u8]) -> ImportRoute where B: IsBlock + Drain {
let hash = &header.hash();
let number = header.number();
let parent = header.parent_hash();
let chain = self.chain.read();
// Commit results
@ -648,6 +652,8 @@ impl Client {
.map(Into::into)
.collect();
assert_eq!(header.hash(), BlockView::new(block_data).header_view().sha3());
//let traces = From::from(block.traces().clone().unwrap_or_else(Vec::new));
let mut batch = DBTransaction::new();
@ -657,6 +663,17 @@ impl Client {
// TODO: Prove it with a test.
let mut state = block.drain();
// check epoch end signal, potentially generating a proof on the current
// state.
self.check_epoch_end_signal(
&header,
block_data,
&receipts,
&state,
&chain,
&mut batch,
);
state.journal_under(&mut batch, number, hash).expect("DB commit failed");
let route = chain.insert_block(&mut batch, block_data, receipts.clone());
@ -674,10 +691,6 @@ impl Client {
self.db.read().write_buffered(batch);
chain.commit();
// check for epoch end. do this after writing first batch so we can prove
// transactions on the block's state.
// TODO: work these changes into the existing DBTransaction.
self.check_epoch_end_signal(&header, block_data, &receipts, &chain);
self.check_epoch_end(&header, &chain);
self.update_last_hashes(&parent, hash);
@ -691,38 +704,82 @@ impl Client {
// check for epoch end signal and write pending transition if it occurs.
// state for the given block must be available.
fn check_epoch_end_signal(&self, header: &Header, block: &[u8], receipts: &[Receipt], chain: &BlockChain) {
fn check_epoch_end_signal(
&self,
header: &Header,
block_bytes: &[u8],
receipts: &[Receipt],
state_db: &StateDB,
chain: &BlockChain,
batch: &mut DBTransaction,
) {
use engines::EpochChange;
let hash = header.hash();
match self.engine.signals_epoch_end(header, Some(block), Some(&receipts)) {
match self.engine.signals_epoch_end(header, Some(block_bytes), Some(&receipts)) {
EpochChange::Yes(proof) => {
use engines::epoch::PendingTransition;
use engines::Proof;
let proof = match proof {
Proof::Known(proof) => proof,
Proof::WithState(with_state) =>
match self.with_proving_caller(BlockId::Hash(hash), move |c| with_state(c)) {
Proof::WithState(with_state) => {
let env_info = EnvInfo {
number: header.number(),
author: header.author().clone(),
timestamp: header.timestamp(),
difficulty: header.difficulty().clone(),
last_hashes: self.build_last_hashes(header.parent_hash().clone()),
gas_used: U256::default(),
gas_limit: u64::max_value().into(),
};
let call = move |addr, data| {
let mut state_db = state_db.boxed_clone();
let backend = ::state::backend::Proving::new(state_db.as_hashdb_mut());
let transaction =
self.contract_call_tx(BlockId::Hash(*header.parent_hash()), addr, data);
let mut state = State::from_existing(
backend,
header.state_root().clone(),
self.engine.account_start_nonce(header.number()),
self.factories.clone(),
).expect("state known to be available for just-imported block; qed");
let options = TransactOptions { tracing: false, vm_tracing: false, check_nonce: false };
let res = Executive::new(&mut state, &env_info, &*self.engine)
.transact(&transaction, options);
let res = match res {
Err(ExecutionError::Internal(e)) =>
Err(format!("Internal error: {}", e)),
Err(e) => {
trace!(target: "client", "Proved call failed: {}", e);
Ok((Vec::new(), state.drop().1.extract_proof()))
}
Ok(res) => Ok((res.output, state.drop().1.extract_proof())),
};
res.map(|(output, proof)| (output, proof.into_iter().map(|x| x.into_vec()).collect()))
};
match (with_state)(&call) {
Ok(proof) => proof,
Err(e) => {
warn!(target: "client", "Failed to generate transition proof for block {}: {}", hash, e);
warn!(target: "client", "Snapshots produced by this client may be incomplete");
Vec::new()
}
},
}
}
};
debug!(target: "client", "Block {} signals epoch end.", hash);
// write pending transition to DB.
let mut batch = DBTransaction::new();
let pending = PendingTransition { proof: proof };
chain.insert_pending_transition(&mut batch, hash, pending);
self.db.read().write_buffered(batch);
chain.insert_pending_transition(batch, hash, pending);
},
EpochChange::No => {},
EpochChange::Unsure(_) => {
@ -749,7 +806,11 @@ impl Client {
block_number: header.number(),
proof: proof,
});
self.db.read().write_buffered(batch);
// always write the batch directly since epoch transition proofs are
// fetched from a DB iterator and DB iterators are only available on
// flushed data.
self.db.read().write(batch).expect("DB flush failed");
}
}
@ -1766,7 +1827,9 @@ impl MiningBlockChainClient for Client {
let number = block.header().number();
let block_data = block.rlp_bytes();
let route = self.commit_block(block, &h, &block_data);
let header = block.header().clone();
let route = self.commit_block(block, &header, &block_data);
trace!(target: "client", "Imported sealed block #{} ({})", number, h);
self.state_db.lock().sync_cache(&route.enacted, &route.retracted, false);
route

View File

@ -66,6 +66,7 @@ impl RollingFinality {
let entry = self.sign_count.entry(signer);
if let (true, &Entry::Vacant(_)) = (would_be_finalized, &entry) {
trace!(target: "finality", "Encountered already finalized block {}", hash);
break
}
@ -75,6 +76,7 @@ impl RollingFinality {
self.headers.push_front((hash, signer));
}
trace!(target: "finality", "Rolling finality state: {:?}", self.headers);
Ok(())
}
@ -128,6 +130,9 @@ impl RollingFinality {
}
}
trace!(target: "finality", "Blocks finalized by {:?}: {:?}", head, newly_finalized);
self.last_pushed = Some(head);
Ok(newly_finalized)
}
}

View File

@ -220,9 +220,9 @@ pub struct AuthorityRound {
builtins: BTreeMap<Address, Builtin>,
transition_service: IoService<()>,
step: Arc<Step>,
proposed: AtomicBool,
can_propose: AtomicBool,
client: RwLock<Option<Weak<EngineClient>>>,
signer: EngineSigner,
signer: RwLock<EngineSigner>,
validators: Box<ValidatorSet>,
validate_score_transition: u64,
eip155_transition: u64,
@ -311,7 +311,7 @@ fn verify_external<F: Fn(Report)>(header: &Header, validators: &ValidatorSet, st
// Give one step slack if step is lagging, double vote is still not possible.
if step.is_future(header_step) {
trace!(target: "engine", "verify_block_unordered: block from the future");
trace!(target: "engine", "verify_block_external: block from the future");
report(Report::Benign(*header.author(), header.number()));
Err(BlockError::InvalidSeal)?
} else {
@ -321,7 +321,7 @@ fn verify_external<F: Fn(Report)>(header: &Header, validators: &ValidatorSet, st
!verify_address(&correct_proposer, &proposer_signature, &header.bare_hash())?;
if is_invalid_proposer {
trace!(target: "engine", "verify_block_unordered: bad proposer for step: {}", header_step);
trace!(target: "engine", "verify_block_external: bad proposer for step: {}", header_step);
Err(EngineError::NotProposer(Mismatch { expected: correct_proposer, found: header.author().clone() }))?
} else {
Ok(())
@ -372,7 +372,7 @@ impl AuthorityRound {
calibrate: our_params.start_step.is_none(),
duration: our_params.step_duration,
}),
proposed: AtomicBool::new(false),
can_propose: AtomicBool::new(true),
client: RwLock::new(None),
signer: Default::default(),
validators: our_params.validators,
@ -439,7 +439,7 @@ impl Engine for AuthorityRound {
fn step(&self) {
self.step.increment();
self.proposed.store(false, AtomicOrdering::SeqCst);
self.can_propose.store(true, AtomicOrdering::SeqCst);
if let Some(ref weak) = *self.client.read() {
if let Some(c) = weak.upgrade() {
c.update_sealing();
@ -471,7 +471,7 @@ impl Engine for AuthorityRound {
}
fn seals_internally(&self) -> Option<bool> {
Some(self.signer.address() != Address::default())
Some(self.signer.read().is_some())
}
/// Attempt to seal the block internally.
@ -481,7 +481,7 @@ impl Engine for AuthorityRound {
fn generate_seal(&self, block: &ExecutedBlock) -> Seal {
// first check to avoid generating signature most of the time
// (but there's still a race to the `compare_and_swap`)
if self.proposed.load(AtomicOrdering::SeqCst) { return Seal::None; }
if !self.can_propose.load(AtomicOrdering::SeqCst) { return Seal::None; }
let header = block.header();
let step = self.step.load();
@ -512,11 +512,11 @@ impl Engine for AuthorityRound {
};
if is_step_proposer(validators, header.parent_hash(), step, header.author()) {
if let Ok(signature) = self.signer.sign(header.bare_hash()) {
if let Ok(signature) = self.sign(header.bare_hash()) {
trace!(target: "engine", "generate_seal: Issuing a block for step {}.", step);
// only issue the seal if we were the first to reach the compare_and_swap.
if !self.proposed.compare_and_swap(false, true, AtomicOrdering::SeqCst) {
if self.can_propose.compare_and_swap(true, false, AtomicOrdering::SeqCst) {
return Seal::Regular(vec![encode(&step).into_vec(), encode(&(&H520::from(signature) as &[u8])).into_vec()]);
}
} else {
@ -614,16 +614,18 @@ impl Engine for AuthorityRound {
Err(EngineError::DoubleVote(header.author().clone()))?;
}
// Report skipped primaries.
if step > parent_step + 1 {
// TODO: use epochmanager to get correct validator set for reporting?
// or just rely on the fact that in general these will be the same
// and some reports might go missing?
trace!(target: "engine", "Author {} built block with step gap. current step: {}, parent step: {}",
if let (true, Some(me)) = (step > parent_step + 1, self.signer.read().address()) {
debug!(target: "engine", "Author {} built block with step gap. current step: {}, parent step: {}",
header.author(), step, parent_step);
let mut reported = HashSet::new();
for s in parent_step + 1..step {
let skipped_primary = step_proposer(&*self.validators, &parent.hash(), s);
self.validators.report_benign(&skipped_primary, header.number(), header.number());
// Do not report this signer.
if skipped_primary != me {
self.validators.report_benign(&skipped_primary, header.number(), header.number());
}
// Stop reporting once validators start repeating.
if !reported.insert(skipped_primary) { break; }
}
}
@ -723,6 +725,9 @@ impl Engine for AuthorityRound {
if epoch_manager.finality_checker.subchain_head() != Some(*chain_head.parent_hash()) {
// build new finality checker from ancestry of chain head,
// not including chain head itself yet.
trace!(target: "finality", "Building finality up to parent of {} ({})",
chain_head.hash(), chain_head.parent_hash());
let mut hash = chain_head.parent_hash().clone();
let epoch_transition_hash = epoch_manager.epoch_transition_hash;
@ -734,6 +739,8 @@ impl Engine for AuthorityRound {
if header.number() == 0 { return None }
let res = (hash, header.author().clone());
trace!(target: "finality", "Ancestry iteration: yielding {:?}", res);
hash = header.parent_hash().clone();
Some(res)
})
@ -766,6 +773,16 @@ impl Engine for AuthorityRound {
let finality_proof = ::rlp::encode_list(&finality_proof);
epoch_manager.note_new_epoch();
info!(target: "engine", "Applying validator set change signalled at block {}", signal_number);
// We turn off can_propose here because upon validator set change there can
// be two valid proposers for a single step: one from the old set and
// one from the new.
//
// This way, upon encountering an epoch change, the proposer from the
// new set will be forced to wait until the next step to avoid sealing a
// block that breaks the invariant that the parent's step < the block's step.
self.can_propose.store(false, AtomicOrdering::SeqCst);
return Some(combine_proofs(signal_number, &pending.proof, &*finality_proof));
}
}
@ -816,11 +833,11 @@ impl Engine for AuthorityRound {
}
fn set_signer(&self, ap: Arc<AccountProvider>, address: Address, password: String) {
self.signer.set(ap, address, password);
self.signer.write().set(ap, address, password);
}
fn sign(&self, hash: H256) -> Result<Signature, Error> {
self.signer.sign(hash).map_err(Into::into)
self.signer.read().sign(hash).map_err(Into::into)
}
fn snapshot_components(&self) -> Option<Box<::snapshot::SnapshotComponents>> {
@ -1018,6 +1035,12 @@ mod tests {
header.set_gas_limit(U256::from_str("222222").unwrap());
header.set_seal(vec![encode(&3usize).into_vec()]);
// Do not report when signer not present.
assert!(aura.verify_block_family(&header, &parent_header, None).is_ok());
assert_eq!(last_benign.load(AtomicOrdering::SeqCst), 0);
aura.set_signer(Arc::new(AccountProvider::transient_provider()), Default::default(), Default::default());
assert!(aura.verify_block_family(&header, &parent_header, None).is_ok());
assert_eq!(last_benign.load(AtomicOrdering::SeqCst), 1);
}

View File

@ -82,7 +82,7 @@ pub struct BasicAuthority {
params: CommonParams,
gas_limit_bound_divisor: U256,
builtins: BTreeMap<Address, Builtin>,
signer: EngineSigner,
signer: RwLock<EngineSigner>,
validators: Box<ValidatorSet>,
}
@ -129,7 +129,7 @@ impl Engine for BasicAuthority {
}
fn seals_internally(&self) -> Option<bool> {
Some(self.signer.address() != Address::default())
Some(self.signer.read().is_some())
}
/// Attempt to seal the block internally.
@ -138,7 +138,7 @@ impl Engine for BasicAuthority {
let author = header.author();
if self.validators.contains(header.parent_hash(), author) {
// account should be pernamently unlocked, otherwise sealing will fail
if let Ok(signature) = self.signer.sign(header.bare_hash()) {
if let Ok(signature) = self.sign(header.bare_hash()) {
return Seal::Regular(vec![::rlp::encode(&(&H520::from(signature) as &[u8])).into_vec()]);
} else {
trace!(target: "basicauthority", "generate_seal: FAIL: accounts secret key unavailable");
@ -240,11 +240,11 @@ impl Engine for BasicAuthority {
}
fn set_signer(&self, ap: Arc<AccountProvider>, address: Address, password: String) {
self.signer.set(ap, address, password);
self.signer.write().set(ap, address, password);
}
fn sign(&self, hash: H256) -> Result<Signature, Error> {
self.signer.sign(hash).map_err(Into::into)
self.signer.read().sign(hash).map_err(Into::into)
}
fn snapshot_components(&self) -> Option<Box<::snapshot::SnapshotComponents>> {

View File

@ -16,21 +16,21 @@
//! A signer used by Engines which need to sign messages.
use util::{Arc, Mutex, RwLock, H256, Address};
use util::{Arc, H256, Address};
use ethkey::Signature;
use account_provider::{self, AccountProvider};
/// Everything that an Engine needs to sign messages.
pub struct EngineSigner {
account_provider: Mutex<Arc<AccountProvider>>,
address: RwLock<Address>,
password: RwLock<Option<String>>,
account_provider: Arc<AccountProvider>,
address: Option<Address>,
password: Option<String>,
}
impl Default for EngineSigner {
fn default() -> Self {
EngineSigner {
account_provider: Mutex::new(Arc::new(AccountProvider::transient_provider())),
account_provider: Arc::new(AccountProvider::transient_provider()),
address: Default::default(),
password: Default::default(),
}
@ -39,25 +39,30 @@ impl Default for EngineSigner {
impl EngineSigner {
/// Set up the signer to sign with given address and password.
pub fn set(&self, ap: Arc<AccountProvider>, address: Address, password: String) {
*self.account_provider.lock() = ap;
*self.address.write() = address;
*self.password.write() = Some(password);
pub fn set(&mut self, ap: Arc<AccountProvider>, address: Address, password: String) {
self.account_provider = ap;
self.address = Some(address);
self.password = Some(password);
debug!(target: "poa", "Setting Engine signer to {}", address);
}
/// Sign a consensus message hash.
pub fn sign(&self, hash: H256) -> Result<Signature, account_provider::SignError> {
self.account_provider.lock().sign(*self.address.read(), self.password.read().clone(), hash)
self.account_provider.sign(self.address.unwrap_or_else(Default::default), self.password.clone(), hash)
}
/// Signing address.
pub fn address(&self) -> Address {
self.address.read().clone()
pub fn address(&self) -> Option<Address> {
self.address.clone()
}
/// Check if the given address is the signing address.
pub fn is_address(&self, address: &Address) -> bool {
*self.address.read() == *address
self.address.map_or(false, |a| a == *address)
}
/// Check if the signing address was set.
pub fn is_some(&self) -> bool {
self.address.is_some()
}
}

View File

@ -86,7 +86,7 @@ pub struct Tendermint {
/// Vote accumulator.
votes: VoteCollector<ConsensusMessage>,
/// Used to sign messages and proposals.
signer: EngineSigner,
signer: RwLock<EngineSigner>,
/// Message for the last PoLC.
lock_change: RwLock<Option<ConsensusMessage>>,
/// Last lock view.
@ -159,19 +159,22 @@ impl Tendermint {
let r = self.view.load(AtomicOrdering::SeqCst);
let s = *self.step.read();
let vote_info = message_info_rlp(&VoteStep::new(h, r, s), block_hash);
match self.signer.sign(vote_info.sha3()).map(Into::into) {
Ok(signature) => {
match (self.signer.read().address(), self.sign(vote_info.sha3()).map(Into::into)) {
(Some(validator), Ok(signature)) => {
let message_rlp = message_full_rlp(&signature, &vote_info);
let message = ConsensusMessage::new(signature, h, r, s, block_hash);
let validator = self.signer.address();
self.votes.vote(message.clone(), &validator);
debug!(target: "engine", "Generated {:?} as {}.", message, validator);
self.handle_valid_message(&message);
Some(message_rlp)
},
Err(e) => {
trace!(target: "engine", "Could not sign the message {}", e);
(None, _) => {
trace!(target: "engine", "No message, since there is no engine signer.");
None
},
(Some(v), Err(e)) => {
trace!(target: "engine", "{} could not sign the message {}", v, e);
None
},
}
@ -272,7 +275,7 @@ impl Tendermint {
/// Check if current signer is the current proposer.
fn is_signer_proposer(&self, bh: &H256) -> bool {
let proposer = self.view_proposer(bh, self.height.load(AtomicOrdering::SeqCst), self.view.load(AtomicOrdering::SeqCst));
self.signer.is_address(&proposer)
self.signer.read().is_address(&proposer)
}
fn is_height(&self, message: &ConsensusMessage) -> bool {
@ -420,7 +423,7 @@ impl Engine for Tendermint {
/// Should this node participate.
fn seals_internally(&self) -> Option<bool> {
Some(self.signer.address() != Address::default())
Some(self.signer.read().is_some())
}
/// Attempt to seal generate a proposal seal.
@ -436,7 +439,7 @@ impl Engine for Tendermint {
let view = self.view.load(AtomicOrdering::SeqCst);
let bh = Some(header.bare_hash());
let vote_info = message_info_rlp(&VoteStep::new(height, view, Step::Propose), bh.clone());
if let Ok(signature) = self.signer.sign(vote_info.sha3()).map(Into::into) {
if let Ok(signature) = self.sign(vote_info.sha3()).map(Into::into) {
// Insert Propose vote.
debug!(target: "engine", "Submitting proposal {} at height {} view {}.", header.bare_hash(), height, view);
self.votes.vote(ConsensusMessage::new(signature, height, view, Step::Propose, bh), author);
@ -565,13 +568,13 @@ impl Engine for Tendermint {
fn set_signer(&self, ap: Arc<AccountProvider>, address: Address, password: String) {
{
self.signer.set(ap, address, password);
self.signer.write().set(ap, address, password);
}
self.to_step(Step::Propose);
}
fn sign(&self, hash: H256) -> Result<Signature, Error> {
self.signer.sign(hash).map_err(Into::into)
self.signer.read().sign(hash).map_err(Into::into)
}
fn stop(&self) {

View File

@ -110,6 +110,9 @@ fn prove_initial(provider: &Provider, header: &Header, caller: &Call) -> Result<
trace!(target: "engine", "obtained proof for initial set: {} validators, {} bytes",
validators.len(), proof.len());
info!(target: "engine", "Signal for switch to contract-based validator set.");
info!(target: "engine", "Initial contract validators: {:?}", validators);
proof
})
}
@ -231,9 +234,7 @@ impl ValidatorSet for ValidatorSafeContract {
.map(|out| (out, Vec::new()))) // generate no proofs in general
}
fn on_epoch_begin(&self, first: bool, _header: &Header, caller: &mut SystemCall) -> Result<(), ::error::Error> {
if first { return Ok(()) } // only signalled changes need to be noted.
fn on_epoch_begin(&self, _first: bool, _header: &Header, caller: &mut SystemCall) -> Result<(), ::error::Error> {
self.provider.finalize_change(caller)
.wait()
.map_err(::engines::EngineError::FailedSystemCall)
@ -271,8 +272,9 @@ impl ValidatorSet for ValidatorSafeContract {
None => ::engines::EpochChange::Unsure(::engines::Unsure::NeedsReceipts),
Some(receipts) => match self.extract_from_event(bloom, header, receipts) {
None => ::engines::EpochChange::No,
Some(_) => {
debug!(target: "engine", "signalling transition within contract");
Some(list) => {
info!(target: "engine", "Signal for transition within contract. New list: {:?}",
&*list);
let proof = encode_proof(&header, receipts);
::engines::EpochChange::Yes(::engines::Proof::Known(proof))