Skip to content
This repository was archived by the owner on Nov 6, 2020. It is now read-only.

Commit 4938d5d

Browse files
tomusdrwdebris
authored andcommitted
Limit the number of transactions in pending set (#8777)
* Unordered iterator. * Use unordered and limited set if full not required. * Split timeout work into smaller timers. * Avoid collecting all pending transactions when mining * Remove println. * Use priority ordering in eth-filter. * Fix ethcore-miner tests and tx propagation. * Review grumbles addressed. * Add test for unordered not populating the cache. * Fix ethcore tests. * Fix light tests. * Fix ethcore-sync tests. * Fix RPC tests.
1 parent 4817b94 commit 4938d5d

File tree

29 files changed

+415
-115
lines changed

29 files changed

+415
-115
lines changed

Cargo.lock

+3-3
Some generated files are not rendered by default. Learn more about customizing how changed files appear on GitHub.

ethcore/light/src/net/mod.rs

+4-1
Original file line numberDiff line numberDiff line change
@@ -72,6 +72,9 @@ const PROPAGATE_TIMEOUT_INTERVAL: Duration = Duration::from_secs(5);
7272
const RECALCULATE_COSTS_TIMEOUT: TimerToken = 3;
7373
const RECALCULATE_COSTS_INTERVAL: Duration = Duration::from_secs(60 * 60);
7474

75+
/// Max number of transactions in a single packet.
76+
const MAX_TRANSACTIONS_TO_PROPAGATE: usize = 64;
77+
7578
// minimum interval between updates.
7679
const UPDATE_INTERVAL: Duration = Duration::from_millis(5000);
7780

@@ -647,7 +650,7 @@ impl LightProtocol {
647650
fn propagate_transactions(&self, io: &IoContext) {
648651
if self.capabilities.read().tx_relay { return }
649652

650-
let ready_transactions = self.provider.ready_transactions();
653+
let ready_transactions = self.provider.ready_transactions(MAX_TRANSACTIONS_TO_PROPAGATE);
651654
if ready_transactions.is_empty() { return }
652655

653656
trace!(target: "pip", "propagate transactions: {} ready", ready_transactions.len());

ethcore/light/src/net/tests/mod.rs

+2-2
Original file line numberDiff line numberDiff line change
@@ -173,8 +173,8 @@ impl Provider for TestProvider {
173173
})
174174
}
175175

176-
fn ready_transactions(&self) -> Vec<PendingTransaction> {
177-
self.0.client.ready_transactions()
176+
fn ready_transactions(&self, max_len: usize) -> Vec<PendingTransaction> {
177+
self.0.client.ready_transactions(max_len)
178178
}
179179
}
180180

ethcore/light/src/provider.rs

+8-5
Original file line numberDiff line numberDiff line change
@@ -125,7 +125,7 @@ pub trait Provider: Send + Sync {
125125
fn header_proof(&self, req: request::CompleteHeaderProofRequest) -> Option<request::HeaderProofResponse>;
126126

127127
/// Provide pending transactions.
128-
fn ready_transactions(&self) -> Vec<PendingTransaction>;
128+
fn ready_transactions(&self, max_len: usize) -> Vec<PendingTransaction>;
129129

130130
/// Provide a proof-of-execution for the given transaction proof request.
131131
/// Returns a vector of all state items necessary to execute the transaction.
@@ -280,8 +280,8 @@ impl<T: ProvingBlockChainClient + ?Sized> Provider for T {
280280
.map(|(_, proof)| ::request::ExecutionResponse { items: proof })
281281
}
282282

283-
fn ready_transactions(&self) -> Vec<PendingTransaction> {
284-
BlockChainClient::ready_transactions(self)
283+
fn ready_transactions(&self, max_len: usize) -> Vec<PendingTransaction> {
284+
BlockChainClient::ready_transactions(self, max_len)
285285
.into_iter()
286286
.map(|tx| tx.pending().clone())
287287
.collect()
@@ -367,9 +367,12 @@ impl<L: AsLightClient + Send + Sync> Provider for LightProvider<L> {
367367
None
368368
}
369369

370-
fn ready_transactions(&self) -> Vec<PendingTransaction> {
370+
fn ready_transactions(&self, max_len: usize) -> Vec<PendingTransaction> {
371371
let chain_info = self.chain_info();
372-
self.txqueue.read().ready_transactions(chain_info.best_block_number, chain_info.best_block_timestamp)
372+
let mut transactions = self.txqueue.read()
373+
.ready_transactions(chain_info.best_block_number, chain_info.best_block_timestamp);
374+
transactions.truncate(max_len);
375+
transactions
373376
}
374377
}
375378

ethcore/src/client/client.rs

+2-2
Original file line numberDiff line numberDiff line change
@@ -1956,8 +1956,8 @@ impl BlockChainClient for Client {
19561956
(*self.build_last_hashes(&self.chain.read().best_block_hash())).clone()
19571957
}
19581958

1959-
fn ready_transactions(&self) -> Vec<Arc<VerifiedTransaction>> {
1960-
self.importer.miner.ready_transactions(self)
1959+
fn ready_transactions(&self, max_len: usize) -> Vec<Arc<VerifiedTransaction>> {
1960+
self.importer.miner.ready_transactions(self, max_len, ::miner::PendingOrdering::Priority)
19611961
}
19621962

19631963
fn signing_chain_id(&self) -> Option<u64> {

ethcore/src/client/test_client.rs

+3-3
Original file line numberDiff line numberDiff line change
@@ -48,7 +48,7 @@ use log_entry::LocalizedLogEntry;
4848
use receipt::{Receipt, LocalizedReceipt, TransactionOutcome};
4949
use error::ImportResult;
5050
use vm::Schedule;
51-
use miner::{Miner, MinerService};
51+
use miner::{self, Miner, MinerService};
5252
use spec::Spec;
5353
use types::basic_account::BasicAccount;
5454
use types::pruning_info::PruningInfo;
@@ -806,8 +806,8 @@ impl BlockChainClient for TestBlockChainClient {
806806
self.traces.read().clone()
807807
}
808808

809-
fn ready_transactions(&self) -> Vec<Arc<VerifiedTransaction>> {
810-
self.miner.ready_transactions(self)
809+
fn ready_transactions(&self, max_len: usize) -> Vec<Arc<VerifiedTransaction>> {
810+
self.miner.ready_transactions(self, max_len, miner::PendingOrdering::Priority)
811811
}
812812

813813
fn signing_chain_id(&self) -> Option<u64> { None }

ethcore/src/client/traits.rs

+1-1
Original file line numberDiff line numberDiff line change
@@ -321,7 +321,7 @@ pub trait BlockChainClient : Sync + Send + AccountData + BlockChain + CallContra
321321
fn last_hashes(&self) -> LastHashes;
322322

323323
/// List all transactions that are allowed into the next block.
324-
fn ready_transactions(&self) -> Vec<Arc<VerifiedTransaction>>;
324+
fn ready_transactions(&self, max_len: usize) -> Vec<Arc<VerifiedTransaction>>;
325325

326326
/// Sorted list of transaction gas prices from at least last sample_size blocks.
327327
fn gas_price_corpus(&self, sample_size: usize) -> ::stats::Corpus<U256> {

ethcore/src/miner/miner.rs

+35-16
Original file line numberDiff line numberDiff line change
@@ -364,18 +364,28 @@ impl Miner {
364364

365365
let client = self.pool_client(chain);
366366
let engine_params = self.engine.params();
367-
let min_tx_gas = self.engine.schedule(chain_info.best_block_number).tx_gas.into();
367+
let min_tx_gas: U256 = self.engine.schedule(chain_info.best_block_number).tx_gas.into();
368368
let nonce_cap: Option<U256> = if chain_info.best_block_number + 1 >= engine_params.dust_protection_transition {
369369
Some((engine_params.nonce_cap_increment * (chain_info.best_block_number + 1)).into())
370370
} else {
371371
None
372372
};
373+
// we will never need more transactions than limit divided by min gas
374+
let max_transactions = if min_tx_gas.is_zero() {
375+
usize::max_value()
376+
} else {
377+
(*open_block.block().header().gas_limit() / min_tx_gas).as_u64() as usize
378+
};
373379

374380
let pending: Vec<Arc<_>> = self.transaction_queue.pending(
375381
client.clone(),
376-
chain_info.best_block_number,
377-
chain_info.best_block_timestamp,
378-
nonce_cap,
382+
pool::PendingSettings {
383+
block_number: chain_info.best_block_number,
384+
current_timestamp: chain_info.best_block_timestamp,
385+
nonce_cap,
386+
max_len: max_transactions,
387+
ordering: miner::PendingOrdering::Priority,
388+
}
379389
);
380390

381391
let took_ms = |elapsed: &Duration| {
@@ -807,20 +817,28 @@ impl miner::MinerService for Miner {
807817
self.transaction_queue.all_transactions()
808818
}
809819

810-
fn ready_transactions<C>(&self, chain: &C) -> Vec<Arc<VerifiedTransaction>> where
820+
fn ready_transactions<C>(&self, chain: &C, max_len: usize, ordering: miner::PendingOrdering)
821+
-> Vec<Arc<VerifiedTransaction>>
822+
where
811823
C: ChainInfo + Nonce + Sync,
812824
{
813825
let chain_info = chain.chain_info();
814826

815827
let from_queue = || {
828+
// We propagate transactions over the nonce cap.
829+
// The mechanism is only to limit number of transactions in pending block
830+
// those transactions are valid and will just be ready to be included in next block.
831+
let nonce_cap = None;
832+
816833
self.transaction_queue.pending(
817834
CachedNonceClient::new(chain, &self.nonce_cache),
818-
chain_info.best_block_number,
819-
chain_info.best_block_timestamp,
820-
// We propagate transactions over the nonce cap.
821-
// The mechanism is only to limit number of transactions in pending block
822-
// those transactions are valid and will just be ready to be included in next block.
823-
None,
835+
pool::PendingSettings {
836+
block_number: chain_info.best_block_number,
837+
current_timestamp: chain_info.best_block_timestamp,
838+
nonce_cap,
839+
max_len,
840+
ordering,
841+
},
824842
)
825843
};
826844

@@ -830,6 +848,7 @@ impl miner::MinerService for Miner {
830848
.iter()
831849
.map(|signed| pool::VerifiedTransaction::from_pending_block_transaction(signed.clone()))
832850
.map(Arc::new)
851+
.take(max_len)
833852
.collect()
834853
}, chain_info.best_block_number)
835854
};
@@ -1083,7 +1102,7 @@ mod tests {
10831102
use rustc_hex::FromHex;
10841103

10851104
use client::{TestBlockChainClient, EachBlockWith, ChainInfo, ImportSealedBlock};
1086-
use miner::MinerService;
1105+
use miner::{MinerService, PendingOrdering};
10871106
use test_helpers::{generate_dummy_client, generate_dummy_client_with_spec_and_accounts};
10881107
use transaction::{Transaction};
10891108

@@ -1179,7 +1198,7 @@ mod tests {
11791198
assert_eq!(res.unwrap(), ());
11801199
assert_eq!(miner.pending_transactions(best_block).unwrap().len(), 1);
11811200
assert_eq!(miner.pending_receipts(best_block).unwrap().len(), 1);
1182-
assert_eq!(miner.ready_transactions(&client).len(), 1);
1201+
assert_eq!(miner.ready_transactions(&client, 10, PendingOrdering::Priority).len(), 1);
11831202
// This method will let us know if pending block was created (before calling that method)
11841203
assert!(!miner.prepare_pending_block(&client));
11851204
}
@@ -1198,7 +1217,7 @@ mod tests {
11981217
assert_eq!(res.unwrap(), ());
11991218
assert_eq!(miner.pending_transactions(best_block), None);
12001219
assert_eq!(miner.pending_receipts(best_block), None);
1201-
assert_eq!(miner.ready_transactions(&client).len(), 1);
1220+
assert_eq!(miner.ready_transactions(&client, 10, PendingOrdering::Priority).len(), 1);
12021221
}
12031222

12041223
#[test]
@@ -1217,11 +1236,11 @@ mod tests {
12171236
assert_eq!(miner.pending_transactions(best_block), None);
12181237
assert_eq!(miner.pending_receipts(best_block), None);
12191238
// By default we use PendingSet::AlwaysSealing, so no transactions yet.
1220-
assert_eq!(miner.ready_transactions(&client).len(), 0);
1239+
assert_eq!(miner.ready_transactions(&client, 10, PendingOrdering::Priority).len(), 0);
12211240
// This method will let us know if pending block was created (before calling that method)
12221241
assert!(miner.prepare_pending_block(&client));
12231242
// After pending block is created we should see a transaction.
1224-
assert_eq!(miner.ready_transactions(&client).len(), 1);
1243+
assert_eq!(miner.ready_transactions(&client, 10, PendingOrdering::Priority).len(), 1);
12251244
}
12261245

12271246
#[test]

ethcore/src/miner/mod.rs

+5-2
Original file line numberDiff line numberDiff line change
@@ -26,6 +26,7 @@ pub mod pool_client;
2626
pub mod stratum;
2727

2828
pub use self::miner::{Miner, MinerOptions, Penalization, PendingSet, AuthoringParams};
29+
pub use ethcore_miner::pool::PendingOrdering;
2930

3031
use std::sync::Arc;
3132
use std::collections::BTreeMap;
@@ -156,10 +157,12 @@ pub trait MinerService : Send + Sync {
156157
fn next_nonce<C>(&self, chain: &C, address: &Address) -> U256
157158
where C: Nonce + Sync;
158159

159-
/// Get a list of all ready transactions.
160+
/// Get a list of all ready transactions either ordered by priority or unordered (cheaper).
160161
///
161162
/// Depending on the settings may look in transaction pool or only in pending block.
162-
fn ready_transactions<C>(&self, chain: &C) -> Vec<Arc<VerifiedTransaction>>
163+
/// If you don't need a full set of transactions, you can add `max_len` and create only a limited set of
164+
/// transactions.
165+
fn ready_transactions<C>(&self, chain: &C, max_len: usize, ordering: PendingOrdering) -> Vec<Arc<VerifiedTransaction>>
163166
where C: ChainInfo + Nonce + Sync;
164167

165168
/// Get a list of all transactions in the pool (some of them might not be ready for inclusion yet).

ethcore/src/tests/client.rs

+5-5
Original file line numberDiff line numberDiff line change
@@ -30,7 +30,7 @@ use test_helpers::{
3030
use types::filter::Filter;
3131
use ethereum_types::{U256, Address};
3232
use kvdb_rocksdb::{Database, DatabaseConfig};
33-
use miner::Miner;
33+
use miner::{Miner, PendingOrdering};
3434
use spec::Spec;
3535
use views::BlockView;
3636
use ethkey::KeyPair;
@@ -343,12 +343,12 @@ fn does_not_propagate_delayed_transactions() {
343343

344344
client.miner().import_own_transaction(&*client, tx0).unwrap();
345345
client.miner().import_own_transaction(&*client, tx1).unwrap();
346-
assert_eq!(0, client.ready_transactions().len());
347-
assert_eq!(0, client.miner().ready_transactions(&*client).len());
346+
assert_eq!(0, client.ready_transactions(10).len());
347+
assert_eq!(0, client.miner().ready_transactions(&*client, 10, PendingOrdering::Priority).len());
348348
push_blocks_to_client(&client, 53, 2, 2);
349349
client.flush_queue();
350-
assert_eq!(2, client.ready_transactions().len());
351-
assert_eq!(2, client.miner().ready_transactions(&*client).len());
350+
assert_eq!(2, client.ready_transactions(10).len());
351+
assert_eq!(2, client.miner().ready_transactions(&*client, 10, PendingOrdering::Priority).len());
352352
}
353353

354354
#[test]

ethcore/sync/src/api.rs

+16-5
Original file line numberDiff line numberDiff line change
@@ -359,6 +359,10 @@ impl SyncProvider for EthSync {
359359
}
360360
}
361361

362+
const PEERS_TIMER: TimerToken = 0;
363+
const SYNC_TIMER: TimerToken = 1;
364+
const TX_TIMER: TimerToken = 2;
365+
362366
struct SyncProtocolHandler {
363367
/// Shared blockchain client.
364368
chain: Arc<BlockChainClient>,
@@ -373,7 +377,9 @@ struct SyncProtocolHandler {
373377
impl NetworkProtocolHandler for SyncProtocolHandler {
374378
fn initialize(&self, io: &NetworkContext) {
375379
if io.subprotocol_name() != WARP_SYNC_PROTOCOL_ID {
376-
io.register_timer(0, Duration::from_secs(1)).expect("Error registering sync timer");
380+
io.register_timer(PEERS_TIMER, Duration::from_millis(700)).expect("Error registering peers timer");
381+
io.register_timer(SYNC_TIMER, Duration::from_millis(1100)).expect("Error registering sync timer");
382+
io.register_timer(TX_TIMER, Duration::from_millis(1300)).expect("Error registering transactions timer");
377383
}
378384
}
379385

@@ -399,12 +405,17 @@ impl NetworkProtocolHandler for SyncProtocolHandler {
399405
}
400406
}
401407

402-
fn timeout(&self, io: &NetworkContext, _timer: TimerToken) {
408+
fn timeout(&self, io: &NetworkContext, timer: TimerToken) {
403409
trace_time!("sync::timeout");
404410
let mut io = NetSyncIo::new(io, &*self.chain, &*self.snapshot_service, &self.overlay);
405-
self.sync.write().maintain_peers(&mut io);
406-
self.sync.write().maintain_sync(&mut io);
407-
self.sync.write().propagate_new_transactions(&mut io);
411+
match timer {
412+
PEERS_TIMER => self.sync.write().maintain_peers(&mut io),
413+
SYNC_TIMER => self.sync.write().maintain_sync(&mut io),
414+
TX_TIMER => {
415+
self.sync.write().propagate_new_transactions(&mut io);
416+
},
417+
_ => warn!("Unknown timer {} triggered.", timer),
418+
}
408419
}
409420
}
410421

ethcore/sync/src/chain/mod.rs

+7-3
Original file line numberDiff line numberDiff line change
@@ -149,6 +149,10 @@ const MAX_NEW_HASHES: usize = 64;
149149
const MAX_NEW_BLOCK_AGE: BlockNumber = 20;
150150
// maximal packet size with transactions (cannot be greater than 16MB - protocol limitation).
151151
const MAX_TRANSACTION_PACKET_SIZE: usize = 8 * 1024 * 1024;
152+
// Maximal number of transactions queried from miner to propagate.
153+
// This set is used to diff with transactions known by the peer and
154+
// we will send a difference of length up to `MAX_TRANSACTIONS_TO_PROPAGATE`.
155+
const MAX_TRANSACTIONS_TO_QUERY: usize = 4096;
152156
// Maximal number of transactions in sent in single packet.
153157
const MAX_TRANSACTIONS_TO_PROPAGATE: usize = 64;
154158
// Min number of blocks to be behind for a snapshot sync
@@ -1143,7 +1147,7 @@ pub mod tests {
11431147
use super::{PeerInfo, PeerAsking};
11441148
use ethcore::header::*;
11451149
use ethcore::client::{BlockChainClient, EachBlockWith, TestBlockChainClient, ChainInfo, BlockInfo};
1146-
use ethcore::miner::MinerService;
1150+
use ethcore::miner::{MinerService, PendingOrdering};
11471151
use private_tx::NoopPrivateTxHandler;
11481152

11491153
pub fn get_dummy_block(order: u32, parent_hash: H256) -> Bytes {
@@ -1355,7 +1359,7 @@ pub mod tests {
13551359
let mut io = TestIo::new(&mut client, &ss, &queue, None);
13561360
io.chain.miner.chain_new_blocks(io.chain, &[], &[], &[], &good_blocks, false);
13571361
sync.chain_new_blocks(&mut io, &[], &[], &[], &good_blocks, &[], &[]);
1358-
assert_eq!(io.chain.miner.ready_transactions(io.chain).len(), 1);
1362+
assert_eq!(io.chain.miner.ready_transactions(io.chain, 10, PendingOrdering::Priority).len(), 1);
13591363
}
13601364
// We need to update nonce status (because we say that the block has been imported)
13611365
for h in &[good_blocks[0]] {
@@ -1371,7 +1375,7 @@ pub mod tests {
13711375
}
13721376

13731377
// then
1374-
assert_eq!(client.miner.ready_transactions(&client).len(), 1);
1378+
assert_eq!(client.miner.ready_transactions(&client, 10, PendingOrdering::Priority).len(), 1);
13751379
}
13761380

13771381
#[test]

ethcore/sync/src/chain/propagator.rs

+2-1
Original file line numberDiff line numberDiff line change
@@ -33,6 +33,7 @@ use super::{
3333
MAX_PEERS_PROPAGATION,
3434
MAX_TRANSACTION_PACKET_SIZE,
3535
MAX_TRANSACTIONS_TO_PROPAGATE,
36+
MAX_TRANSACTIONS_TO_QUERY,
3637
MIN_PEERS_PROPAGATION,
3738
CONSENSUS_DATA_PACKET,
3839
NEW_BLOCK_HASHES_PACKET,
@@ -114,7 +115,7 @@ impl SyncPropagator {
114115
return 0;
115116
}
116117

117-
let transactions = io.chain().ready_transactions();
118+
let transactions = io.chain().ready_transactions(MAX_TRANSACTIONS_TO_QUERY);
118119
if transactions.is_empty() {
119120
return 0;
120121
}

0 commit comments

Comments
 (0)