Skip to content
This repository has been archived by the owner on Nov 15, 2023. It is now read-only.

past-session validator discovery APIs #2009

Merged
43 commits merged into from
Nov 26, 2020
Merged
Show file tree
Hide file tree
Changes from 30 commits
Commits
Show all changes
43 commits
Select commit Hold shift + click to select a range
4fecb64
guide: fix formatting for SessionInfo module
ordian Nov 5, 2020
f3b5430
primitives: SessionInfo type
ordian Nov 6, 2020
62843e9
punt on approval keys
ordian Nov 9, 2020
1f720aa
ah, revert the type alias
ordian Nov 9, 2020
25dd0c9
session info runtime module skeleton
ordian Nov 10, 2020
f1dd12a
update the guide
ordian Nov 13, 2020
b2ef662
runtime/configuration: sync with the guide
ordian Nov 17, 2020
c2512be
runtime/configuration: setters for newly added fields
ordian Nov 17, 2020
a6432bf
runtime/configuration: set codec indexes
ordian Nov 17, 2020
d350875
runtime/configuration: update test
ordian Nov 17, 2020
93e29b2
primitives: fix SessionInfo definition
ordian Nov 17, 2020
4a5ca12
runtime/session_info: initial impl
ordian Nov 17, 2020
e594e5d
runtime/session_info: use initializer for session handling (wip)
ordian Nov 17, 2020
c9f4e7c
Merge branch 'master' into ao-session-info
ordian Nov 18, 2020
d308491
Merge branch 'master' into ao-session-info
ordian Nov 19, 2020
b01341a
runtime/session_info: mock authority discovery trait
ordian Nov 19, 2020
2e563d3
guide: update the initializer's order
ordian Nov 19, 2020
532429e
runtime/session_info: tests skeleton
ordian Nov 19, 2020
b0dd493
runtime/session_info: store n_delay_tranches in Configuration
ordian Nov 19, 2020
ca9dd13
runtime/session_info: punt on approval keys
ordian Nov 19, 2020
088efb3
runtime/session_info: add some basic tests
ordian Nov 19, 2020
4599423
Merge branch 'master' into ao-session-info
ordian Nov 19, 2020
76ec9ab
Update primitives/src/v1.rs
ordian Nov 19, 2020
f8a11cb
small fixes
ordian Nov 19, 2020
2f6905a
remove codec index annotation on structs
ordian Nov 19, 2020
d50073b
Merge branch 'master' into ao-session-info
ordian Nov 23, 2020
612c170
Merge branch 'master' into ao-session-info
ordian Nov 23, 2020
35cb563
fix off-by-one error
ordian Nov 23, 2020
3d22556
validator_discovery: accept a session index
ordian Nov 24, 2020
5304563
runtime: replace validator_discovery api with session_info
ordian Nov 24, 2020
60aede3
Merge branch 'master' into ao-validator-discovery-previous-sessions
ordian Nov 24, 2020
21eb2a4
Update runtime/parachains/src/session_info.rs
ordian Nov 25, 2020
25e13e1
runtime/session_info: add a comment about missing entries
ordian Nov 25, 2020
158a5fa
runtime/session_info: define the keys
ordian Nov 25, 2020
8c32d84
Merge branch 'ao-session-info' into ao-validator-discovery-previous-s…
ordian Nov 25, 2020
09c9359
util: expose connect_to_past_session_validators
ordian Nov 25, 2020
1d7ad3b
util: allow session_info requests for jobs
ordian Nov 25, 2020
992ed07
runtime-api: add mock test for session_info
ordian Nov 25, 2020
835ae2f
collator-protocol: add session_index to test state
ordian Nov 25, 2020
f1cc838
Merge branch 'master' into ao-validator-discovery-previous-sessions
ordian Nov 25, 2020
248fd37
util: fix error message for runtime error
ordian Nov 25, 2020
1a16356
fix compilation
ordian Nov 25, 2020
947a4fc
fix tests after merge with master
ordian Nov 25, 2020
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
15 changes: 8 additions & 7 deletions node/core/runtime-api/src/lib.rs
Original file line number Diff line number Diff line change
Expand Up @@ -134,7 +134,7 @@ fn make_runtime_api_request<Client>(
Request::CandidatePendingAvailability(para, sender) =>
query!(candidate_pending_availability(para), sender),
Request::CandidateEvents(sender) => query!(candidate_events(), sender),
Request::ValidatorDiscovery(ids, sender) => query!(validator_discovery(ids), sender),
Request::SessionInfo(index, sender) => query!(session_info(index), sender),
Request::DmqContents(id, sender) => query!(dmq_contents(id), sender),
Request::InboundHrmpChannelsContents(id, sender) => query!(inbound_hrmp_channels_contents(id), sender),
}
Expand Down Expand Up @@ -201,8 +201,8 @@ mod tests {
use polkadot_primitives::v1::{
ValidatorId, ValidatorIndex, GroupRotationInfo, CoreState, PersistedValidationData,
Id as ParaId, OccupiedCoreAssumption, ValidationData, SessionIndex, ValidationCode,
CommittedCandidateReceipt, CandidateEvent, AuthorityDiscoveryId, InboundDownwardMessage,
BlockNumber, InboundHrmpMessage,
CommittedCandidateReceipt, CandidateEvent, InboundDownwardMessage,
BlockNumber, InboundHrmpMessage, SessionInfo,
};
use polkadot_node_subsystem_test_helpers as test_helpers;
use sp_core::testing::TaskExecutor;
Expand All @@ -216,6 +216,7 @@ mod tests {
availability_cores: Vec<CoreState>,
validation_data: HashMap<ParaId, ValidationData>,
session_index_for_child: SessionIndex,
session_info: HashMap<SessionIndex, SessionInfo>,
validation_code: HashMap<ParaId, ValidationCode>,
historical_validation_code: HashMap<ParaId, Vec<(BlockNumber, ValidationCode)>>,
validation_outputs_results: HashMap<ParaId, bool>,
Expand Down Expand Up @@ -289,6 +290,10 @@ mod tests {
self.session_index_for_child.clone()
}

fn session_info(&self, index: SessionIndex) -> Option<SessionInfo> {
self.session_info.get(&index).cloned()
}

fn validation_code(
&self,
para: ParaId,
Expand Down Expand Up @@ -321,10 +326,6 @@ mod tests {
self.candidate_events.clone()
}

fn validator_discovery(ids: Vec<ValidatorId>) -> Vec<Option<AuthorityDiscoveryId>> {
vec![None; ids.len()]
}

fn dmq_contents(
&self,
recipient: ParaId,
Expand Down
46 changes: 24 additions & 22 deletions node/network/collator-protocol/src/collator_side.rs
Original file line number Diff line number Diff line change
Expand Up @@ -738,6 +738,7 @@ mod tests {
use polkadot_primitives::v1::{
BlockData, CandidateDescriptor, CollatorPair, ScheduledCore,
ValidatorIndex, GroupRotationInfo, AuthorityDiscoveryId,
SessionInfo,
};
use polkadot_subsystem::{ActiveLeavesUpdate, messages::{RuntimeApiMessage, RuntimeApiRequest}};
use polkadot_node_subsystem_util::TimeoutExt;
Expand Down Expand Up @@ -870,20 +871,6 @@ mod tests {
.collect()
}

fn next_group_validator_ids(&self) -> Vec<ValidatorId> {
self.next_group_validator_indices()
.iter()
.map(|i| self.validator_public[*i as usize].clone())
.collect()
}

/// Returns the unique count of validators in the current and next group.
fn current_and_next_group_unique_validator_count(&self) -> usize {
let mut indices = self.next_group_validator_indices().iter().collect::<HashSet<_>>();
indices.extend(self.current_group_validator_indices());
indices.len()
}

/// Generate a new relay parent and inform the subsystem about the new view.
///
/// If `merge_views == true` it means the subsystem will be informed that we working on the old `relay_parent`
Expand Down Expand Up @@ -1085,25 +1072,40 @@ mod tests {
}
);

let current_index = 1;
ordian marked this conversation as resolved.
Show resolved Hide resolved

// obtain the validator_id to authority_id mapping
assert_matches!(
overseer_recv(virtual_overseer).await,
AllMessages::RuntimeApi(RuntimeApiMessage::Request(
relay_parent,
RuntimeApiRequest::ValidatorDiscovery(validators, tx),
RuntimeApiRequest::SessionIndexForChild(tx),
)) => {
assert_eq!(relay_parent, test_state.relay_parent);
assert_eq!(validators.len(), test_state.current_and_next_group_unique_validator_count());
tx.send(Ok(current_index)).unwrap();
}
);

let current_validators = test_state.current_group_validator_ids();
let next_validators = test_state.next_group_validator_ids();
assert_matches!(
overseer_recv(virtual_overseer).await,
AllMessages::RuntimeApi(RuntimeApiMessage::Request(
relay_parent,
RuntimeApiRequest::SessionInfo(index, tx),
)) => {
assert_eq!(relay_parent, test_state.relay_parent);
assert_eq!(index, current_index);

assert!(validators.iter().all(|v| current_validators.contains(&v) || next_validators.contains(&v)));
let validators = test_state.current_group_validator_ids();
let current_discovery_keys = test_state.current_group_validator_authority_ids();
let next_discovery_keys = test_state.next_group_validator_authority_ids();

let current_validators = test_state.current_group_validator_authority_ids();
let next_validators = test_state.next_group_validator_authority_ids();
let discovery_keys = [&current_discovery_keys[..], &next_discovery_keys[..]].concat();

tx.send(Ok(current_validators.into_iter().chain(next_validators).map(Some).collect())).unwrap();
tx.send(Ok(Some(SessionInfo {
validators,
discovery_keys,
..Default::default()
}))).unwrap();
}
);

Expand Down
10 changes: 0 additions & 10 deletions node/network/collator-protocol/src/lib.rs
Original file line number Diff line number Diff line change
Expand Up @@ -60,16 +60,6 @@ enum Error {
Prometheus(#[from] prometheus::PrometheusError),
}

impl From<util::validator_discovery::Error> for Error {
fn from(me: util::validator_discovery::Error) -> Self {
match me {
util::validator_discovery::Error::Subsystem(s) => Error::Subsystem(s),
util::validator_discovery::Error::RuntimeApi(ra) => Error::RuntimeApi(ra),
util::validator_discovery::Error::Oneshot(c) => Error::Oneshot(c),
}
}
}

type Result<T> = std::result::Result<T, Error>;

/// What side of the collator protocol is being engaged
Expand Down
2 changes: 2 additions & 0 deletions node/subsystem-util/src/lib.rs
Original file line number Diff line number Diff line change
Expand Up @@ -37,6 +37,7 @@ use polkadot_primitives::v1::{
CandidateEvent, CommittedCandidateReceipt, CoreState, EncodeAs, PersistedValidationData,
GroupRotationInfo, Hash, Id as ParaId, ValidationData, OccupiedCoreAssumption,
SessionIndex, Signed, SigningContext, ValidationCode, ValidatorId, ValidatorIndex,
SessionInfo,
};
use sp_core::{
traits::SpawnNamed,
Expand Down Expand Up @@ -274,6 +275,7 @@ specialize_requests_ctx! {
fn request_validation_code_ctx(para_id: ParaId, assumption: OccupiedCoreAssumption) -> Option<ValidationCode>; ValidationCode;
fn request_candidate_pending_availability_ctx(para_id: ParaId) -> Option<CommittedCandidateReceipt>; CandidatePendingAvailability;
fn request_candidate_events_ctx() -> Vec<CandidateEvent>; CandidateEvents;
fn request_session_info_ctx(index: SessionIndex) -> Option<SessionInfo>; SessionInfo;
ordian marked this conversation as resolved.
Show resolved Hide resolved
}

/// From the given set of validators, find the first key we can sign with, if any.
Expand Down
57 changes: 29 additions & 28 deletions node/subsystem-util/src/validator_discovery.rs
Original file line number Diff line number Diff line change
Expand Up @@ -20,52 +20,53 @@ use std::collections::HashMap;
use std::pin::Pin;

use futures::{
channel::{mpsc, oneshot},
channel::mpsc,
task::{Poll, self},
stream,
};
use streamunordered::{StreamUnordered, StreamYield};
use thiserror::Error;

use polkadot_node_subsystem::{
errors::RuntimeApiError, SubsystemError,
messages::{AllMessages, RuntimeApiMessage, RuntimeApiRequest, NetworkBridgeMessage},
errors::RuntimeApiError,
messages::{AllMessages, NetworkBridgeMessage},
SubsystemContext,
};
use polkadot_primitives::v1::{Hash, ValidatorId, AuthorityDiscoveryId};
use sc_network::PeerId;

/// Error when making a request to connect to validators.
#[derive(Debug, Error)]
pub enum Error {
/// Attempted to send or receive on a oneshot channel which had been canceled
#[error(transparent)]
Oneshot(#[from] oneshot::Canceled),
/// A subsystem error.
#[error(transparent)]
Subsystem(#[from] SubsystemError),
/// An error in the Runtime API.
#[error(transparent)]
RuntimeApi(#[from] RuntimeApiError),
}
use crate::Error;

/// Utility function to make it easier to connect to validators.
pub async fn connect_to_validators<Context: SubsystemContext>(
ctx: &mut Context,
relay_parent: Hash,
validators: Vec<ValidatorId>,
) -> Result<ConnectionRequest, Error> {
// ValidatorId -> AuthorityDiscoveryId
let (tx, rx) = oneshot::channel();

ctx.send_message(AllMessages::RuntimeApi(
RuntimeApiMessage::Request(
relay_parent,
RuntimeApiRequest::ValidatorDiscovery(validators.clone(), tx),
)
)).await;
let current_index = crate::request_session_index_for_child_ctx(relay_parent, ctx).await?.await??;
let session_info = crate::request_session_info_ctx(
relay_parent,
current_index,
ctx,
).await?.await??;

let (session_validators, discovery_keys) = match session_info {
Some(info) => (info.validators, info.discovery_keys),
None => return Err(RuntimeApiError::from(
"No session_info found for the current index".to_owned()
).into()),
};

let id_to_index = session_validators.iter()
.zip(0usize..)
.collect::<HashMap<_, _>>();

// We assume the same ordering in authorities as in validators so we can do an index search
let maybe_authorities: Vec<_> = validators.iter()
.map(|id| {
let validator_index = id_to_index.get(&id);
validator_index.and_then(|i| discovery_keys.get(*i).cloned())
})
.collect();

let maybe_authorities = rx.await??;
let authorities: Vec<_> = maybe_authorities.iter()
.cloned()
.filter_map(|id| id)
Expand Down
12 changes: 3 additions & 9 deletions node/subsystem/src/messages.rs
Original file line number Diff line number Diff line change
Expand Up @@ -31,7 +31,7 @@ use polkadot_node_primitives::{
CollationGenerationConfig, MisbehaviorReport, SignedFullStatement, ValidationResult,
};
use polkadot_primitives::v1::{
AuthorityDiscoveryId, AvailableData, BackedCandidate, BlockNumber,
AuthorityDiscoveryId, AvailableData, BackedCandidate, BlockNumber, SessionInfo,
Header as BlockHeader, CandidateDescriptor, CandidateEvent, CandidateReceipt,
CollatorId, CommittedCandidateReceipt, CoreState, ErasureChunk,
GroupRotationInfo, Hash, Id as ParaId, OccupiedCoreAssumption,
Expand Down Expand Up @@ -434,14 +434,8 @@ pub enum RuntimeApiRequest {
/// Get all events concerning candidates (backing, inclusion, time-out) in the parent of
/// the block in whose state this request is executed.
CandidateEvents(RuntimeApiSender<Vec<CandidateEvent>>),
/// Get the `AuthorityDiscoveryId`s corresponding to the given `ValidatorId`s.
/// Currently this request is limited to validators in the current session.
///
/// Returns `None` for validators not found in the current session.
ValidatorDiscovery(
Vec<ValidatorId>,
RuntimeApiSender<Vec<Option<AuthorityDiscoveryId>>>,
),
/// Get the session info for the given session, if stored.
SessionInfo(SessionIndex, RuntimeApiSender<Option<SessionInfo>>),
/// Get all the pending inbound messages in the downward message queue for a para.
DmqContents(
ParaId,
Expand Down
41 changes: 34 additions & 7 deletions primitives/src/v1.rs
Original file line number Diff line number Diff line change
Expand Up @@ -671,6 +671,37 @@ pub enum CandidateEvent<H = Hash> {
CandidateTimedOut(CandidateReceipt<H>, HeadData),
}

/// Information about validator sets of a session.
#[derive(Clone, Encode, Decode, RuntimeDebug)]
#[cfg_attr(feature = "std", derive(PartialEq, Default))]
pub struct SessionInfo {
/// Validators in canonical ordering.
pub validators: Vec<ValidatorId>,
/// Validators' authority discovery keys for the session in canonical ordering.
pub discovery_keys: Vec<AuthorityDiscoveryId>,
/// The assignment and approval keys for validators.
// FIXME: implement this
#[codec(skip)]
pub approval_keys: Vec<()>,
/// Validators in shuffled ordering - these are the validator groups as produced
/// by the `Scheduler` module for the session and are typically referred to by
/// `GroupIndex`.
pub validator_groups: Vec<Vec<ValidatorIndex>>,
/// The number of availability cores used by the protocol during this session.
pub n_cores: u32,
/// The zeroth delay tranche width.
pub zeroth_delay_tranche_width: u32,
/// The number of samples we do of relay_vrf_modulo.
pub relay_vrf_modulo_samples: u32,
/// The number of delay tranches in total.
pub n_delay_tranches: u32,
/// How many slots (BABE / SASSAFRAS) must pass before an assignment is considered a
/// no-show.
pub no_show_slots: u32,
/// The number of validators needed to approve a block.
pub needed_approvals: u32,
}

sp_api::decl_runtime_apis! {
/// The API for querying the state of parachains on-chain.
pub trait ParachainHost<H: Decode = Hash, N: Encode + Decode = BlockNumber> {
Expand Down Expand Up @@ -710,6 +741,9 @@ sp_api::decl_runtime_apis! {
/// This can be used to instantiate a `SigningContext`.
fn session_index_for_child() -> SessionIndex;

/// Get the session info for the given session, if stored.
fn session_info(index: SessionIndex) -> Option<SessionInfo>;

/// Fetch the validation code used by a para, making the given `OccupiedCoreAssumption`.
///
/// Returns `None` if either the para is not registered or the assumption is `Freed`
Expand All @@ -735,13 +769,6 @@ sp_api::decl_runtime_apis! {
#[skip_initialize_block]
fn candidate_events() -> Vec<CandidateEvent<H>>;

/// Get the `AuthorityDiscoveryId`s corresponding to the given `ValidatorId`s.
/// Currently this request is limited to validators in the current session.
///
/// We assume that every validator runs authority discovery,
/// which would allow us to establish point-to-point connection to given validators.
fn validator_discovery(validators: Vec<ValidatorId>) -> Vec<Option<AuthorityDiscoveryId>>;

/// Get all the pending inbound messages in the downward message queue for a para.
fn dmq_contents(
recipient: Id,
Expand Down
2 changes: 1 addition & 1 deletion roadmap/implementers-guide/src/runtime/initializer.md
Original file line number Diff line number Diff line change
Expand Up @@ -23,7 +23,7 @@ The other parachains modules are initialized in this order:
1. Paras
1. Scheduler
1. Inclusion
1. Validity
1. SessionInfo
1. DMP
1. UMP
1. HRMP
Expand Down
7 changes: 3 additions & 4 deletions roadmap/implementers-guide/src/runtime/session_info.md
Original file line number Diff line number Diff line change
Expand Up @@ -30,11 +30,11 @@ struct SessionInfo {
// no-show.
no_show_slots: u32,
/// The number of validators needed to approve a block.
needed_approvals: u32,
needed_approvals: u32,
}
```

Storage Layout:
Storage Layout:

```rust
/// The earliest session for which previous session info is stored.
Expand All @@ -45,11 +45,10 @@ Sessions: map SessionIndex => Option<SessionInfo>,

## Session Change

1. Update the `CurrentSessionIndex`.
1. Update `EarliestStoredSession` based on `config.dispute_period` and remove all entries from `Sessions` from the previous value up to the new value.
1. Create a new entry in `Sessions` with information about the current session.

## Routines

* `earliest_stored_session() -> SessionIndex`: Yields the earliest session for which we have information stored.
* `session_info(session: SessionIndex) -> Option<SessionInfo>`: Yields the session info for the given session, if stored.
* `session_info(session: SessionIndex) -> Option<SessionInfo>`: Yields the session info for the given session, if stored.
Loading