feat(block): 新增异步BIO框架并集成到virtio-blk驱动 (#1659)
* feat(block): 新增异步BIO框架并集成到virtio-blk驱动 - 新增BIO请求结构体(BioRequest)和FIFO队列(BioQueue),支持异步读写操作 - 在BlockDevice trait中增加submit_bio、submit_bio_read、submit_bio_write方法,提供异步接口和同步 回退机制 - 重构virtio-blk驱动,使用异步BIO框架替代原有的同步读写,提升IO性能 - 新增IO线程处理BIO请求,支持批量处理和budget控制,避免CPU独占 - 使用tasklet处理中断下半部,完成BIO请求的异步回调 - 将BlockDevManager和BlockDevMeta中的SpinLock替换为Mutex,改善锁机制 Signed-off-by: longjin <longjin@DragonOS.org> * refactor(kernel): 将FutexData的锁类型从SpinLock替换为Mutex,并调整进程退出时调度实体停用的顺序 - 将FutexData结构体及其方法中的SpinLock和SpinLockGuard替换为Mutex和MutexGuard - 在进程退出流程中,将调度实体deactivate操作移至清理用户态资源之后,以避免潜在的 用户态缺页异常 Signed-off-by: longjin <longjin@DragonOS.org> * fix: 修复进程退出的时候,持自旋锁访问用户内存触发磁盘读取,导致panic的问题 Signed-off-by: longjin <longjin@DragonOS.org> --------- Signed-off-by: longjin <longjin@DragonOS.org>
This commit is contained in:
parent
dbb5a8346a
commit
a3cbe901c4
|
|
@ -0,0 +1,165 @@
|
|||
use core::slice::SlicePattern;
|
||||
|
||||
use alloc::{boxed::Box, sync::Arc, vec::Vec};
|
||||
use system_error::SystemError;
|
||||
|
||||
use crate::{libs::spinlock::SpinLock, sched::completion::Completion};
|
||||
|
||||
use super::block_device::{BlockId, LBA_SIZE};
|
||||
|
||||
/// BIO操作类型
|
||||
#[derive(Debug, Clone, Copy, PartialEq, Eq)]
|
||||
pub enum BioType {
|
||||
Read,
|
||||
Write,
|
||||
}
|
||||
|
||||
/// BIO请求状态
|
||||
#[derive(Debug, Clone, Copy, PartialEq, Eq)]
|
||||
enum BioState {
|
||||
Init,
|
||||
Submitted,
|
||||
Completed,
|
||||
Failed,
|
||||
}
|
||||
|
||||
/// 单个BIO请求
|
||||
pub struct BioRequest {
|
||||
inner: SpinLock<InnerBioRequest>,
|
||||
}
|
||||
|
||||
struct InnerBioRequest {
|
||||
bio_type: BioType,
|
||||
lba_start: BlockId,
|
||||
count: usize,
|
||||
buffer: Box<[u8]>, // 预分配缓冲区,todo: 引入页面整理之后,要加Pin来固定地址
|
||||
state: BioState,
|
||||
completion: Arc<Completion>,
|
||||
result: Option<Result<usize, SystemError>>,
|
||||
/// virtio-drivers返回的token,用于中断时匹配
|
||||
token: Option<u16>,
|
||||
}
|
||||
|
||||
impl BioRequest {
|
||||
/// 创建一个读请求
|
||||
pub fn new_read(lba_start: BlockId, count: usize) -> Arc<Self> {
|
||||
let buffer = vec![0u8; count * LBA_SIZE].into_boxed_slice();
|
||||
Arc::new(Self {
|
||||
inner: SpinLock::new(InnerBioRequest {
|
||||
bio_type: BioType::Read,
|
||||
lba_start,
|
||||
count,
|
||||
buffer,
|
||||
state: BioState::Init,
|
||||
completion: Arc::new(Completion::new()),
|
||||
result: None,
|
||||
token: None,
|
||||
}),
|
||||
})
|
||||
}
|
||||
|
||||
/// 创建一个写请求
|
||||
pub fn new_write(lba_start: BlockId, count: usize, data: &[u8]) -> Arc<Self> {
|
||||
let mut buffer = vec![0u8; count * LBA_SIZE].into_boxed_slice();
|
||||
let copy_len = data.len().min(buffer.len());
|
||||
buffer[..copy_len].copy_from_slice(&data[..copy_len]);
|
||||
|
||||
Arc::new(Self {
|
||||
inner: SpinLock::new(InnerBioRequest {
|
||||
bio_type: BioType::Write,
|
||||
lba_start,
|
||||
count,
|
||||
buffer,
|
||||
state: BioState::Init,
|
||||
completion: Arc::new(Completion::new()),
|
||||
result: None,
|
||||
token: None,
|
||||
}),
|
||||
})
|
||||
}
|
||||
|
||||
/// 标记为已提交,设置token
|
||||
pub fn mark_submitted(&self, token: u16) -> Result<(), SystemError> {
|
||||
let mut inner = self.inner.lock_irqsave();
|
||||
if inner.state != BioState::Init {
|
||||
return Err(SystemError::EINVAL);
|
||||
}
|
||||
inner.state = BioState::Submitted;
|
||||
inner.token = Some(token);
|
||||
Ok(())
|
||||
}
|
||||
|
||||
/// 获取缓冲区的可变引用(仅用于提交时)
|
||||
pub fn buffer_mut(&self) -> *mut [u8] {
|
||||
let mut inner = self.inner.lock_irqsave();
|
||||
inner.buffer.as_mut() as *mut [u8]
|
||||
}
|
||||
|
||||
/// 获取缓冲区的不可变引用
|
||||
pub fn buffer(&self) -> *const [u8] {
|
||||
let inner = self.inner.lock_irqsave();
|
||||
inner.buffer.as_slice() as *const [u8]
|
||||
}
|
||||
|
||||
/// 将数据写入BIO缓冲区(用于同步回退路径)
|
||||
pub fn write_buffer(&self, data: &[u8]) {
|
||||
let mut inner = self.inner.lock_irqsave();
|
||||
let copy_len = data.len().min(inner.buffer.len());
|
||||
inner.buffer[..copy_len].copy_from_slice(&data[..copy_len]);
|
||||
}
|
||||
|
||||
/// 获取BIO类型
|
||||
pub fn bio_type(&self) -> BioType {
|
||||
self.inner.lock_irqsave().bio_type
|
||||
}
|
||||
|
||||
/// 获取起始LBA
|
||||
pub fn lba_start(&self) -> BlockId {
|
||||
self.inner.lock_irqsave().lba_start
|
||||
}
|
||||
|
||||
/// 获取扇区数
|
||||
pub fn count(&self) -> usize {
|
||||
self.inner.lock_irqsave().count
|
||||
}
|
||||
|
||||
/// 获取token
|
||||
#[allow(dead_code)]
|
||||
pub fn token(&self) -> Option<u16> {
|
||||
self.inner.lock_irqsave().token
|
||||
}
|
||||
|
||||
/// 完成BIO请求
|
||||
pub fn complete(&self, result: Result<usize, SystemError>) {
|
||||
let completion = {
|
||||
let mut inner = self.inner.lock_irqsave();
|
||||
if matches!(inner.state, BioState::Completed | BioState::Failed) {
|
||||
return;
|
||||
}
|
||||
inner.state = if result.is_ok() {
|
||||
BioState::Completed
|
||||
} else {
|
||||
BioState::Failed
|
||||
};
|
||||
inner.result = Some(result);
|
||||
inner.completion.clone()
|
||||
};
|
||||
completion.complete();
|
||||
}
|
||||
|
||||
/// 等待BIO完成并返回结果
|
||||
pub fn wait(&self) -> Result<Vec<u8>, SystemError> {
|
||||
let completion = self.inner.lock_irqsave().completion.clone();
|
||||
|
||||
// 等待完成
|
||||
completion.wait_for_completion()?;
|
||||
|
||||
// 获取结果
|
||||
let inner = self.inner.lock_irqsave();
|
||||
match inner.result.as_ref() {
|
||||
Some(Ok(_)) => Ok(inner.buffer.to_vec()),
|
||||
Some(Err(e)) => Err(e.clone()),
|
||||
None => Err(SystemError::EIO),
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,65 @@
|
|||
use alloc::{collections::VecDeque, sync::Arc, vec::Vec};
|
||||
|
||||
use crate::libs::{spinlock::SpinLock, wait_queue::WaitQueue};
|
||||
|
||||
use super::bio::BioRequest;
|
||||
|
||||
/// 简单的FIFO BIO队列
|
||||
pub struct BioQueue {
|
||||
inner: SpinLock<InnerBioQueue>,
|
||||
wait_queue: WaitQueue,
|
||||
batch_size: usize,
|
||||
}
|
||||
|
||||
struct InnerBioQueue {
|
||||
queue: VecDeque<Arc<BioRequest>>,
|
||||
}
|
||||
|
||||
impl BioQueue {
|
||||
pub const DEFAULT_BATCH_SIZE: usize = 16;
|
||||
|
||||
pub fn new() -> Arc<Self> {
|
||||
Arc::new(Self {
|
||||
inner: SpinLock::new(InnerBioQueue {
|
||||
queue: VecDeque::new(),
|
||||
}),
|
||||
wait_queue: WaitQueue::default(),
|
||||
batch_size: Self::DEFAULT_BATCH_SIZE,
|
||||
})
|
||||
}
|
||||
|
||||
/// 提交BIO请求(非阻塞)
|
||||
pub fn submit(&self, bio: Arc<BioRequest>) {
|
||||
let should_wakeup = {
|
||||
let mut inner = self.inner.lock_irqsave();
|
||||
let was_empty = inner.queue.is_empty();
|
||||
inner.queue.push_back(bio);
|
||||
was_empty
|
||||
};
|
||||
|
||||
if should_wakeup {
|
||||
self.wait_queue.wakeup(None);
|
||||
}
|
||||
}
|
||||
|
||||
/// 批量取出请求(用于worker线程)
|
||||
pub fn drain_batch(&self) -> Vec<Arc<BioRequest>> {
|
||||
let mut inner = self.inner.lock_irqsave();
|
||||
let batch_size = self.batch_size.min(inner.queue.len());
|
||||
if batch_size == 0 {
|
||||
return Vec::new();
|
||||
}
|
||||
inner.queue.drain(..batch_size).collect()
|
||||
}
|
||||
|
||||
/// 检查队列是否为空
|
||||
pub fn is_empty(&self) -> bool {
|
||||
self.inner.lock_irqsave().queue.is_empty()
|
||||
}
|
||||
|
||||
/// Worker等待新请求
|
||||
pub fn wait_for_work(&self) -> Result<(), system_error::SystemError> {
|
||||
self.wait_queue
|
||||
.wait_event_interruptible(|| !self.is_empty(), None::<fn()>)
|
||||
}
|
||||
}
|
||||
|
|
@ -428,6 +428,51 @@ pub trait BlockDevice: Device {
|
|||
fn callback_gendisk_registered(&self, _gendisk: &Arc<GenDisk>) -> Result<(), SystemError> {
|
||||
Ok(())
|
||||
}
|
||||
|
||||
/// 提交异步BIO请求(默认不支持,由驱动选择性实现)
|
||||
fn submit_bio(&self, _bio: Arc<super::bio::BioRequest>) -> Result<(), SystemError> {
|
||||
Err(SystemError::ENOSYS)
|
||||
}
|
||||
|
||||
/// 提交异步读BIO(优先 submit_bio,不支持则同步回退)
|
||||
fn submit_bio_read(
|
||||
&self,
|
||||
lba_start: BlockId,
|
||||
count: usize,
|
||||
) -> Result<Arc<super::bio::BioRequest>, SystemError> {
|
||||
let bio = super::bio::BioRequest::new_read(lba_start, count);
|
||||
match self.submit_bio(bio.clone()) {
|
||||
Ok(()) => Ok(bio),
|
||||
Err(SystemError::ENOSYS) => {
|
||||
log::debug!("BlockDevice submit_bio_read ENOSYS, falling back to sync read");
|
||||
let mut buf = vec![0; count * LBA_SIZE];
|
||||
self.read_at_sync(lba_start, count, &mut buf)?;
|
||||
bio.write_buffer(&buf);
|
||||
bio.complete(Ok(count * LBA_SIZE));
|
||||
Ok(bio)
|
||||
}
|
||||
Err(e) => Err(e),
|
||||
}
|
||||
}
|
||||
|
||||
/// 提交异步写BIO(优先 submit_bio,不支持则同步回退)
|
||||
fn submit_bio_write(
|
||||
&self,
|
||||
lba_start: BlockId,
|
||||
count: usize,
|
||||
data: &[u8],
|
||||
) -> Result<Arc<super::bio::BioRequest>, SystemError> {
|
||||
let bio = super::bio::BioRequest::new_write(lba_start, count, data);
|
||||
match self.submit_bio(bio.clone()) {
|
||||
Ok(()) => Ok(bio),
|
||||
Err(SystemError::ENOSYS) => {
|
||||
self.write_at_sync(lba_start, count, data)?;
|
||||
bio.complete(Ok(count * LBA_SIZE));
|
||||
Ok(bio)
|
||||
}
|
||||
Err(e) => Err(e),
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
/// @brief 块设备框架函数集
|
||||
|
|
|
|||
|
|
@ -16,7 +16,7 @@ use crate::{
|
|||
vfs::{utils::DName, IndexNode},
|
||||
},
|
||||
init::initcall::INITCALL_POSTCORE,
|
||||
libs::spinlock::{SpinLock, SpinLockGuard},
|
||||
libs::mutex::{Mutex, MutexGuard},
|
||||
};
|
||||
|
||||
use super::{
|
||||
|
|
@ -41,7 +41,7 @@ pub fn block_dev_manager_init() -> Result<(), SystemError> {
|
|||
|
||||
/// 磁盘设备管理器
|
||||
pub struct BlockDevManager {
|
||||
inner: SpinLock<InnerBlockDevManager>,
|
||||
inner: Mutex<InnerBlockDevManager>,
|
||||
}
|
||||
|
||||
struct InnerBlockDevManager {
|
||||
|
|
@ -52,14 +52,14 @@ struct InnerBlockDevManager {
|
|||
impl BlockDevManager {
|
||||
pub fn new() -> Self {
|
||||
BlockDevManager {
|
||||
inner: SpinLock::new(InnerBlockDevManager {
|
||||
inner: Mutex::new(InnerBlockDevManager {
|
||||
disks: HashMap::new(),
|
||||
minors: HashMap::new(),
|
||||
}),
|
||||
}
|
||||
}
|
||||
|
||||
fn inner(&self) -> SpinLockGuard<'_, InnerBlockDevManager> {
|
||||
fn inner(&self) -> MutexGuard<'_, InnerBlockDevManager> {
|
||||
self.inner.lock()
|
||||
}
|
||||
|
||||
|
|
@ -306,7 +306,7 @@ pub struct BlockDevMeta {
|
|||
pub devname: DevName,
|
||||
pub major: Major,
|
||||
pub base_minor: u32,
|
||||
inner: SpinLock<InnerBlockDevMeta>,
|
||||
inner: Mutex<InnerBlockDevMeta>,
|
||||
}
|
||||
|
||||
pub struct InnerBlockDevMeta {
|
||||
|
|
@ -320,14 +320,14 @@ impl BlockDevMeta {
|
|||
devname,
|
||||
major,
|
||||
base_minor: block_dev_manager().next_minor(major),
|
||||
inner: SpinLock::new(InnerBlockDevMeta {
|
||||
inner: Mutex::new(InnerBlockDevMeta {
|
||||
gendisks: GenDiskMap::new(),
|
||||
dev_idx: 0, // 默认索引为0
|
||||
}),
|
||||
}
|
||||
}
|
||||
|
||||
pub(crate) fn inner(&self) -> SpinLockGuard<'_, InnerBlockDevMeta> {
|
||||
pub(crate) fn inner(&self) -> MutexGuard<'_, InnerBlockDevMeta> {
|
||||
self.inner.lock()
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -1,7 +1,10 @@
|
|||
pub mod bio;
|
||||
pub mod bio_queue;
|
||||
pub mod block_device;
|
||||
pub mod disk_info;
|
||||
pub mod gendisk;
|
||||
pub mod manager;
|
||||
|
||||
#[derive(Debug)]
|
||||
#[allow(dead_code)]
|
||||
pub enum SeekFrom {
|
||||
|
|
|
|||
|
|
@ -4,20 +4,28 @@ use core::{
|
|||
};
|
||||
|
||||
use alloc::{
|
||||
boxed::Box,
|
||||
string::{String, ToString},
|
||||
sync::{Arc, Weak},
|
||||
vec::Vec,
|
||||
};
|
||||
use bitmap::{static_bitmap, traits::BitMapOps};
|
||||
use hashbrown::HashMap;
|
||||
use log::error;
|
||||
use system_error::SystemError;
|
||||
use unified_init::macros::unified_init;
|
||||
use virtio_drivers::device::blk::{VirtIOBlk, SECTOR_SIZE};
|
||||
use virtio_drivers::{
|
||||
device::blk::{BlkReq, BlkResp, VirtIOBlk, SECTOR_SIZE},
|
||||
Error as VirtioError,
|
||||
};
|
||||
|
||||
use crate::{
|
||||
arch::CurrentIrqArch,
|
||||
driver::{
|
||||
base::{
|
||||
block::{
|
||||
bio::{BioRequest, BioType},
|
||||
bio_queue::BioQueue,
|
||||
block_device::{BlockDevice, BlockId, GeneralBlockRange, LBA_SIZE},
|
||||
disk_info::Partition,
|
||||
manager::{block_dev_manager, BlockDevMeta},
|
||||
|
|
@ -40,7 +48,11 @@ use crate::{
|
|||
VIRTIO_VENDOR_ID,
|
||||
},
|
||||
},
|
||||
exception::{irqdesc::IrqReturn, IrqNumber},
|
||||
exception::{
|
||||
irqdesc::IrqReturn,
|
||||
tasklet::{tasklet_schedule, Tasklet},
|
||||
InterruptArch, IrqNumber,
|
||||
},
|
||||
filesystem::{
|
||||
devfs::{DevFS, DeviceINode, LockedDevFSInode},
|
||||
kernfs::KernFSInode,
|
||||
|
|
@ -54,12 +66,168 @@ use crate::{
|
|||
rwsem::{RwSemReadGuard, RwSemWriteGuard},
|
||||
spinlock::{SpinLock, SpinLockGuard},
|
||||
},
|
||||
process::{
|
||||
kthread::{KernelThreadClosure, KernelThreadMechanism},
|
||||
ProcessControlBlock, ProcessManager,
|
||||
},
|
||||
sched::prio::MAX_RT_PRIO,
|
||||
time::{sleep::nanosleep, PosixTimeSpec},
|
||||
};
|
||||
|
||||
const VIRTIO_BLK_BASENAME: &str = "virtio_blk";
|
||||
|
||||
// IO线程的budget配置
|
||||
const IO_BUDGET: usize = 32; // 每次最多处理32个请求
|
||||
const SLEEP_MS: usize = 20; // 达到budget后睡眠20ms
|
||||
|
||||
/// Token映射表:virtqueue token -> BioRequest
|
||||
/// BIO请求的完整上下文,包含virtio需要的req和resp
|
||||
struct BioContext {
|
||||
bio: Arc<BioRequest>,
|
||||
req: Box<BlkReq>,
|
||||
resp: Box<BlkResp>,
|
||||
}
|
||||
|
||||
struct BioTokenMap {
|
||||
inner: SpinLock<HashMap<u16, BioContext>>,
|
||||
}
|
||||
|
||||
impl BioTokenMap {
|
||||
pub fn new() -> Arc<Self> {
|
||||
Arc::new(Self {
|
||||
inner: SpinLock::new(HashMap::new()),
|
||||
})
|
||||
}
|
||||
|
||||
pub fn insert(&self, token: u16, ctx: BioContext) -> Result<(), BioContext> {
|
||||
let mut inner = self.inner.lock_irqsave();
|
||||
if inner.contains_key(&token) {
|
||||
return Err(ctx);
|
||||
}
|
||||
inner.insert(token, ctx);
|
||||
Ok(())
|
||||
}
|
||||
|
||||
pub fn remove(&self, token: u16) -> Option<BioContext> {
|
||||
self.inner.lock_irqsave().remove(&token)
|
||||
}
|
||||
}
|
||||
|
||||
static mut VIRTIO_BLK_DRIVER: Option<Arc<VirtIOBlkDriver>> = None;
|
||||
|
||||
/// 中断下半部:完成 BIO 请求
|
||||
struct BioCompletionTasklet {
|
||||
token_map: Arc<BioTokenMap>,
|
||||
device: Weak<VirtIOBlkDevice>,
|
||||
tasklet: Arc<Tasklet>,
|
||||
}
|
||||
|
||||
impl BioCompletionTasklet {
|
||||
fn new(device: Weak<VirtIOBlkDevice>, token_map: Arc<BioTokenMap>) -> Arc<Self> {
|
||||
Arc::new_cyclic(|weak: &alloc::sync::Weak<Self>| {
|
||||
let weak_for_cb = weak.clone();
|
||||
let tasklet = Tasklet::new(
|
||||
move |_, _| {
|
||||
if let Some(tasklet) = weak_for_cb.upgrade() {
|
||||
tasklet.run();
|
||||
}
|
||||
},
|
||||
0,
|
||||
None,
|
||||
);
|
||||
BioCompletionTasklet {
|
||||
token_map,
|
||||
device,
|
||||
tasklet,
|
||||
}
|
||||
})
|
||||
}
|
||||
|
||||
fn schedule(&self) {
|
||||
tasklet_schedule(&self.tasklet);
|
||||
}
|
||||
|
||||
fn run(&self) {
|
||||
let device = match self.device.upgrade() {
|
||||
Some(dev) => dev,
|
||||
None => return,
|
||||
};
|
||||
loop {
|
||||
let mut inner = device.inner();
|
||||
let token = match inner.device_inner.peek_used() {
|
||||
Some(token) => token,
|
||||
None => break,
|
||||
};
|
||||
|
||||
let ctx = match self.token_map.remove(token) {
|
||||
Some(ctx) => ctx,
|
||||
None => {
|
||||
error!("VirtIOBlk: token {} not found in token_map", token);
|
||||
continue;
|
||||
}
|
||||
};
|
||||
|
||||
let mut ctx = ctx;
|
||||
let bio = ctx.bio.clone();
|
||||
let count = bio.count();
|
||||
let result = match bio.bio_type() {
|
||||
BioType::Read => {
|
||||
let buf_ptr = bio.buffer_mut();
|
||||
let buf = unsafe { &mut *buf_ptr };
|
||||
match unsafe {
|
||||
inner.device_inner.complete_read_blocks(
|
||||
token,
|
||||
&ctx.req,
|
||||
&mut buf[..count * LBA_SIZE],
|
||||
&mut ctx.resp,
|
||||
)
|
||||
} {
|
||||
Ok(_) => Ok(count * LBA_SIZE),
|
||||
Err(VirtioError::NotReady) => {
|
||||
if self.token_map.insert(token, ctx).is_err() {
|
||||
error!("VirtIOBlk: token {} reinsert failed", token);
|
||||
bio.complete(Err(SystemError::EIO));
|
||||
}
|
||||
break;
|
||||
}
|
||||
Err(e) => {
|
||||
error!("VirtIOBlk complete_read_blocks failed: {:?}", e);
|
||||
Err(SystemError::EIO)
|
||||
}
|
||||
}
|
||||
}
|
||||
BioType::Write => {
|
||||
let buf_ptr = bio.buffer();
|
||||
let buf = unsafe { &*buf_ptr };
|
||||
match unsafe {
|
||||
inner.device_inner.complete_write_blocks(
|
||||
token,
|
||||
&ctx.req,
|
||||
&buf[..count * LBA_SIZE],
|
||||
&mut ctx.resp,
|
||||
)
|
||||
} {
|
||||
Ok(_) => Ok(count * LBA_SIZE),
|
||||
Err(VirtioError::NotReady) => {
|
||||
if self.token_map.insert(token, ctx).is_err() {
|
||||
error!("VirtIOBlk: token {} reinsert failed", token);
|
||||
bio.complete(Err(SystemError::EIO));
|
||||
}
|
||||
break;
|
||||
}
|
||||
Err(e) => {
|
||||
error!("VirtIOBlk complete_write_blocks failed: {:?}", e);
|
||||
Err(SystemError::EIO)
|
||||
}
|
||||
}
|
||||
}
|
||||
};
|
||||
drop(inner);
|
||||
bio.complete(result);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
#[inline(always)]
|
||||
#[allow(dead_code)]
|
||||
fn virtio_blk_driver() -> Arc<VirtIOBlkDriver> {
|
||||
|
|
@ -201,8 +369,13 @@ impl VirtIOBlkDevice {
|
|||
|
||||
let mut device_inner: VirtIOBlk<HalImpl, VirtIOTransport> = device_inner.unwrap();
|
||||
device_inner.enable_interrupts();
|
||||
|
||||
// 创建BIO队列和token映射表
|
||||
let bio_queue = BioQueue::new();
|
||||
let bio_token_map = BioTokenMap::new();
|
||||
|
||||
let dev = Arc::new_cyclic(|self_ref| Self {
|
||||
blkdev_meta: BlockDevMeta::new(devname, Major::VIRTIO_BLK_MAJOR),
|
||||
blkdev_meta: BlockDevMeta::new(devname.clone(), Major::VIRTIO_BLK_MAJOR),
|
||||
self_ref: self_ref.clone(),
|
||||
dev_id,
|
||||
locked_kobj_state: LockedKObjectState::default(),
|
||||
|
|
@ -213,6 +386,10 @@ impl VirtIOBlkDevice {
|
|||
device_common: DeviceCommonData::default(),
|
||||
kobject_common: KObjectCommonData::default(),
|
||||
irq,
|
||||
bio_queue: Some(bio_queue.clone()),
|
||||
bio_token_map: Some(bio_token_map.clone()),
|
||||
io_thread_pcb: None, // 稍后初始化
|
||||
completion_tasklet: None,
|
||||
}),
|
||||
parent: RwLock::new(Weak::default()),
|
||||
fs: RwLock::new(Weak::default()),
|
||||
|
|
@ -222,11 +399,41 @@ impl VirtIOBlkDevice {
|
|||
),
|
||||
});
|
||||
|
||||
let device_weak = Arc::downgrade(&dev);
|
||||
|
||||
// 创建BIO完成 tasklet
|
||||
let completion_tasklet = BioCompletionTasklet::new(device_weak.clone(), bio_token_map);
|
||||
dev.inner().completion_tasklet = Some(completion_tasklet);
|
||||
|
||||
// 创建IO线程
|
||||
let thread_name = format!("virtio_blk_io_{}", devname.id());
|
||||
let io_thread = KernelThreadMechanism::create_and_run(
|
||||
KernelThreadClosure::EmptyClosure((
|
||||
alloc::boxed::Box::new(move || bio_io_thread_loop(device_weak.clone())),
|
||||
(),
|
||||
)),
|
||||
thread_name.clone(),
|
||||
);
|
||||
|
||||
if let Some(io_thread) = io_thread {
|
||||
// 设置FIFO调度策略
|
||||
if let Err(err) = ProcessManager::set_fifo_policy(&io_thread, MAX_RT_PRIO - 1) {
|
||||
error!("Failed to set FIFO policy for {}: {:?}", thread_name, err);
|
||||
}
|
||||
|
||||
// 保存IO线程PCB
|
||||
dev.inner().io_thread_pcb = Some(io_thread.clone());
|
||||
} else {
|
||||
error!("Failed to create IO thread for {}", thread_name);
|
||||
virtioblk_manager().free_id(devname.id());
|
||||
return None;
|
||||
}
|
||||
|
||||
Some(dev)
|
||||
}
|
||||
|
||||
fn inner(&self) -> SpinLockGuard<'_, InnerVirtIOBlkDevice> {
|
||||
self.inner.lock()
|
||||
self.inner.lock_irqsave()
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -317,11 +524,6 @@ impl BlockDevice for VirtIOBlkDevice {
|
|||
let inner = self.inner();
|
||||
let blocks = inner.device_inner.capacity() as usize * SECTOR_SIZE / LBA_SIZE;
|
||||
drop(inner);
|
||||
log::debug!(
|
||||
"VirtIOBlkDevice '{:?}' disk_range: 0..{}",
|
||||
self.dev_name(),
|
||||
blocks
|
||||
);
|
||||
GeneralBlockRange::new(0, blocks).unwrap()
|
||||
}
|
||||
|
||||
|
|
@ -331,19 +533,11 @@ impl BlockDevice for VirtIOBlkDevice {
|
|||
count: usize,
|
||||
buf: &mut [u8],
|
||||
) -> Result<usize, SystemError> {
|
||||
let mut inner = self.inner();
|
||||
|
||||
inner
|
||||
.device_inner
|
||||
.read_blocks(lba_id_start, &mut buf[..count * LBA_SIZE])
|
||||
.map_err(|e| {
|
||||
error!(
|
||||
"VirtIOBlkDevice '{:?}' read_at_sync failed: {:?}",
|
||||
self.dev_id, e
|
||||
);
|
||||
SystemError::EIO
|
||||
})?;
|
||||
|
||||
let bio = self.submit_bio_read(lba_id_start, count)?;
|
||||
let data = bio
|
||||
.wait()
|
||||
.inspect_err(|e| log::error!("VirtIOBlkDevice read_at_sync error: {:?}", e))?;
|
||||
buf[..count * LBA_SIZE].copy_from_slice(&data[..count * LBA_SIZE]);
|
||||
Ok(count)
|
||||
}
|
||||
|
||||
|
|
@ -353,10 +547,10 @@ impl BlockDevice for VirtIOBlkDevice {
|
|||
count: usize,
|
||||
buf: &[u8],
|
||||
) -> Result<usize, SystemError> {
|
||||
self.inner()
|
||||
.device_inner
|
||||
.write_blocks(lba_id_start, &buf[..count * LBA_SIZE])
|
||||
.map_err(|_| SystemError::EIO)?;
|
||||
let bio = self.submit_bio_write(lba_id_start, count, &buf[..count * LBA_SIZE])?;
|
||||
let _ = bio
|
||||
.wait()
|
||||
.inspect_err(|e| log::error!("VirtIOBlkDevice write_at_sync error: {:?}", e))?;
|
||||
Ok(count)
|
||||
}
|
||||
|
||||
|
|
@ -386,6 +580,17 @@ impl BlockDevice for VirtIOBlkDevice {
|
|||
.expect("Failed to get MBR partition table");
|
||||
mbr_table.partitions(Arc::downgrade(&device))
|
||||
}
|
||||
|
||||
/// 提交异步BIO请求
|
||||
fn submit_bio(&self, bio: Arc<BioRequest>) -> Result<(), SystemError> {
|
||||
let inner = self.inner();
|
||||
if let Some(bio_queue) = &inner.bio_queue {
|
||||
bio_queue.submit(bio);
|
||||
Ok(())
|
||||
} else {
|
||||
Err(SystemError::ENOSYS)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
struct InnerVirtIOBlkDevice {
|
||||
|
|
@ -395,6 +600,11 @@ struct InnerVirtIOBlkDevice {
|
|||
device_common: DeviceCommonData,
|
||||
kobject_common: KObjectCommonData,
|
||||
irq: Option<IrqNumber>,
|
||||
// 异步IO支持(阶段2新增)
|
||||
bio_queue: Option<Arc<BioQueue>>,
|
||||
bio_token_map: Option<Arc<BioTokenMap>>, // 阶段3将使用
|
||||
io_thread_pcb: Option<Arc<ProcessControlBlock>>,
|
||||
completion_tasklet: Option<Arc<BioCompletionTasklet>>,
|
||||
}
|
||||
|
||||
impl Debug for InnerVirtIOBlkDevice {
|
||||
|
|
@ -412,7 +622,14 @@ impl VirtIODevice for VirtIOBlkDevice {
|
|||
&self,
|
||||
_irq: crate::exception::IrqNumber,
|
||||
) -> Result<IrqReturn, system_error::SystemError> {
|
||||
// todo: handle virtio blk irq
|
||||
let mut inner = self.inner();
|
||||
if !inner.device_inner.ack_interrupt() {
|
||||
return Ok(crate::exception::irqdesc::IrqReturn::NotHandled);
|
||||
}
|
||||
let tasklet = inner.completion_tasklet.clone();
|
||||
if let Some(tasklet) = tasklet {
|
||||
tasklet.schedule();
|
||||
}
|
||||
Ok(crate::exception::irqdesc::IrqReturn::Handled)
|
||||
}
|
||||
|
||||
|
|
@ -576,6 +793,39 @@ impl KObject for VirtIOBlkDevice {
|
|||
}
|
||||
}
|
||||
|
||||
impl Drop for VirtIOBlkDevice {
|
||||
fn drop(&mut self) {
|
||||
let (bio_queue, bio_token_map) = {
|
||||
let inner = self.inner.lock_irqsave();
|
||||
(inner.bio_queue.clone(), inner.bio_token_map.clone())
|
||||
};
|
||||
|
||||
if let Some(bio_queue) = bio_queue {
|
||||
loop {
|
||||
let batch = bio_queue.drain_batch();
|
||||
if batch.is_empty() {
|
||||
break;
|
||||
}
|
||||
for bio in batch {
|
||||
bio.complete(Err(SystemError::ENODEV));
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
if let Some(token_map) = bio_token_map {
|
||||
let pending: Vec<BioContext> = token_map
|
||||
.inner
|
||||
.lock_irqsave()
|
||||
.drain()
|
||||
.map(|(_, v)| v)
|
||||
.collect();
|
||||
for ctx in pending {
|
||||
ctx.bio.complete(Err(SystemError::ENODEV));
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
#[unified_init(INITCALL_POSTCORE)]
|
||||
fn virtio_blk_driver_init() -> Result<(), SystemError> {
|
||||
let driver = VirtIOBlkDriver::new();
|
||||
|
|
@ -643,8 +893,8 @@ impl VirtIODriver for VirtIOBlkDriver {
|
|||
);
|
||||
SystemError::EINVAL
|
||||
})?;
|
||||
|
||||
block_dev_manager().register(dev as Arc<dyn BlockDevice>)?;
|
||||
|
||||
return Ok(());
|
||||
}
|
||||
|
||||
|
|
@ -762,3 +1012,151 @@ impl KObject for VirtIOBlkDriver {
|
|||
*self.kobj_state.write() = state;
|
||||
}
|
||||
}
|
||||
|
||||
/// IO线程入口函数
|
||||
fn bio_io_thread_loop(device_weak: Weak<VirtIOBlkDevice>) -> i32 {
|
||||
loop {
|
||||
let device = match device_weak.upgrade() {
|
||||
Some(dev) => dev,
|
||||
None => {
|
||||
// 设备已被销毁,线程退出
|
||||
break;
|
||||
}
|
||||
};
|
||||
|
||||
let bio_queue: Option<Arc<BioQueue>> = {
|
||||
let inner = device.inner();
|
||||
inner.bio_queue.clone()
|
||||
};
|
||||
|
||||
if let Some(bio_queue) = bio_queue {
|
||||
// 等待队列中有请求
|
||||
if let Err(e) = bio_queue.wait_for_work() {
|
||||
log::error!("virtio bio wait_for_work interrupted: {:?}", e);
|
||||
continue;
|
||||
}
|
||||
|
||||
let mut processed = 0;
|
||||
|
||||
// 批量提交新请求,遵守budget限制
|
||||
while processed < IO_BUDGET {
|
||||
let batch = bio_queue.drain_batch();
|
||||
if batch.is_empty() {
|
||||
break; // 队列空了,退出
|
||||
}
|
||||
|
||||
for bio in batch {
|
||||
if let Err(e) = submit_bio_to_virtio(&device, bio.clone()) {
|
||||
log::error!("virtio submit_bio_to_virtio failed: {:?}", e);
|
||||
// 失败时立即完成BIO
|
||||
bio.complete(Err(e));
|
||||
}
|
||||
processed += 1;
|
||||
|
||||
if processed >= IO_BUDGET {
|
||||
break; // 达到budget上限
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
// 达到budget,主动睡眠20ms,避免独占CPU
|
||||
if processed >= IO_BUDGET {
|
||||
let sleep_time = PosixTimeSpec::new(0, (SLEEP_MS as i64) * 1_000_000); // 20ms
|
||||
let _ = nanosleep(sleep_time);
|
||||
}
|
||||
} else {
|
||||
break;
|
||||
}
|
||||
}
|
||||
|
||||
0
|
||||
}
|
||||
|
||||
/// 将BIO请求提交到VirtIO设备(异步)
|
||||
fn submit_bio_to_virtio(
|
||||
device: &Arc<VirtIOBlkDevice>,
|
||||
bio: Arc<BioRequest>,
|
||||
) -> Result<(), SystemError> {
|
||||
// 获取BIO信息
|
||||
let lba_start = bio.lba_start();
|
||||
let count = bio.count();
|
||||
let bio_type = bio.bio_type();
|
||||
|
||||
// 获取token_map(clone以避免借用冲突)
|
||||
let token_map = {
|
||||
let inner = device.inner();
|
||||
inner
|
||||
.bio_token_map
|
||||
.as_ref()
|
||||
.ok_or(SystemError::EINVAL)?
|
||||
.clone()
|
||||
};
|
||||
|
||||
// 创建请求和响应结构
|
||||
let mut req = Box::new(BlkReq::default());
|
||||
let mut resp = Box::new(BlkResp::default());
|
||||
|
||||
// 获取buffer指针(在整个异步操作期间,bio会被BioContext持有,保证buffer有效)
|
||||
let irq_guard = unsafe { CurrentIrqArch::save_and_disable_irq() };
|
||||
|
||||
// 提交异步请求,获取token
|
||||
let token = {
|
||||
let mut inner = device.inner();
|
||||
match bio_type {
|
||||
BioType::Read => {
|
||||
let buf_ptr = bio.buffer_mut();
|
||||
let buf = unsafe { &mut *buf_ptr };
|
||||
unsafe {
|
||||
inner.device_inner.read_blocks_nb(
|
||||
lba_start,
|
||||
&mut req,
|
||||
&mut buf[..count * LBA_SIZE],
|
||||
&mut resp,
|
||||
)
|
||||
}
|
||||
.map_err(|e| {
|
||||
error!("VirtIOBlk async read_blocks_nb failed: {:?}", e);
|
||||
SystemError::EIO
|
||||
})?
|
||||
}
|
||||
BioType::Write => {
|
||||
let buf_ptr = bio.buffer();
|
||||
let buf = unsafe { &*buf_ptr };
|
||||
unsafe {
|
||||
inner.device_inner.write_blocks_nb(
|
||||
lba_start,
|
||||
&mut req,
|
||||
&buf[..count * LBA_SIZE],
|
||||
&mut resp,
|
||||
)
|
||||
}
|
||||
.map_err(|e| {
|
||||
error!("VirtIOBlk async write_blocks_nb failed: {:?}", e);
|
||||
SystemError::EIO
|
||||
})?
|
||||
}
|
||||
}
|
||||
};
|
||||
|
||||
// 保存上下文到token_map
|
||||
let ctx = BioContext {
|
||||
bio: bio.clone(),
|
||||
req,
|
||||
resp,
|
||||
};
|
||||
if token_map.insert(token, ctx).is_err() {
|
||||
drop(irq_guard);
|
||||
return Err(SystemError::EEXIST);
|
||||
}
|
||||
|
||||
// 标记BIO为已提交
|
||||
if let Err(e) = bio.mark_submitted(token) {
|
||||
token_map.remove(token);
|
||||
drop(irq_guard);
|
||||
return Err(e);
|
||||
}
|
||||
|
||||
drop(irq_guard);
|
||||
|
||||
Ok(())
|
||||
}
|
||||
|
|
|
|||
|
|
@ -196,14 +196,12 @@ impl VirtIODeviceManager {
|
|||
dev.set_virtio_device_index(virtio_index);
|
||||
dev.set_device_name(format!("virtio{}", virtio_index.data()));
|
||||
|
||||
log::debug!("virtio_device_add: dev: {:?}", dev.name());
|
||||
// log::debug!("virtio_device_add: dev: {:?}", dev.name());
|
||||
self.setup_irq(&dev)?;
|
||||
// 添加设备到设备管理器
|
||||
device_manager().add_device(dev.clone() as Arc<dyn Device>)?;
|
||||
let r = device_manager()
|
||||
.add_groups(&(dev.clone() as Arc<dyn Device>), &[&VirtIODeviceAttrGroup]);
|
||||
log::debug!("virtio_device_add: to setup irq");
|
||||
self.setup_irq(&dev).ok();
|
||||
log::debug!("virtio_device_add: setup irq done");
|
||||
|
||||
return r;
|
||||
}
|
||||
|
|
@ -212,7 +210,11 @@ impl VirtIODeviceManager {
|
|||
///
|
||||
/// 为virtio设备设置中断。
|
||||
fn setup_irq(&self, dev: &Arc<dyn VirtIODevice>) -> Result<(), SystemError> {
|
||||
let irq = dev.irq().ok_or(SystemError::EINVAL)?;
|
||||
let irq = dev.irq();
|
||||
if irq.is_none() {
|
||||
return Ok(());
|
||||
}
|
||||
let irq = irq.unwrap();
|
||||
if let Err(e) = irq_manager().request_irq(
|
||||
irq,
|
||||
dev.device_name(),
|
||||
|
|
|
|||
|
|
@ -12,6 +12,7 @@
|
|||
#![feature(fn_align)]
|
||||
#![feature(linked_list_retain)]
|
||||
#![feature(ptr_internals)]
|
||||
#![feature(slice_pattern)]
|
||||
#![feature(slice_ptr_get)]
|
||||
#![feature(negative_impls)]
|
||||
#![feature(sync_unsafe_cell)]
|
||||
|
|
|
|||
|
|
@ -19,7 +19,7 @@ use crate::{
|
|||
arch::{CurrentIrqArch, MMArch},
|
||||
exception::InterruptArch,
|
||||
libs::{
|
||||
spinlock::{SpinLock, SpinLockGuard},
|
||||
mutex::{Mutex, MutexGuard},
|
||||
wait_queue::{Waiter, Waker},
|
||||
},
|
||||
mm::{ucontext::AddressSpace, MemoryManagementArch, VirtAddr},
|
||||
|
|
@ -36,11 +36,11 @@ use super::constant::*;
|
|||
static mut FUTEX_DATA: Option<FutexData> = None;
|
||||
|
||||
pub struct FutexData {
|
||||
data: SpinLock<HashMap<FutexKey, FutexHashBucket>>,
|
||||
data: Mutex<HashMap<FutexKey, FutexHashBucket>>,
|
||||
}
|
||||
|
||||
impl FutexData {
|
||||
pub fn futex_map() -> SpinLockGuard<'static, HashMap<FutexKey, FutexHashBucket>> {
|
||||
pub fn futex_map() -> MutexGuard<'static, HashMap<FutexKey, FutexHashBucket>> {
|
||||
unsafe { FUTEX_DATA.as_ref().unwrap().data.lock() }
|
||||
}
|
||||
|
||||
|
|
@ -250,7 +250,7 @@ impl Futex {
|
|||
pub fn init() {
|
||||
unsafe {
|
||||
FUTEX_DATA = Some(FutexData {
|
||||
data: SpinLock::new(HashMap::new()),
|
||||
data: Mutex::new(HashMap::new()),
|
||||
})
|
||||
};
|
||||
}
|
||||
|
|
|
|||
|
|
@ -563,18 +563,12 @@ impl ProcessManager {
|
|||
pid = pcb.pid();
|
||||
pcb.wait_queue.mark_dead();
|
||||
|
||||
let rq = cpu_rq(smp_get_processor_id().data() as usize);
|
||||
let (rq, guard) = rq.self_lock();
|
||||
rq.deactivate_task(
|
||||
pcb.clone(),
|
||||
DequeueFlag::DEQUEUE_SLEEP | DequeueFlag::DEQUEUE_NOCLOCK,
|
||||
);
|
||||
drop(guard);
|
||||
|
||||
// 进行进程退出后的工作
|
||||
let thread = pcb.thread.write_irqsave();
|
||||
|
||||
if let Some(addr) = thread.clear_child_tid {
|
||||
let clear_child_tid = thread.clear_child_tid;
|
||||
let vfork_done = thread.vfork_done.clone();
|
||||
drop(thread);
|
||||
if let Some(addr) = clear_child_tid {
|
||||
// 按 Linux 语义:先清零 userland 的 *clear_child_tid,再 futex_wake(addr)
|
||||
let cleared_ok = unsafe {
|
||||
match clear_user_protected(addr, core::mem::size_of::<i32>()) {
|
||||
|
|
@ -600,10 +594,18 @@ impl ProcessManager {
|
|||
|
||||
RobustListHead::exit_robust_list(pcb.clone());
|
||||
// 如果是vfork出来的进程,则需要处理completion
|
||||
if thread.vfork_done.is_some() {
|
||||
thread.vfork_done.as_ref().unwrap().complete_all();
|
||||
if let Some(vd) = vfork_done {
|
||||
vd.complete_all();
|
||||
}
|
||||
drop(thread);
|
||||
|
||||
// clear_child_tid/robust_list 可能触发用户态缺页,必须在调度实体 deactive 前完成
|
||||
let rq = cpu_rq(smp_get_processor_id().data() as usize);
|
||||
let (rq, guard) = rq.self_lock();
|
||||
rq.deactivate_task(
|
||||
pcb.clone(),
|
||||
DequeueFlag::DEQUEUE_SLEEP | DequeueFlag::DEQUEUE_NOCLOCK,
|
||||
);
|
||||
drop(guard);
|
||||
|
||||
unsafe { pcb.basic_mut().set_user_vm(None) };
|
||||
pcb.exit_files();
|
||||
|
|
|
|||
Loading…
Reference in New Issue